2019-03-26 18:28:37 +00:00
|
|
|
#pragma once
|
|
|
|
#include <Processors/IProcessor.h>
|
2019-04-05 10:52:07 +00:00
|
|
|
#include <Processors/Executors/PipelineExecutor.h>
|
|
|
|
|
|
|
|
#include <DataStreams/IBlockInputStream.h>
|
|
|
|
#include <DataStreams/IBlockOutputStream.h>
|
|
|
|
|
2019-03-26 18:28:37 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
class TableStructureReadLock;
|
|
|
|
using TableStructureReadLockPtr = std::shared_ptr<TableStructureReadLock>;
|
|
|
|
using TableStructureReadLocks = std::vector<TableStructureReadLockPtr>;
|
|
|
|
|
|
|
|
class Context;
|
|
|
|
|
2019-04-08 14:55:20 +00:00
|
|
|
class IOutputFormat;
|
|
|
|
|
2019-03-26 18:28:37 +00:00
|
|
|
class QueryPipeline
|
|
|
|
{
|
|
|
|
public:
|
|
|
|
QueryPipeline() = default;
|
|
|
|
|
|
|
|
/// Each source must have single output port and no inputs. All outputs must have same header.
|
|
|
|
void init(Processors sources);
|
|
|
|
bool initialized() { return !processors.empty(); }
|
|
|
|
|
2019-04-09 10:17:25 +00:00
|
|
|
enum class StreamType
|
|
|
|
{
|
|
|
|
Main = 0,
|
|
|
|
Totals,
|
|
|
|
Extremes,
|
|
|
|
};
|
|
|
|
|
2019-03-26 18:28:37 +00:00
|
|
|
using ProcessorGetter = std::function<ProcessorPtr(const Block & header)>;
|
2019-04-09 10:17:25 +00:00
|
|
|
using ProcessorGetterWithStreamKind = std::function<ProcessorPtr(const Block & header, StreamType stream_type)>;
|
2019-03-26 18:28:37 +00:00
|
|
|
|
2019-04-05 10:52:07 +00:00
|
|
|
void addSimpleTransform(const ProcessorGetter & getter);
|
2019-04-09 10:17:25 +00:00
|
|
|
void addSimpleTransform(const ProcessorGetterWithStreamKind & getter);
|
2019-03-26 18:28:37 +00:00
|
|
|
void addPipe(Processors pipe);
|
|
|
|
void addTotalsHavingTransform(ProcessorPtr transform);
|
|
|
|
void addExtremesTransform(ProcessorPtr transform);
|
|
|
|
void addCreatingSetsTransform(ProcessorPtr transform);
|
|
|
|
void setOutput(ProcessorPtr output);
|
|
|
|
|
|
|
|
/// Will read from this stream after all data was read from other streams.
|
|
|
|
void addDelayedStream(ProcessorPtr source);
|
2019-04-08 09:31:49 +00:00
|
|
|
bool hasDelayedStream() const { return delayed_stream_port; }
|
2019-03-26 18:28:37 +00:00
|
|
|
|
|
|
|
void resize(size_t num_streams);
|
|
|
|
|
|
|
|
void unitePipelines(std::vector<QueryPipeline> && pipelines, const Context & context);
|
|
|
|
|
2019-04-05 10:52:07 +00:00
|
|
|
PipelineExecutorPtr execute(size_t num_threads);
|
2019-03-26 18:28:37 +00:00
|
|
|
|
2019-04-08 09:31:49 +00:00
|
|
|
size_t getNumStreams() const { return streams.size() + (hasDelayedStream() ? 1 : 0); }
|
|
|
|
size_t getNumMainStreams() const { return streams.size(); }
|
2019-04-03 11:21:38 +00:00
|
|
|
bool hasMoreThanOneStream() const { return getNumStreams() > 1; }
|
2019-03-26 18:28:37 +00:00
|
|
|
|
|
|
|
const Block & getHeader() const { return current_header; }
|
|
|
|
|
|
|
|
void addTableLock(const TableStructureReadLockPtr & lock) { table_locks.push_back(lock); }
|
|
|
|
|
|
|
|
/// For compatibility with IBlockInputStream.
|
|
|
|
void setProgressCallback(const ProgressCallback & callback);
|
|
|
|
void setProcessListElement(QueryStatus * elem);
|
|
|
|
|
2019-04-08 14:55:20 +00:00
|
|
|
/// Call after execution.
|
|
|
|
void finalize();
|
|
|
|
|
2019-03-26 18:28:37 +00:00
|
|
|
private:
|
|
|
|
|
|
|
|
/// All added processors.
|
|
|
|
Processors processors;
|
|
|
|
|
|
|
|
/// Port for each independent "stream".
|
|
|
|
std::vector<OutputPort *> streams;
|
|
|
|
|
|
|
|
/// Special ports for extremes and totals having.
|
|
|
|
OutputPort * totals_having_port = nullptr;
|
|
|
|
OutputPort * extremes_port = nullptr;
|
|
|
|
|
2019-04-08 09:31:49 +00:00
|
|
|
/// Special port for delayed stream.
|
|
|
|
OutputPort * delayed_stream_port = nullptr;
|
|
|
|
|
2019-03-26 18:28:37 +00:00
|
|
|
/// Common header for each stream.
|
|
|
|
Block current_header;
|
|
|
|
|
|
|
|
TableStructureReadLocks table_locks;
|
|
|
|
|
2019-04-08 14:55:20 +00:00
|
|
|
IOutputFormat * output_format = nullptr;
|
2019-03-26 18:28:37 +00:00
|
|
|
|
2019-04-05 10:52:07 +00:00
|
|
|
PipelineExecutorPtr executor;
|
|
|
|
std::shared_ptr<ThreadPool> pool;
|
2019-03-26 18:28:37 +00:00
|
|
|
|
|
|
|
void checkInitialized();
|
|
|
|
void checkSource(const ProcessorPtr & source);
|
|
|
|
void concatDelayedStream();
|
2019-04-08 14:55:20 +00:00
|
|
|
|
2019-04-09 10:17:25 +00:00
|
|
|
template <typename TProcessorGetter>
|
|
|
|
void addSimpleTransformImpl(const TProcessorGetter & getter);
|
|
|
|
|
2019-04-08 14:55:20 +00:00
|
|
|
void calcRowsBeforeLimit();
|
2019-03-26 18:28:37 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
}
|