mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-25 00:52:02 +00:00
Merge pull request #30247 from ClickHouse/removing-data-streams-folder
Removing data streams folder
This commit is contained in:
commit
8ab5270ded
@ -28,7 +28,7 @@
|
||||
#include <IO/ConnectionTimeouts.h>
|
||||
#include <IO/ConnectionTimeoutsContext.h>
|
||||
#include <IO/UseSSL.h>
|
||||
#include <DataStreams/RemoteQueryExecutor.h>
|
||||
#include <QueryPipeline/RemoteQueryExecutor.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Client/Connection.h>
|
||||
#include <Common/InterruptListener.h>
|
||||
@ -432,7 +432,7 @@ private:
|
||||
Progress progress;
|
||||
executor.setProgressCallback([&progress](const Progress & value) { progress.incrementPiecewiseAtomically(value); });
|
||||
|
||||
BlockStreamProfileInfo info;
|
||||
ProfileInfo info;
|
||||
while (Block block = executor.read())
|
||||
info.update(block);
|
||||
|
||||
|
@ -9,8 +9,8 @@
|
||||
#include <IO/ConnectionTimeoutsContext.h>
|
||||
#include <Interpreters/InterpreterInsertQuery.h>
|
||||
#include <Processors/Transforms/ExpressionTransform.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <Processors/Chain.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/Chain.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/Executors/PushingPipelineExecutor.h>
|
||||
#include <Processors/Sources/RemoteSource.h>
|
||||
|
@ -49,7 +49,7 @@
|
||||
#include <Parsers/ASTLiteral.h>
|
||||
#include <Parsers/ASTExpressionList.h>
|
||||
#include <Formats/FormatSettings.h>
|
||||
#include <DataStreams/RemoteQueryExecutor.h>
|
||||
#include <QueryPipeline/RemoteQueryExecutor.h>
|
||||
#include <IO/ConnectionTimeouts.h>
|
||||
#include <IO/Operators.h>
|
||||
#include <IO/ReadBufferFromString.h>
|
||||
|
@ -11,11 +11,11 @@
|
||||
#include <Poco/ThreadPool.h>
|
||||
#include <Processors/Formats/IOutputFormat.h>
|
||||
#include <Processors/Formats/IInputFormat.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Processors/Executors/CompletedPipelineExecutor.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <Server/HTTP/HTMLForm.h>
|
||||
#include <IO/ReadBufferFromString.h>
|
||||
|
||||
|
@ -15,7 +15,7 @@
|
||||
#include <DataTypes/DataTypeFactory.h>
|
||||
#include <DataTypes/DataTypeUUID.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <Processors/LimitTransform.h>
|
||||
#include <Common/SipHash.h>
|
||||
#include <Common/UTF8Helpers.h>
|
||||
@ -25,7 +25,7 @@
|
||||
#include <Common/assert_cast.h>
|
||||
#include <Formats/registerFormats.h>
|
||||
#include <Processors/Formats/IInputFormat.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/Executors/PushingPipelineExecutor.h>
|
||||
#include <Core/Block.h>
|
||||
|
@ -14,7 +14,7 @@
|
||||
#include <Poco/Net/HTTPServerResponse.h>
|
||||
#include <Poco/Net/HTMLForm.h>
|
||||
#include <Poco/ThreadPool.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Processors/Executors/CompletedPipelineExecutor.h>
|
||||
#include <Processors/Formats/IInputFormat.h>
|
||||
#include <base/logger_useful.h>
|
||||
|
@ -62,7 +62,7 @@
|
||||
#include <TableFunctions/registerTableFunctions.h>
|
||||
#include <Formats/registerFormats.h>
|
||||
#include <Storages/registerStorages.h>
|
||||
#include <DataStreams/ConnectionCollector.h>
|
||||
#include <QueryPipeline/ConnectionCollector.h>
|
||||
#include <Dictionaries/registerDictionaries.h>
|
||||
#include <Disks/registerDisks.h>
|
||||
#include <Common/Config/ConfigReloader.h>
|
||||
|
@ -1,8 +1,8 @@
|
||||
#include "LibraryBridgeHelper.h"
|
||||
|
||||
#include <DataStreams/formatBlock.h>
|
||||
#include <Formats/formatBlock.h>
|
||||
#include <Dictionaries/DictionarySourceHelpers.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <Processors/Formats/IInputFormat.h>
|
||||
#include <IO/WriteBufferFromOStream.h>
|
||||
#include <IO/WriteBufferFromString.h>
|
||||
|
@ -49,7 +49,7 @@ add_subdirectory (Backups)
|
||||
add_subdirectory (Columns)
|
||||
add_subdirectory (Common)
|
||||
add_subdirectory (Core)
|
||||
add_subdirectory (DataStreams)
|
||||
add_subdirectory (QueryPipeline)
|
||||
add_subdirectory (DataTypes)
|
||||
add_subdirectory (Dictionaries)
|
||||
add_subdirectory (Disks)
|
||||
@ -185,7 +185,7 @@ add_object_library(clickhouse_backups Backups)
|
||||
add_object_library(clickhouse_core Core)
|
||||
add_object_library(clickhouse_core_mysql Core/MySQL)
|
||||
add_object_library(clickhouse_compression Compression)
|
||||
add_object_library(clickhouse_datastreams DataStreams)
|
||||
add_object_library(clickhouse_querypipeline QueryPipeline)
|
||||
add_object_library(clickhouse_datatypes DataTypes)
|
||||
add_object_library(clickhouse_datatypes_serializations DataTypes/Serializations)
|
||||
add_object_library(clickhouse_databases Databases)
|
||||
@ -214,6 +214,7 @@ add_object_library(clickhouse_processors_transforms Processors/Transforms)
|
||||
add_object_library(clickhouse_processors_sources Processors/Sources)
|
||||
add_object_library(clickhouse_processors_sinks Processors/Sinks)
|
||||
add_object_library(clickhouse_processors_merges Processors/Merges)
|
||||
add_object_library(clickhouse_processors_ttl Processors/TTL)
|
||||
add_object_library(clickhouse_processors_merges_algorithms Processors/Merges/Algorithms)
|
||||
add_object_library(clickhouse_processors_queryplan Processors/QueryPlan)
|
||||
add_object_library(clickhouse_processors_queryplan_optimizations Processors/QueryPlan/Optimizations)
|
||||
|
@ -45,14 +45,13 @@
|
||||
#include <Processors/Formats/Impl/NullFormat.h>
|
||||
#include <Processors/Formats/IInputFormat.h>
|
||||
#include <Processors/Formats/IOutputFormat.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Processors/Executors/PullingAsyncPipelineExecutor.h>
|
||||
#include <Processors/Transforms/AddingDefaultsTransform.h>
|
||||
#include <Interpreters/ReplaceQueryParameterVisitor.h>
|
||||
#include <IO/WriteBufferFromOStream.h>
|
||||
#include <IO/CompressionMethod.h>
|
||||
#include <DataStreams/InternalTextLogs.h>
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
#include <Client/InternalTextLogs.h>
|
||||
|
||||
namespace fs = std::filesystem;
|
||||
|
||||
@ -294,7 +293,7 @@ void ClientBase::onReceiveExceptionFromServer(std::unique_ptr<Exception> && e)
|
||||
}
|
||||
|
||||
|
||||
void ClientBase::onProfileInfo(const BlockStreamProfileInfo & profile_info)
|
||||
void ClientBase::onProfileInfo(const ProfileInfo & profile_info)
|
||||
{
|
||||
if (profile_info.hasAppliedLimit() && output_format)
|
||||
output_format->setRowsBeforeLimit(profile_info.getRowsBeforeLimit());
|
||||
|
@ -112,7 +112,7 @@ private:
|
||||
void onTotals(Block & block, ASTPtr parsed_query);
|
||||
void onExtremes(Block & block, ASTPtr parsed_query);
|
||||
void onReceiveExceptionFromServer(std::unique_ptr<Exception> && e);
|
||||
void onProfileInfo(const BlockStreamProfileInfo & profile_info);
|
||||
void onProfileInfo(const ProfileInfo & profile_info);
|
||||
void onEndOfStream();
|
||||
void onProfileEvents(Block & block);
|
||||
|
||||
|
@ -10,8 +10,8 @@
|
||||
#include <IO/WriteHelpers.h>
|
||||
#include <IO/copyData.h>
|
||||
#include <IO/TimeoutSetter.h>
|
||||
#include <DataStreams/NativeReader.h>
|
||||
#include <DataStreams/NativeWriter.h>
|
||||
#include <Formats/NativeReader.h>
|
||||
#include <Formats/NativeWriter.h>
|
||||
#include <Client/Connection.h>
|
||||
#include <Client/ConnectionParameters.h>
|
||||
#include <Common/ClickHouseRevision.h>
|
||||
@ -25,8 +25,8 @@
|
||||
#include "Core/Block.h"
|
||||
#include <Interpreters/ClientInfo.h>
|
||||
#include <Compression/CompressionFactory.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/ISink.h>
|
||||
#include <Processors/Executors/PipelineExecutor.h>
|
||||
#include <pcg_random.hpp>
|
||||
@ -1017,9 +1017,9 @@ Progress Connection::receiveProgress() const
|
||||
}
|
||||
|
||||
|
||||
BlockStreamProfileInfo Connection::receiveProfileInfo() const
|
||||
ProfileInfo Connection::receiveProfileInfo() const
|
||||
{
|
||||
BlockStreamProfileInfo profile_info;
|
||||
ProfileInfo profile_info;
|
||||
profile_info.read(*in);
|
||||
return profile_info;
|
||||
}
|
||||
|
@ -257,7 +257,7 @@ private:
|
||||
std::vector<String> receiveMultistringMessage(UInt64 msg_type) const;
|
||||
std::unique_ptr<Exception> receiveException() const;
|
||||
Progress receiveProgress() const;
|
||||
BlockStreamProfileInfo receiveProfileInfo() const;
|
||||
ProfileInfo receiveProfileInfo() const;
|
||||
|
||||
void initInputBuffers();
|
||||
void initBlockInput();
|
||||
|
@ -6,9 +6,9 @@
|
||||
#include <Core/Block.h>
|
||||
#include <Core/Protocol.h>
|
||||
|
||||
#include <DataStreams/BlockStreamProfileInfo.h>
|
||||
#include <QueryPipeline/ProfileInfo.h>
|
||||
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <IO/ConnectionTimeouts.h>
|
||||
#include <IO/Progress.h>
|
||||
|
||||
@ -30,7 +30,7 @@ struct Packet
|
||||
std::unique_ptr<Exception> exception;
|
||||
std::vector<String> multistring_message;
|
||||
Progress progress;
|
||||
BlockStreamProfileInfo profile_info;
|
||||
ProfileInfo profile_info;
|
||||
std::vector<UUID> part_uuids;
|
||||
|
||||
Packet() : type(Protocol::Server::Hello) {}
|
||||
|
@ -1,4 +1,4 @@
|
||||
#include "InternalTextLogs.h"
|
||||
#include <Client/InternalTextLogs.h>
|
||||
#include <Core/Block.h>
|
||||
#include <Interpreters/InternalTextLogsQueue.h>
|
||||
#include <Common/typeid_cast.h>
|
@ -2,7 +2,7 @@
|
||||
|
||||
#include "Connection.h"
|
||||
#include <Interpreters/Context.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <IO/TimeoutSetter.h>
|
||||
#include <Interpreters/Session.h>
|
||||
|
||||
|
@ -1,4 +1,4 @@
|
||||
#include <DataStreams/SquashingTransform.h>
|
||||
#include <Common/SquashingTransform.h>
|
||||
#include <iostream>
|
||||
|
||||
|
@ -707,4 +707,27 @@ ColumnPtr getColumnFromBlock(const Block & block, const NameAndTypePair & column
|
||||
return current_column;
|
||||
}
|
||||
|
||||
|
||||
Block materializeBlock(const Block & block)
|
||||
{
|
||||
if (!block)
|
||||
return block;
|
||||
|
||||
Block res = block;
|
||||
size_t columns = res.columns();
|
||||
for (size_t i = 0; i < columns; ++i)
|
||||
{
|
||||
auto & element = res.getByPosition(i);
|
||||
element.column = element.column->convertToFullColumnIfConst();
|
||||
}
|
||||
|
||||
return res;
|
||||
}
|
||||
|
||||
void materializeBlockInplace(Block & block)
|
||||
{
|
||||
for (size_t i = 0; i < block.columns(); ++i)
|
||||
block.getByPosition(i).column = block.getByPosition(i).column->convertToFullColumnIfConst();
|
||||
}
|
||||
|
||||
}
|
||||
|
@ -196,4 +196,8 @@ void getBlocksDifference(const Block & lhs, const Block & rhs, std::string & out
|
||||
/// Properly handles cases, when column is a subcolumn and when it is compressed.
|
||||
ColumnPtr getColumnFromBlock(const Block & block, const NameAndTypePair & column);
|
||||
|
||||
/// Converts columns-constants to full columns ("materializes" them).
|
||||
Block materializeBlock(const Block & block);
|
||||
void materializeBlockInplace(Block & block);
|
||||
|
||||
}
|
||||
|
@ -9,7 +9,7 @@
|
||||
#include <IO/ReadBufferFromFile.h>
|
||||
#include <IO/LimitReadBuffer.h>
|
||||
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <Processors/Executors/PipelineExecutor.h>
|
||||
#include <Processors/Sinks/SinkToStorage.h>
|
||||
#include <Processors/Sinks/EmptySink.h>
|
||||
|
@ -1,7 +1,7 @@
|
||||
#pragma once
|
||||
|
||||
#include <Core/SettingsFields.h>
|
||||
#include <DataStreams/SizeLimits.h>
|
||||
#include <QueryPipeline/SizeLimits.h>
|
||||
#include <Formats/FormatSettings.h>
|
||||
|
||||
|
||||
|
@ -1,27 +0,0 @@
|
||||
#include <DataStreams/finalizeBlock.h>
|
||||
#include <DataTypes/DataTypeAggregateFunction.h>
|
||||
#include <Columns/ColumnAggregateFunction.h>
|
||||
#include <Common/typeid_cast.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
{
|
||||
void finalizeBlock(Block & block)
|
||||
{
|
||||
for (size_t i = 0; i < block.columns(); ++i)
|
||||
{
|
||||
ColumnWithTypeAndName & current = block.getByPosition(i);
|
||||
const DataTypeAggregateFunction * unfinalized_type = typeid_cast<const DataTypeAggregateFunction *>(current.type.get());
|
||||
|
||||
if (unfinalized_type)
|
||||
{
|
||||
current.type = unfinalized_type->getReturnType();
|
||||
if (current.column)
|
||||
{
|
||||
auto mut_column = IColumn::mutate(std::move(current.column));
|
||||
current.column = ColumnAggregateFunction::convertToValues(std::move(mut_column));
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
@ -1,9 +0,0 @@
|
||||
#pragma once
|
||||
|
||||
#include <Core/Block.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
/// Converts aggregate function columns with non-finalized states to final values
|
||||
void finalizeBlock(Block & block);
|
||||
}
|
@ -1,29 +0,0 @@
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
Block materializeBlock(const Block & block)
|
||||
{
|
||||
if (!block)
|
||||
return block;
|
||||
|
||||
Block res = block;
|
||||
size_t columns = res.columns();
|
||||
for (size_t i = 0; i < columns; ++i)
|
||||
{
|
||||
auto & element = res.getByPosition(i);
|
||||
element.column = element.column->convertToFullColumnIfConst();
|
||||
}
|
||||
|
||||
return res;
|
||||
}
|
||||
|
||||
void materializeBlockInplace(Block & block)
|
||||
{
|
||||
for (size_t i = 0; i < block.columns(); ++i)
|
||||
block.getByPosition(i).column = block.getByPosition(i).column->convertToFullColumnIfConst();
|
||||
}
|
||||
|
||||
}
|
@ -1,14 +0,0 @@
|
||||
#pragma once
|
||||
|
||||
#include <Core/Block.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
/** Converts columns-constants to full columns ("materializes" them).
|
||||
*/
|
||||
Block materializeBlock(const Block & block);
|
||||
void materializeBlockInplace(Block & block);
|
||||
|
||||
}
|
@ -4,7 +4,7 @@
|
||||
#include <Databases/DatabaseReplicatedSettings.h>
|
||||
#include <Common/ZooKeeper/ZooKeeper.h>
|
||||
#include <Core/BackgroundSchedulePool.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Interpreters/Context.h>
|
||||
|
||||
|
||||
|
@ -11,9 +11,9 @@
|
||||
# include <DataTypes/convertMySQLDataType.h>
|
||||
# include <Databases/MySQL/DatabaseMySQL.h>
|
||||
# include <Databases/MySQL/FetchTablesColumnsList.h>
|
||||
# include <Formats/MySQLSource.h>
|
||||
# include <Processors/Sources/MySQLSource.h>
|
||||
# include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
# include <Processors/QueryPipelineBuilder.h>
|
||||
# include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
# include <IO/Operators.h>
|
||||
# include <Interpreters/Context.h>
|
||||
# include <Parsers/ASTCreateQuery.h>
|
||||
|
@ -9,8 +9,8 @@
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <DataTypes/DataTypesNumber.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <Formats/MySQLSource.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/Sources/MySQLSource.h>
|
||||
#include <IO/WriteBufferFromString.h>
|
||||
#include <IO/WriteHelpers.h>
|
||||
#include <IO/Operators.h>
|
||||
|
@ -5,9 +5,9 @@
|
||||
#include <Core/Block.h>
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <DataTypes/DataTypesNumber.h>
|
||||
#include <Formats/MySQLSource.h>
|
||||
#include <Processors/Sources/MySQLSource.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <IO/ReadBufferFromFile.h>
|
||||
#include <IO/WriteBufferFromFile.h>
|
||||
#include <Common/quoteString.h>
|
||||
|
@ -9,14 +9,14 @@
|
||||
# include <random>
|
||||
# include <Columns/ColumnTuple.h>
|
||||
# include <Columns/ColumnDecimal.h>
|
||||
# include <Processors/QueryPipelineBuilder.h>
|
||||
# include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
# include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
# include <Processors/Executors/CompletedPipelineExecutor.h>
|
||||
# include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
# include <Processors/Transforms/CountingTransform.h>
|
||||
# include <Databases/MySQL/DatabaseMaterializedMySQL.h>
|
||||
# include <Databases/MySQL/MaterializeMetadata.h>
|
||||
# include <Formats/MySQLSource.h>
|
||||
# include <Processors/Sources/MySQLSource.h>
|
||||
# include <IO/ReadBufferFromString.h>
|
||||
# include <Interpreters/Context.h>
|
||||
# include <Interpreters/executeQuery.h>
|
||||
|
@ -8,7 +8,7 @@
|
||||
|
||||
# include <mutex>
|
||||
# include <Core/MySQL/MySQLClient.h>
|
||||
# include <DataStreams/BlockIO.h>
|
||||
# include <QueryPipeline/BlockIO.h>
|
||||
# include <DataTypes/DataTypeString.h>
|
||||
# include <DataTypes/DataTypesNumber.h>
|
||||
# include <Databases/DatabaseOrdinary.h>
|
||||
|
@ -14,7 +14,7 @@
|
||||
#include <Dictionaries/HierarchyDictionariesUtils.h>
|
||||
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
|
||||
namespace ProfileEvents
|
||||
{
|
||||
|
@ -2,11 +2,11 @@
|
||||
#include <memory>
|
||||
#include <Client/ConnectionPool.h>
|
||||
#include <Processors/Sources/RemoteSource.h>
|
||||
#include <DataStreams/RemoteQueryExecutor.h>
|
||||
#include <QueryPipeline/RemoteQueryExecutor.h>
|
||||
#include <Interpreters/ActionsDAG.h>
|
||||
#include <Interpreters/ExpressionActions.h>
|
||||
#include <Processors/Transforms/ExpressionTransform.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <IO/ConnectionTimeouts.h>
|
||||
#include <Interpreters/Session.h>
|
||||
#include <Interpreters/executeQuery.h>
|
||||
|
@ -16,7 +16,7 @@
|
||||
#include <Dictionaries/IDictionary.h>
|
||||
#include <Dictionaries/DictionaryStructure.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
|
@ -8,7 +8,7 @@
|
||||
#include <Dictionaries/DictionaryFactory.h>
|
||||
#include <Dictionaries/HierarchyDictionariesUtils.h>
|
||||
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
|
||||
namespace DB
|
||||
|
@ -4,8 +4,8 @@
|
||||
#include <base/LocalDateTime.h>
|
||||
#include <Common/ShellCommand.h>
|
||||
|
||||
#include <DataStreams/ShellCommandSource.h>
|
||||
#include <DataStreams/formatBlock.h>
|
||||
#include <Processors/Sources/ShellCommandSource.h>
|
||||
#include <Formats/formatBlock.h>
|
||||
|
||||
#include <Interpreters/Context.h>
|
||||
#include <IO/WriteHelpers.h>
|
||||
|
@ -4,7 +4,7 @@
|
||||
#include <base/LocalDateTime.h>
|
||||
#include <Common/ShellCommand.h>
|
||||
|
||||
#include <DataStreams/formatBlock.h>
|
||||
#include <Formats/formatBlock.h>
|
||||
|
||||
#include <Interpreters/Context.h>
|
||||
#include <IO/WriteHelpers.h>
|
||||
|
@ -7,7 +7,7 @@
|
||||
|
||||
#include <Dictionaries/IDictionarySource.h>
|
||||
#include <Dictionaries/DictionaryStructure.h>
|
||||
#include <DataStreams/ShellCommandSource.h>
|
||||
#include <Processors/Sources/ShellCommandSource.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
|
@ -10,7 +10,7 @@
|
||||
#include <Columns/ColumnNullable.h>
|
||||
#include <Functions/FunctionHelpers.h>
|
||||
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
|
||||
#include <Dictionaries//DictionarySource.h>
|
||||
|
@ -1,5 +1,5 @@
|
||||
#include "HTTPDictionarySource.h"
|
||||
#include <DataStreams/formatBlock.h>
|
||||
#include <Formats/formatBlock.h>
|
||||
#include <IO/ConnectionTimeouts.h>
|
||||
#include <IO/ConnectionTimeoutsContext.h>
|
||||
#include <IO/ReadWriteBufferFromHTTP.h>
|
||||
|
@ -1,7 +1,7 @@
|
||||
#pragma once
|
||||
|
||||
#include <Columns/IColumn.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
|
||||
#include <vector>
|
||||
#include <atomic>
|
||||
|
@ -11,8 +11,8 @@
|
||||
#include "registerDictionaries.h"
|
||||
#include <Core/Settings.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Storages/ExternalDataSourceConfiguration.h>
|
||||
|
||||
|
||||
|
@ -12,7 +12,7 @@
|
||||
# include "DictionaryStructure.h"
|
||||
# include "ExternalQueryBuilder.h"
|
||||
# include "IDictionarySource.h"
|
||||
# include <Formats/MySQLSource.h>
|
||||
# include <Processors/Sources/MySQLSource.h>
|
||||
|
||||
namespace Poco
|
||||
{
|
||||
|
@ -7,7 +7,7 @@
|
||||
#include <Columns/ColumnTuple.h>
|
||||
#include <DataTypes/DataTypeArray.h>
|
||||
#include <Functions/FunctionHelpers.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
#include <Dictionaries/DictionaryFactory.h>
|
||||
#include <Dictionaries/DictionarySource.h>
|
||||
|
@ -11,7 +11,7 @@
|
||||
#include <Processors/Transforms/PostgreSQLSource.h>
|
||||
#include "readInvalidateQuery.h"
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Storages/ExternalDataSourceConfiguration.h>
|
||||
#endif
|
||||
|
||||
|
@ -16,7 +16,7 @@
|
||||
#include "readInvalidateQuery.h"
|
||||
#include "registerDictionaries.h"
|
||||
#include <Common/escapeForFileName.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Processors/Formats/IInputFormat.h>
|
||||
|
||||
|
||||
|
@ -1,5 +1,5 @@
|
||||
#include "readInvalidateQuery.h"
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <IO/WriteBufferFromString.h>
|
||||
#include <Formats/FormatSettings.h>
|
||||
|
@ -8,7 +8,7 @@
|
||||
#include <Common/typeid_cast.h>
|
||||
#include <base/range.h>
|
||||
|
||||
#include <DataStreams/NativeReader.h>
|
||||
#include <Formats/NativeReader.h>
|
||||
#include <DataTypes/DataTypeLowCardinality.h>
|
||||
|
||||
|
@ -1,6 +1,6 @@
|
||||
#pragma once
|
||||
|
||||
#include <DataStreams/MarkInCompressedFile.h>
|
||||
#include <Formats/MarkInCompressedFile.h>
|
||||
#include <Common/PODArray.h>
|
||||
#include <Core/Block.h>
|
||||
|
@ -5,8 +5,8 @@
|
||||
#include <IO/VarInt.h>
|
||||
#include <Compression/CompressedWriteBuffer.h>
|
||||
|
||||
#include <DataStreams/MarkInCompressedFile.h>
|
||||
#include <DataStreams/NativeWriter.h>
|
||||
#include <Formats/MarkInCompressedFile.h>
|
||||
#include <Formats/NativeWriter.h>
|
||||
|
||||
#include <Common/typeid_cast.h>
|
||||
#include <DataTypes/DataTypeLowCardinality.h>
|
@ -1,6 +1,6 @@
|
||||
#include <DataStreams/TemporaryFileStream.h>
|
||||
#include <DataStreams/NativeReader.h>
|
||||
#include <DataStreams/NativeWriter.h>
|
||||
#include <Formats/TemporaryFileStream.h>
|
||||
#include <Formats/NativeReader.h>
|
||||
#include <Formats/NativeWriter.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/ISource.h>
|
||||
#include <Compression/CompressedWriteBuffer.h>
|
||||
@ -41,27 +41,4 @@ void TemporaryFileStream::write(const std::string & path, const Block & header,
|
||||
compressed_buf.finalize();
|
||||
}
|
||||
|
||||
TemporaryFileLazySource::TemporaryFileLazySource(const std::string & path_, const Block & header_)
|
||||
: ISource(header_)
|
||||
, path(path_)
|
||||
, done(false)
|
||||
{}
|
||||
|
||||
Chunk TemporaryFileLazySource::generate()
|
||||
{
|
||||
if (done)
|
||||
return {};
|
||||
|
||||
if (!stream)
|
||||
stream = std::make_unique<TemporaryFileStream>(path, header);
|
||||
|
||||
auto block = stream->block_in->read();
|
||||
if (!block)
|
||||
{
|
||||
done = true;
|
||||
stream.reset();
|
||||
}
|
||||
return Chunk(block.getColumns(), block.rows());
|
||||
}
|
||||
|
||||
}
|
@ -1,10 +1,10 @@
|
||||
#pragma once
|
||||
|
||||
#include <Processors/ISource.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Compression/CompressedReadBuffer.h>
|
||||
#include <IO/ReadBufferFromFile.h>
|
||||
#include <DataStreams/NativeReader.h>
|
||||
#include <Formats/NativeReader.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
@ -23,22 +23,4 @@ struct TemporaryFileStream
|
||||
static void write(const std::string & path, const Block & header, QueryPipelineBuilder builder, const std::string & codec);
|
||||
};
|
||||
|
||||
|
||||
class TemporaryFileLazySource : public ISource
|
||||
{
|
||||
public:
|
||||
TemporaryFileLazySource(const std::string & path_, const Block & header_);
|
||||
String getName() const override { return "TemporaryFileLazySource"; }
|
||||
|
||||
protected:
|
||||
Chunk generate() override;
|
||||
|
||||
private:
|
||||
const std::string path;
|
||||
Block header;
|
||||
bool done;
|
||||
|
||||
std::unique_ptr<TemporaryFileStream> stream;
|
||||
};
|
||||
|
||||
}
|
@ -1,8 +1,8 @@
|
||||
#include <Core/Block.h>
|
||||
#include <DataStreams/formatBlock.h>
|
||||
#include <Formats/formatBlock.h>
|
||||
#include <Processors/Formats/IOutputFormat.h>
|
||||
#include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Processors/Executors/CompletedPipelineExecutor.h>
|
||||
|
||||
namespace DB
|
@ -1,6 +1,5 @@
|
||||
#include <memory>
|
||||
#include <Columns/ColumnString.h>
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <Formats/FormatFactory.h>
|
||||
#include <Functions/FunctionFactory.h>
|
||||
|
@ -9,8 +9,7 @@
|
||||
#include <DataTypes/DataTypeLowCardinality.h>
|
||||
#include <Columns/ColumnArray.h>
|
||||
#include <Columns/ColumnTuple.h>
|
||||
#include <DataStreams/NativeWriter.h>
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
#include <Formats/NativeWriter.h>
|
||||
#include <IO/WriteBufferFromFile.h>
|
||||
#include <Compression/CompressedWriteBuffer.h>
|
||||
#include <Interpreters/Aggregator.h>
|
||||
|
@ -19,7 +19,7 @@
|
||||
#include <Common/assert_cast.h>
|
||||
#include <Common/filesystemHelpers.h>
|
||||
|
||||
#include <DataStreams/SizeLimits.h>
|
||||
#include <QueryPipeline/SizeLimits.h>
|
||||
|
||||
#include <Disks/SingleDiskVolume.h>
|
||||
|
||||
|
@ -1,7 +1,7 @@
|
||||
#include <Interpreters/AsynchronousInsertQueue.h>
|
||||
|
||||
#include <Core/Settings.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Interpreters/InterpreterInsertQuery.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Processors/Transforms/getSourceFromASTInsertQuery.h>
|
||||
@ -9,7 +9,7 @@
|
||||
#include <Processors/Executors/StreamingFormatExecutor.h>
|
||||
#include <Processors/Executors/CompletedPipelineExecutor.h>
|
||||
#include <Processors/Transforms/AddingDefaultsTransform.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <IO/ConcatReadBuffer.h>
|
||||
#include <IO/ReadBufferFromMemory.h>
|
||||
#include <IO/ReadBufferFromString.h>
|
||||
|
@ -6,7 +6,7 @@
|
||||
#include <Interpreters/IInterpreter.h>
|
||||
#include <Interpreters/ProcessList.h>
|
||||
#include <Interpreters/OptimizeShardingKeyRewriteInVisitor.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <Processors/QueryPlan/QueryPlan.h>
|
||||
#include <Processors/QueryPlan/ReadFromRemote.h>
|
||||
#include <Processors/QueryPlan/UnionStep.h>
|
||||
|
@ -2,7 +2,6 @@
|
||||
|
||||
#include <Columns/ColumnTuple.h>
|
||||
#include <Columns/ColumnNullable.h>
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
#include <DataTypes/DataTypeTuple.h>
|
||||
#include <DataTypes/DataTypeNullable.h>
|
||||
#include <IO/WriteHelpers.h>
|
||||
|
@ -23,7 +23,6 @@
|
||||
|
||||
#include <Storages/StorageDictionary.h>
|
||||
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
|
||||
#include <Core/ColumnNumbers.h>
|
||||
#include <Common/typeid_cast.h>
|
||||
|
@ -21,7 +21,7 @@
|
||||
#include <Columns/ColumnString.h>
|
||||
#include <Columns/ColumnFixedString.h>
|
||||
|
||||
#include <DataStreams/SizeLimits.h>
|
||||
#include <QueryPipeline/SizeLimits.h>
|
||||
|
||||
#include <Core/Block.h>
|
||||
|
||||
|
@ -1,6 +1,6 @@
|
||||
#pragma once
|
||||
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Interpreters/Context_fwd.h>
|
||||
#include <Parsers/IAST_fwd.h>
|
||||
|
||||
|
@ -1,6 +1,6 @@
|
||||
#include <Storages/IStorage.h>
|
||||
#include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <Parsers/queryToString.h>
|
||||
#include <Common/typeid_cast.h>
|
||||
|
@ -1,7 +1,7 @@
|
||||
#include <Storages/IStorage.h>
|
||||
#include <Parsers/TablePropertiesQueriesASTs.h>
|
||||
#include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <DataTypes/DataTypesNumber.h>
|
||||
#include <Columns/ColumnsNumber.h>
|
||||
#include <Interpreters/Context.h>
|
||||
|
@ -1,6 +1,6 @@
|
||||
#include <Interpreters/InterpreterExplainQuery.h>
|
||||
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <Interpreters/InDepthNodeVisitor.h>
|
||||
@ -17,7 +17,7 @@
|
||||
#include <Processors/QueryPlan/QueryPlan.h>
|
||||
#include <Processors/QueryPlan/Optimizations/QueryPlanOptimizationSettings.h>
|
||||
#include <Processors/QueryPlan/BuildQueryPipelineSettings.h>
|
||||
#include <Processors/printPipeline.h>
|
||||
#include <QueryPipeline/printPipeline.h>
|
||||
|
||||
#include <Common/JSONBuilder.h>
|
||||
|
||||
|
@ -1,6 +1,6 @@
|
||||
#pragma once
|
||||
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Interpreters/IInterpreter.h>
|
||||
#include <Parsers/ASTInsertQuery.h>
|
||||
#include <Storages/StorageInMemoryMetadata.h>
|
||||
|
@ -1,5 +1,3 @@
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
|
||||
#include <DataTypes/DataTypeAggregateFunction.h>
|
||||
|
||||
#include <Parsers/ASTFunction.h>
|
||||
@ -35,7 +33,7 @@
|
||||
#include <Interpreters/QueryAliasesVisitor.h>
|
||||
#include <Interpreters/replaceAliasColumnsInQuery.h>
|
||||
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <Processors/QueryPlan/AggregatingStep.h>
|
||||
#include <Processors/QueryPlan/ArrayJoinStep.h>
|
||||
#include <Processors/QueryPlan/CreatingSetsStep.h>
|
||||
|
@ -2,7 +2,7 @@
|
||||
#include <Parsers/TablePropertiesQueriesASTs.h>
|
||||
#include <Parsers/formatAST.h>
|
||||
#include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <DataTypes/DataTypesNumber.h>
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <Columns/ColumnString.h>
|
||||
|
@ -15,7 +15,7 @@ limitations under the License. */
|
||||
#include <Interpreters/InterpreterWatchQuery.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Access/AccessFlags.h>
|
||||
#include <DataStreams/StreamLocalLimits.h>
|
||||
#include <QueryPipeline/StreamLocalLimits.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
|
@ -12,7 +12,7 @@ See the License for the specific language governing permissions and
|
||||
limitations under the License. */
|
||||
|
||||
#include <Core/QueryProcessingStage.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Interpreters/IInterpreter.h>
|
||||
#include <Parsers/IAST_fwd.h>
|
||||
#include <Storages/IStorage.h>
|
||||
|
@ -4,8 +4,7 @@
|
||||
#include <Columns/ColumnLowCardinality.h>
|
||||
|
||||
#include <Core/SortCursor.h>
|
||||
#include <DataStreams/TemporaryFileStream.h>
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
#include <Formats/TemporaryFileStream.h>
|
||||
#include <DataTypes/DataTypeNullable.h>
|
||||
#include <DataTypes/DataTypesNumber.h>
|
||||
#include <Interpreters/MergeJoin.h>
|
||||
@ -13,7 +12,7 @@
|
||||
#include <Interpreters/join_common.h>
|
||||
#include <Interpreters/sortBlock.h>
|
||||
#include <Processors/Sources/BlocksListSource.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/Transforms/MergeSortingTransform.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
|
||||
|
@ -7,7 +7,7 @@
|
||||
#include <Core/SortDescription.h>
|
||||
#include <Interpreters/IJoin.h>
|
||||
#include <Interpreters/SortedBlocksWriter.h>
|
||||
#include <DataStreams/SizeLimits.h>
|
||||
#include <QueryPipeline/SizeLimits.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
@ -11,7 +11,7 @@
|
||||
#include <Processors/Transforms/CreatingSetsTransform.h>
|
||||
#include <Processors/Transforms/MaterializingTransform.h>
|
||||
#include <Processors/Sources/NullSource.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/QueryPlan/QueryPlan.h>
|
||||
#include <Processors/QueryPlan/ExpressionStep.h>
|
||||
#include <Processors/QueryPlan/FilterStep.h>
|
||||
|
@ -1,7 +1,7 @@
|
||||
#pragma once
|
||||
|
||||
#include <Core/Defines.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <IO/Progress.h>
|
||||
#include <Interpreters/CancellationCode.h>
|
||||
#include <Interpreters/ClientInfo.h>
|
||||
|
@ -2,7 +2,7 @@
|
||||
|
||||
#include <shared_mutex>
|
||||
#include <Core/Block.h>
|
||||
#include <DataStreams/SizeLimits.h>
|
||||
#include <QueryPipeline/SizeLimits.h>
|
||||
#include <DataTypes/IDataType.h>
|
||||
#include <Interpreters/SetVariants.h>
|
||||
#include <Parsers/IAST.h>
|
||||
|
@ -1,11 +1,11 @@
|
||||
#include <Core/SortCursor.h>
|
||||
#include <Interpreters/SortedBlocksWriter.h>
|
||||
#include <Processors/QueryPipelineBuilder.h>
|
||||
#include <QueryPipeline/QueryPipelineBuilder.h>
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/Sources/SourceFromSingleChunk.h>
|
||||
#include <Processors/Merges/MergingSortedTransform.h>
|
||||
#include <DataStreams/TemporaryFileStream.h>
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
#include <Processors/Sources/TemporaryFileLazySource.h>
|
||||
#include <Formats/TemporaryFileStream.h>
|
||||
#include <Disks/IVolume.h>
|
||||
|
||||
|
||||
|
@ -6,8 +6,8 @@
|
||||
#include <Common/filesystemHelpers.h>
|
||||
#include <Core/Block.h>
|
||||
#include <Core/SortDescription.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <DataStreams/SizeLimits.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <QueryPipeline/SizeLimits.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
|
@ -7,7 +7,7 @@
|
||||
#include <Interpreters/IJoin.h>
|
||||
#include <Interpreters/join_common.h>
|
||||
#include <Interpreters/asof.h>
|
||||
#include <DataStreams/SizeLimits.h>
|
||||
#include <QueryPipeline/SizeLimits.h>
|
||||
#include <DataTypes/getLeastSupertype.h>
|
||||
#include <Storages/IStorage_fwd.h>
|
||||
#include <Common/Exception.h>
|
||||
|
@ -4,8 +4,8 @@
|
||||
#include <Functions/FunctionFactory.h>
|
||||
#include <Functions/FunctionHelpers.h>
|
||||
|
||||
#include <DataStreams/ShellCommandSource.h>
|
||||
#include <DataStreams/formatBlock.h>
|
||||
#include <Processors/Sources/ShellCommandSource.h>
|
||||
#include <Formats/formatBlock.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
|
@ -3,7 +3,7 @@
|
||||
#include <string>
|
||||
|
||||
#include <DataTypes/IDataType.h>
|
||||
#include <DataStreams/ShellCommandSource.h>
|
||||
#include <Processors/Sources/ShellCommandSource.h>
|
||||
#include <Interpreters/IExternalLoadable.h>
|
||||
|
||||
|
||||
|
@ -2,8 +2,8 @@
|
||||
|
||||
#include <IO/WriteHelpers.h>
|
||||
|
||||
#include <DataStreams/ShellCommandSource.h>
|
||||
#include <DataStreams/formatBlock.h>
|
||||
#include <Processors/Sources/ShellCommandSource.h>
|
||||
#include <Formats/formatBlock.h>
|
||||
|
||||
#include <Functions/FunctionFactory.h>
|
||||
#include <Functions/FunctionHelpers.h>
|
||||
|
@ -15,7 +15,7 @@
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <DataTypes/DataTypeNullable.h>
|
||||
#include <Processors/Sinks/EmptySink.h>
|
||||
#include <Processors/Pipe.h>
|
||||
#include <QueryPipeline/Pipe.h>
|
||||
#include <filesystem>
|
||||
|
||||
|
||||
|
@ -1,6 +1,6 @@
|
||||
#pragma once
|
||||
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Processors/Sources/SourceWithProgress.h>
|
||||
#include <Interpreters/Context_fwd.h>
|
||||
#include <Parsers/IAST_fwd.h>
|
||||
|
@ -9,7 +9,7 @@
|
||||
#include <IO/LimitReadBuffer.h>
|
||||
#include <IO/copyData.h>
|
||||
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Processors/Transforms/CountingTransform.h>
|
||||
#include <Processors/Transforms/getSourceFromASTInsertQuery.h>
|
||||
|
||||
@ -49,7 +49,6 @@
|
||||
#include <Common/ProfileEvents.h>
|
||||
|
||||
#include <Common/SensitiveDataMasker.h>
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
#include <IO/CompressionMethod.h>
|
||||
|
||||
#include <Processors/Transforms/LimitsCheckingTransform.h>
|
||||
|
@ -1,7 +1,7 @@
|
||||
#pragma once
|
||||
|
||||
#include <Core/QueryProcessingStage.h>
|
||||
#include <DataStreams/BlockIO.h>
|
||||
#include <QueryPipeline/BlockIO.h>
|
||||
#include <Interpreters/Context_fwd.h>
|
||||
#include <Formats/FormatSettings.h>
|
||||
|
||||
|
@ -4,8 +4,6 @@
|
||||
#include <Columns/ColumnNullable.h>
|
||||
#include <Columns/ColumnConst.h>
|
||||
|
||||
#include <DataStreams/materializeBlock.h>
|
||||
|
||||
#include <DataTypes/DataTypeLowCardinality.h>
|
||||
#include <DataTypes/DataTypeNullable.h>
|
||||
#include <DataTypes/DataTypesNumber.h>
|
||||
|
@ -18,9 +18,6 @@ Common/UInt128.h
|
||||
Core/Block.h
|
||||
Core/Defines.h
|
||||
Core/Settings.h
|
||||
DataStreams/PushingToViewsBlockOutputStream.cpp
|
||||
DataStreams/PushingToViewsBlockOutputStream.h
|
||||
DataStreams/copyData.cpp
|
||||
Databases/DatabasesCommon.cpp
|
||||
IO/WriteBufferValidUTF8.cpp
|
||||
Interpreters/InterpreterAlterQuery.cpp
|
||||
|
@ -1,6 +1,6 @@
|
||||
#include <Processors/Executors/CompletedPipelineExecutor.h>
|
||||
#include <Processors/Executors/PipelineExecutor.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Common/setThreadName.h>
|
||||
#include <base/scope_guard_safe.h>
|
||||
#include <iostream>
|
||||
|
@ -5,7 +5,7 @@
|
||||
#include <Common/setThreadName.h>
|
||||
#include <Common/MemoryTracker.h>
|
||||
#include <Processors/Executors/PipelineExecutor.h>
|
||||
#include <Processors/printPipeline.h>
|
||||
#include <QueryPipeline/printPipeline.h>
|
||||
#include <Processors/ISource.h>
|
||||
#include <Interpreters/ProcessList.h>
|
||||
#include <Interpreters/OpenTelemetrySpanLog.h>
|
||||
|
@ -3,7 +3,7 @@
|
||||
#include <Processors/Formats/LazyOutputFormat.h>
|
||||
#include <Processors/Transforms/AggregatingTransform.h>
|
||||
#include <Processors/Sources/NullSource.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
|
||||
#include <Common/setThreadName.h>
|
||||
#include <base/scope_guard_safe.h>
|
||||
@ -225,12 +225,12 @@ Block PullingAsyncPipelineExecutor::getExtremesBlock()
|
||||
return header.cloneWithColumns(extremes.detachColumns());
|
||||
}
|
||||
|
||||
BlockStreamProfileInfo & PullingAsyncPipelineExecutor::getProfileInfo()
|
||||
ProfileInfo & PullingAsyncPipelineExecutor::getProfileInfo()
|
||||
{
|
||||
if (lazy_format)
|
||||
return lazy_format->getProfileInfo();
|
||||
|
||||
static BlockStreamProfileInfo profile_info;
|
||||
static ProfileInfo profile_info;
|
||||
static std::once_flag flag;
|
||||
/// Calculate rows before limit here to avoid race.
|
||||
std::call_once(flag, []() { profile_info.getRowsBeforeLimit(); });
|
||||
|
@ -8,7 +8,7 @@ class QueryPipeline;
|
||||
class Block;
|
||||
class Chunk;
|
||||
class LazyOutputFormat;
|
||||
struct BlockStreamProfileInfo;
|
||||
struct ProfileInfo;
|
||||
|
||||
/// Asynchronous pulling executor for QueryPipeline.
|
||||
/// Always creates extra thread. If query is executed in single thread, use PullingPipelineExecutor.
|
||||
@ -44,7 +44,7 @@ public:
|
||||
Block getExtremesBlock();
|
||||
|
||||
/// Get query profile info.
|
||||
BlockStreamProfileInfo & getProfileInfo();
|
||||
ProfileInfo & getProfileInfo();
|
||||
|
||||
/// Internal executor data.
|
||||
struct Data;
|
||||
|
@ -1,7 +1,7 @@
|
||||
#include <Processors/Executors/PullingPipelineExecutor.h>
|
||||
#include <Processors/Executors/PipelineExecutor.h>
|
||||
#include <Processors/Formats/PullingOutputFormat.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <Processors/Transforms/AggregatingTransform.h>
|
||||
#include <Processors/Sources/NullSource.h>
|
||||
|
||||
@ -118,7 +118,7 @@ Block PullingPipelineExecutor::getExtremesBlock()
|
||||
return header.cloneWithColumns(extremes.detachColumns());
|
||||
}
|
||||
|
||||
BlockStreamProfileInfo & PullingPipelineExecutor::getProfileInfo()
|
||||
ProfileInfo & PullingPipelineExecutor::getProfileInfo()
|
||||
{
|
||||
return pulling_format->getProfileInfo();
|
||||
}
|
||||
|
@ -10,7 +10,7 @@ class Chunk;
|
||||
class QueryPipeline;
|
||||
class PipelineExecutor;
|
||||
class PullingOutputFormat;
|
||||
struct BlockStreamProfileInfo;
|
||||
struct ProfileInfo;
|
||||
|
||||
using PipelineExecutorPtr = std::shared_ptr<PipelineExecutor>;
|
||||
|
||||
@ -46,7 +46,7 @@ public:
|
||||
Block getExtremesBlock();
|
||||
|
||||
/// Get query profile info.
|
||||
BlockStreamProfileInfo & getProfileInfo();
|
||||
ProfileInfo & getProfileInfo();
|
||||
|
||||
private:
|
||||
std::atomic_bool has_data_flag = false;
|
||||
|
@ -1,7 +1,7 @@
|
||||
#include <Processors/Executors/PushingAsyncPipelineExecutor.h>
|
||||
#include <Processors/Executors/PipelineExecutor.h>
|
||||
#include <Processors/ISource.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <iostream>
|
||||
|
||||
#include <Common/setThreadName.h>
|
||||
|
@ -1,7 +1,7 @@
|
||||
#include <Processors/Executors/PushingPipelineExecutor.h>
|
||||
#include <Processors/Executors/PipelineExecutor.h>
|
||||
#include <Processors/ISource.h>
|
||||
#include <Processors/QueryPipeline.h>
|
||||
#include <QueryPipeline/QueryPipeline.h>
|
||||
#include <iostream>
|
||||
|
||||
|
||||
|
Some files were not shown because too many files have changed in this diff Show More
Loading…
Reference in New Issue
Block a user