2015-06-24 11:03:53 +00:00
|
|
|
|
#pragma once
|
|
|
|
|
|
|
|
|
|
#include <DB/Core/NamesAndTypes.h>
|
|
|
|
|
#include <DB/Storages/MergeTree/RangesInDataPart.h>
|
|
|
|
|
#include <mutex>
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
|
{
|
|
|
|
|
|
|
|
|
|
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// A batch of work for MergeTreeThreadBlockInputStream
|
2015-06-24 11:03:53 +00:00
|
|
|
|
struct MergeTreeReadTask
|
|
|
|
|
{
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// data part which should be read while performing this task
|
2015-06-24 11:03:53 +00:00
|
|
|
|
MergeTreeData::DataPartPtr data_part;
|
2015-09-16 15:26:20 +00:00
|
|
|
|
/** Ranges to read from `data_part`.
|
|
|
|
|
* Specified in reverse order for MergeTreeThreadBlockInputStream's convenience of calling .pop_back(). */
|
2015-06-24 11:03:53 +00:00
|
|
|
|
MarkRanges mark_ranges;
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// for virtual `part_index` virtual column
|
2015-06-24 11:03:53 +00:00
|
|
|
|
std::size_t part_index_in_query;
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// ordered list of column names used in this query, allows returning blocks with consistent ordering
|
2015-06-24 11:03:53 +00:00
|
|
|
|
const Names & ordered_names;
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// used to determine whether column should be filtered during PREWHERE or WHERE
|
2015-06-24 11:03:53 +00:00
|
|
|
|
const NameSet & column_name_set;
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// column names to read during WHERE
|
2015-06-24 11:03:53 +00:00
|
|
|
|
const NamesAndTypesList & columns;
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// column names to read during PREWHERE
|
2015-06-24 11:03:53 +00:00
|
|
|
|
const NamesAndTypesList & pre_columns;
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// should PREWHERE column be returned to requesting side?
|
2015-06-24 11:03:53 +00:00
|
|
|
|
const bool remove_prewhere_column;
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/// resulting block may require reordering in accordance with `ordered_names`
|
2015-07-03 15:08:21 +00:00
|
|
|
|
const bool should_reorder;
|
2015-06-24 11:03:53 +00:00
|
|
|
|
|
2015-07-03 15:08:21 +00:00
|
|
|
|
MergeTreeReadTask(
|
|
|
|
|
const MergeTreeData::DataPartPtr & data_part, const MarkRanges & ranges, const std::size_t part_index_in_query,
|
|
|
|
|
const Names & ordered_names, const NameSet & column_name_set, const NamesAndTypesList & columns,
|
|
|
|
|
const NamesAndTypesList & pre_columns, const bool remove_prewhere_column, const bool should_reorder)
|
2015-06-24 11:03:53 +00:00
|
|
|
|
: data_part{data_part}, mark_ranges{ranges}, part_index_in_query{part_index_in_query},
|
|
|
|
|
ordered_names{ordered_names}, column_name_set{column_name_set}, columns{columns}, pre_columns{pre_columns},
|
2015-07-03 15:08:21 +00:00
|
|
|
|
remove_prewhere_column{remove_prewhere_column}, should_reorder{should_reorder}
|
2015-06-24 11:03:53 +00:00
|
|
|
|
{}
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
using MergeTreeReadTaskPtr = std::unique_ptr<MergeTreeReadTask>;
|
|
|
|
|
|
2015-09-09 17:39:28 +00:00
|
|
|
|
/** Provides read tasks for MergeTreeThreadBlockInputStream`s in fine-grained batches, allowing for more
|
|
|
|
|
* uniform distribution of work amongst multiple threads. All parts and their ranges are divided into `threads`
|
|
|
|
|
* workloads with at most `sum_marks / threads` marks. Then, threads are performing reads from these workloads
|
2015-12-13 04:52:13 +00:00
|
|
|
|
* in "sequential" manner, requesting work in small batches. As soon as some thread has exhausted
|
2015-09-09 17:39:28 +00:00
|
|
|
|
* it's workload, it either is signaled that no more work is available (`do_not_steal_tasks == false`) or
|
|
|
|
|
* continues taking small batches from other threads' workloads (`do_not_steal_tasks == true`).
|
|
|
|
|
*/
|
2016-11-20 12:43:20 +00:00
|
|
|
|
class MergeTreeReadPool : private boost::noncopyable
|
2015-06-24 11:03:53 +00:00
|
|
|
|
{
|
2015-12-13 04:52:13 +00:00
|
|
|
|
public:
|
2016-10-24 02:02:37 +00:00
|
|
|
|
/** Pull could dynamically lower (backoff) number of threads, if read operation are too slow.
|
|
|
|
|
* Settings for that backoff.
|
2015-12-13 04:52:13 +00:00
|
|
|
|
*/
|
|
|
|
|
struct BackoffSettings
|
|
|
|
|
{
|
2016-10-24 02:02:37 +00:00
|
|
|
|
/// Pay attention only to reads, that took at least this amount of time. If set to 0 - means backoff is disabled.
|
2015-12-13 04:52:13 +00:00
|
|
|
|
size_t min_read_latency_ms = 1000;
|
2016-10-24 02:02:37 +00:00
|
|
|
|
/// Count events, when read throughput is less than specified bytes per second.
|
2015-12-13 04:52:13 +00:00
|
|
|
|
size_t max_throughput = 1048576;
|
2016-10-24 02:02:37 +00:00
|
|
|
|
/// Do not pay attention to event, if not enough time passed since previous event.
|
2015-12-13 04:52:13 +00:00
|
|
|
|
size_t min_interval_between_events_ms = 1000;
|
2016-10-24 02:02:37 +00:00
|
|
|
|
/// Number of events to do backoff - to lower number of threads in pool.
|
2015-12-13 04:52:13 +00:00
|
|
|
|
size_t min_events = 2;
|
|
|
|
|
|
2016-10-24 02:02:37 +00:00
|
|
|
|
/// Constants above is just an example.
|
2015-12-13 04:52:13 +00:00
|
|
|
|
BackoffSettings(const Settings & settings)
|
|
|
|
|
: min_read_latency_ms(settings.read_backoff_min_latency_ms.totalMilliseconds()),
|
|
|
|
|
max_throughput(settings.read_backoff_max_throughput),
|
|
|
|
|
min_interval_between_events_ms(settings.read_backoff_min_interval_between_events_ms.totalMilliseconds()),
|
|
|
|
|
min_events(settings.read_backoff_min_events)
|
|
|
|
|
{
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
BackoffSettings() : min_read_latency_ms(0) {}
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
BackoffSettings backoff_settings;
|
|
|
|
|
|
|
|
|
|
private:
|
2016-10-24 02:02:37 +00:00
|
|
|
|
/** State to track numbers of slow reads.
|
2015-12-13 04:52:13 +00:00
|
|
|
|
*/
|
|
|
|
|
struct BackoffState
|
|
|
|
|
{
|
|
|
|
|
size_t current_threads;
|
|
|
|
|
Stopwatch time_since_prev_event {CLOCK_MONOTONIC_COARSE};
|
|
|
|
|
size_t num_events = 0;
|
|
|
|
|
|
|
|
|
|
BackoffState(size_t threads) : current_threads(threads) {}
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
BackoffState backoff_state;
|
|
|
|
|
|
2015-06-24 11:03:53 +00:00
|
|
|
|
public:
|
2015-07-03 15:08:21 +00:00
|
|
|
|
MergeTreeReadPool(
|
2015-07-23 17:54:07 +00:00
|
|
|
|
const std::size_t threads, const std::size_t sum_marks, const std::size_t min_marks_for_concurrent_read,
|
|
|
|
|
RangesInDataParts parts, MergeTreeData & data, const ExpressionActionsPtr & prewhere_actions,
|
2015-09-01 12:24:38 +00:00
|
|
|
|
const String & prewhere_column_name, const bool check_columns, const Names & column_names,
|
2015-12-13 04:52:13 +00:00
|
|
|
|
const BackoffSettings & backoff_settings,
|
2016-11-20 12:43:20 +00:00
|
|
|
|
const bool do_not_steal_tasks = false);
|
2015-06-24 11:03:53 +00:00
|
|
|
|
|
2016-11-20 12:43:20 +00:00
|
|
|
|
MergeTreeReadTaskPtr getTask(const std::size_t min_marks_to_read, const std::size_t thread);
|
2015-06-24 11:03:53 +00:00
|
|
|
|
|
2016-10-24 02:02:37 +00:00
|
|
|
|
/** Each worker could call this method and pass information about read performance.
|
|
|
|
|
* If read performance is too low, pool could decide to lower number of threads: do not assign more tasks to several threads.
|
|
|
|
|
* This allows to overcome excessive load to disk subsystem, when reads are not from page cache.
|
2015-12-13 04:52:13 +00:00
|
|
|
|
*/
|
2016-11-20 12:43:20 +00:00
|
|
|
|
void profileFeedback(const ReadBufferFromFileBase::ProfileInfo info);
|
2015-12-13 04:52:13 +00:00
|
|
|
|
|
2015-12-05 07:01:18 +00:00
|
|
|
|
private:
|
2015-07-23 17:54:07 +00:00
|
|
|
|
std::vector<std::size_t> fillPerPartInfo(
|
|
|
|
|
RangesInDataParts & parts, const ExpressionActionsPtr & prewhere_actions, const String & prewhere_column_name,
|
2016-11-20 12:43:20 +00:00
|
|
|
|
const bool check_columns);
|
2015-07-23 17:54:07 +00:00
|
|
|
|
|
|
|
|
|
void fillPerThreadInfo(
|
|
|
|
|
const std::size_t threads, const std::size_t sum_marks, std::vector<std::size_t> per_part_sum_marks,
|
2016-11-20 12:43:20 +00:00
|
|
|
|
RangesInDataParts & parts, const std::size_t min_marks_for_concurrent_read);
|
2015-06-24 11:03:53 +00:00
|
|
|
|
|
2016-10-24 02:02:37 +00:00
|
|
|
|
|
2015-06-24 11:03:53 +00:00
|
|
|
|
/** Если некоторых запрошенных столбцов нет в куске,
|
2016-10-24 02:02:37 +00:00
|
|
|
|
* то выясняем, какие столбцы может быть необходимо дополнительно прочитать,
|
|
|
|
|
* чтобы можно было вычислить DEFAULT выражение для этих столбцов.
|
|
|
|
|
* Добавляет их в columns.
|
|
|
|
|
*/
|
2016-11-20 12:43:20 +00:00
|
|
|
|
NameSet injectRequiredColumns(const MergeTreeData::DataPartPtr & part, Names & columns) const;
|
2015-06-24 11:03:53 +00:00
|
|
|
|
|
|
|
|
|
std::vector<std::unique_ptr<Poco::ScopedReadRWLock>> per_part_columns_lock;
|
|
|
|
|
MergeTreeData & data;
|
2015-07-03 15:08:21 +00:00
|
|
|
|
Names column_names;
|
2015-12-13 04:52:13 +00:00
|
|
|
|
bool do_not_steal_tasks;
|
2015-06-24 11:03:53 +00:00
|
|
|
|
std::vector<NameSet> per_part_column_name_set;
|
|
|
|
|
std::vector<NamesAndTypesList> per_part_columns;
|
|
|
|
|
std::vector<NamesAndTypesList> per_part_pre_columns;
|
|
|
|
|
/// @todo actually all of these values are either true or false for the whole query, thus no vector required
|
2015-09-09 17:39:28 +00:00
|
|
|
|
std::vector<char> per_part_remove_prewhere_column;
|
|
|
|
|
std::vector<char> per_part_should_reorder;
|
2015-06-24 11:03:53 +00:00
|
|
|
|
|
2015-09-09 17:39:28 +00:00
|
|
|
|
struct Part
|
2015-07-23 17:54:07 +00:00
|
|
|
|
{
|
|
|
|
|
MergeTreeData::DataPartPtr data_part;
|
|
|
|
|
std::size_t part_index_in_query;
|
|
|
|
|
};
|
|
|
|
|
|
2015-09-09 17:39:28 +00:00
|
|
|
|
std::vector<Part> parts;
|
2015-07-23 17:54:07 +00:00
|
|
|
|
|
2015-09-09 17:39:28 +00:00
|
|
|
|
struct ThreadTask
|
2015-07-23 17:54:07 +00:00
|
|
|
|
{
|
2015-09-09 17:39:28 +00:00
|
|
|
|
struct PartIndexAndRange
|
2015-07-23 17:54:07 +00:00
|
|
|
|
{
|
|
|
|
|
std::size_t part_idx;
|
|
|
|
|
MarkRanges ranges;
|
|
|
|
|
};
|
|
|
|
|
|
2015-09-09 17:39:28 +00:00
|
|
|
|
std::vector<PartIndexAndRange> parts_and_ranges;
|
2015-07-23 17:54:07 +00:00
|
|
|
|
std::vector<std::size_t> sum_marks_in_parts;
|
|
|
|
|
};
|
|
|
|
|
|
2015-09-09 17:39:28 +00:00
|
|
|
|
std::vector<ThreadTask> threads_tasks;
|
2015-07-23 17:54:07 +00:00
|
|
|
|
|
2015-09-25 13:39:06 +00:00
|
|
|
|
std::set<std::size_t> remaining_thread_tasks;
|
2015-07-23 17:54:07 +00:00
|
|
|
|
|
2015-06-24 11:03:53 +00:00
|
|
|
|
mutable std::mutex mutex;
|
2015-12-13 04:52:13 +00:00
|
|
|
|
|
|
|
|
|
Logger * log = &Logger::get("MergeTreeReadPool");
|
2015-06-24 11:03:53 +00:00
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
using MergeTreeReadPoolPtr = std::shared_ptr<MergeTreeReadPool>;
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
}
|