diff --git a/dbms/src/DataStreams/CheckConstraintsBlockOutputStream.cpp b/dbms/src/DataStreams/CheckConstraintsBlockOutputStream.cpp index f771a5cf20c4b6911d5257f6043e0c281d9d30de..b9f4412c034385f52a0da38237b3e3e439baedaf 100644 --- a/dbms/src/DataStreams/CheckConstraintsBlockOutputStream.cpp +++ b/dbms/src/DataStreams/CheckConstraintsBlockOutputStream.cpp @@ -1,6 +1,7 @@ #include #include #include +#include #include #include #include diff --git a/dbms/src/Functions/FunctionsConversion.h b/dbms/src/Functions/FunctionsConversion.h index cfa70ecc205872ef1b0b61a7b1378c7d005e9a28..100737b43c7a16c17b7e28ccf36f5e8230980066 100644 --- a/dbms/src/Functions/FunctionsConversion.h +++ b/dbms/src/Functions/FunctionsConversion.h @@ -35,7 +35,6 @@ #include #include #include -#include #include #include #include diff --git a/dbms/src/Interpreters/ActionsVisitor.h b/dbms/src/Interpreters/ActionsVisitor.h index b850c6156558f1aaaad7554328cfc1c62309e47b..f6db551ff3322ef34b38b0f8ca9baa612bea27d4 100644 --- a/dbms/src/Interpreters/ActionsVisitor.h +++ b/dbms/src/Interpreters/ActionsVisitor.h @@ -2,7 +2,6 @@ #include #include -#include #include #include @@ -13,6 +12,9 @@ namespace DB class Context; class ASTFunction; +class ExpressionActions; +using ExpressionActionsPtr = std::shared_ptr; + /// The case of an explicit enumeration of values. SetPtr makeExplicitSet( const ASTFunction * node, const Block & sample_block, bool create_ordered_set, diff --git a/dbms/src/Interpreters/ExpressionAnalyzer.cpp b/dbms/src/Interpreters/ExpressionAnalyzer.cpp index c6c6f08f8156e12f00026aa615e6e2d4037c374a..56c999d06813c1422f0de5a85719ce3c4727074a 100644 --- a/dbms/src/Interpreters/ExpressionAnalyzer.cpp +++ b/dbms/src/Interpreters/ExpressionAnalyzer.cpp @@ -70,9 +70,51 @@ using LogAST = DebugASTLog; /// set to true to enable logs namespace ErrorCodes { extern const int UNKNOWN_IDENTIFIER; + extern const int ILLEGAL_PREWHERE; extern const int LOGICAL_ERROR; } +namespace +{ + +/// Check if there is an ignore function. It's used for disabling constant folding in query +/// predicates because some performance tests use ignore function as a non-optimize guard. +bool allowEarlyConstantFolding(const ExpressionActions & actions, const Context & context) +{ + if (!context.getSettingsRef().enable_early_constant_folding) + return false; + + for (auto & action : actions.getActions()) + { + if (action.type == action.APPLY_FUNCTION && action.function_base) + { + auto name = action.function_base->getName(); + if (name == "ignore") + return false; + } + } + return true; +} + +} + +bool sanitizeBlock(Block & block) +{ + for (auto & col : block) + { + if (!col.column) + { + if (isNotCreatable(col.type->getTypeId())) + return false; + col.column = col.type->createColumn(); + } + else if (isColumnConst(*col.column) && !col.column->empty()) + col.column = col.column->cloneEmpty(); + } + return true; +} + + ExpressionAnalyzer::ExpressionAnalyzer( const ASTPtr & query_, const SyntaxAnalyzerResultPtr & syntax_analyzer_result_, @@ -884,6 +926,13 @@ ExpressionActionsPtr ExpressionAnalyzer::getConstActions() return actions; } +ExpressionActionsPtr SelectQueryExpressionAnalyzer::simpleSelectActions() +{ + ExpressionActionsChain new_chain(context); + appendSelect(new_chain, false); + return new_chain.getLastActions(); +} + void SelectQueryExpressionAnalyzer::getAggregateInfo(Names & key_names, AggregateDescriptions & aggregates) const { for (const auto & name_and_type : aggregation_keys) @@ -892,4 +941,231 @@ void SelectQueryExpressionAnalyzer::getAggregateInfo(Names & key_names, Aggregat aggregates = aggregate_descriptions; } +ExpressionAnalysisResult::ExpressionAnalysisResult(const ASTSelectQuery & query, + SelectQueryExpressionAnalyzer & query_analyzer, + bool first_stage_, + bool second_stage_, + const Context & context, + const StoragePtr & storage, + bool only_types, + const FilterInfoPtr & filter_info_, + const Block & source_header) + : first_stage(first_stage_) + , second_stage(second_stage_) +{ + /// first_stage: Do I need to perform the first part of the pipeline - running on remote servers during distributed processing. + /// second_stage: Do I need to execute the second part of the pipeline - running on the initiating server during distributed processing. + + /** First we compose a chain of actions and remember the necessary steps from it. + * Regardless of from_stage and to_stage, we will compose a complete sequence of actions to perform optimization and + * throw out unnecessary columns based on the entire query. In unnecessary parts of the query, we will not execute subqueries. + */ + + bool finalized = false; + size_t where_step_num = 0; + + auto finalizeChain = [&](ExpressionActionsChain & chain) + { + if (!finalized) + { + chain.finalize(); + finalize(chain, context, where_step_num); + chain.clear(); + } + finalized = true; + }; + + { + ExpressionActionsChain chain(context); + Names additional_required_columns_after_prewhere; + + if (storage && (query.sample_size() || context.getSettingsRef().parallel_replicas_count > 1)) + { + Names columns_for_sampling = storage->getColumnsRequiredForSampling(); + additional_required_columns_after_prewhere.insert(additional_required_columns_after_prewhere.end(), + columns_for_sampling.begin(), columns_for_sampling.end()); + } + + if (storage && query.final()) + { + Names columns_for_final = storage->getColumnsRequiredForFinal(); + additional_required_columns_after_prewhere.insert(additional_required_columns_after_prewhere.end(), + columns_for_final.begin(), columns_for_final.end()); + } + + if (storage && filter_info_) + { + filter_info = filter_info_; + query_analyzer.appendPreliminaryFilter(chain, filter_info->actions, filter_info->column_name); + } + + if (query_analyzer.appendPrewhere(chain, !first_stage, additional_required_columns_after_prewhere)) + { + prewhere_info = std::make_shared( + chain.steps.front().actions, query.prewhere()->getColumnName()); + + if (allowEarlyConstantFolding(*prewhere_info->prewhere_actions, context)) + { + Block before_prewhere_sample = source_header; + if (sanitizeBlock(before_prewhere_sample)) + { + prewhere_info->prewhere_actions->execute(before_prewhere_sample); + auto & column_elem = before_prewhere_sample.getByName(query.prewhere()->getColumnName()); + /// If the filter column is a constant, record it. + if (column_elem.column) + prewhere_constant_filter_description = ConstantFilterDescription(*column_elem.column); + } + } + chain.addStep(); + } + + need_aggregate = query_analyzer.hasAggregation(); + + query_analyzer.appendArrayJoin(chain, only_types || !first_stage); + + if (query_analyzer.appendJoin(chain, only_types || !first_stage)) + { + before_join = chain.getLastActions(); + if (!hasJoin()) + throw Exception("No expected JOIN", ErrorCodes::LOGICAL_ERROR); + chain.addStep(); + } + + if (query_analyzer.appendWhere(chain, only_types || !first_stage)) + { + where_step_num = chain.steps.size() - 1; + before_where = chain.getLastActions(); + if (allowEarlyConstantFolding(*before_where, context)) + { + Block before_where_sample; + if (chain.steps.size() > 1) + before_where_sample = chain.steps[chain.steps.size() - 2].actions->getSampleBlock(); + else + before_where_sample = source_header; + if (sanitizeBlock(before_where_sample)) + { + before_where->execute(before_where_sample); + auto & column_elem = before_where_sample.getByName(query.where()->getColumnName()); + /// If the filter column is a constant, record it. + if (column_elem.column) + where_constant_filter_description = ConstantFilterDescription(*column_elem.column); + } + } + chain.addStep(); + } + + if (need_aggregate) + { + query_analyzer.appendGroupBy(chain, only_types || !first_stage); + query_analyzer.appendAggregateFunctionsArguments(chain, only_types || !first_stage); + before_aggregation = chain.getLastActions(); + + finalizeChain(chain); + + if (query_analyzer.appendHaving(chain, only_types || !second_stage)) + { + before_having = chain.getLastActions(); + chain.addStep(); + } + } + + bool has_stream_with_non_joned_rows = (before_join && before_join->getTableJoinAlgo()->hasStreamWithNonJoinedRows()); + optimize_read_in_order = + context.getSettingsRef().optimize_read_in_order + && storage && query.orderBy() + && !query_analyzer.hasAggregation() + && !query.final() + && !has_stream_with_non_joned_rows; + + /// If there is aggregation, we execute expressions in SELECT and ORDER BY on the initiating server, otherwise on the source servers. + query_analyzer.appendSelect(chain, only_types || (need_aggregate ? !second_stage : !first_stage)); + selected_columns = chain.getLastStep().required_output; + has_order_by = query_analyzer.appendOrderBy(chain, only_types || (need_aggregate ? !second_stage : !first_stage), optimize_read_in_order); + before_order_and_select = chain.getLastActions(); + chain.addStep(); + + if (query_analyzer.appendLimitBy(chain, only_types || !second_stage)) + { + before_limit_by = chain.getLastActions(); + chain.addStep(); + } + + query_analyzer.appendProjectResult(chain); + final_projection = chain.getLastActions(); + + finalizeChain(chain); + } + + /// Before executing WHERE and HAVING, remove the extra columns from the block (mostly the aggregation keys). + removeExtraColumns(); + + subqueries_for_sets = query_analyzer.getSubqueriesForSets(); + + checkActions(); +} + +void ExpressionAnalysisResult::finalize(const ExpressionActionsChain & chain, const Context & context_, size_t where_step_num) +{ + if (hasPrewhere()) + { + const ExpressionActionsChain::Step & step = chain.steps.at(0); + prewhere_info->remove_prewhere_column = step.can_remove_required_output.at(0); + + Names columns_to_remove; + for (size_t i = 1; i < step.required_output.size(); ++i) + { + if (step.can_remove_required_output[i]) + columns_to_remove.push_back(step.required_output[i]); + } + + if (!columns_to_remove.empty()) + { + auto columns = prewhere_info->prewhere_actions->getSampleBlock().getNamesAndTypesList(); + ExpressionActionsPtr actions = std::make_shared(columns, context_); + for (const auto & column : columns_to_remove) + actions->add(ExpressionAction::removeColumn(column)); + + prewhere_info->remove_columns_actions = std::move(actions); + } + + columns_to_remove_after_prewhere = std::move(columns_to_remove); + } + else if (hasFilter()) + { + /// Can't have prewhere and filter set simultaneously + filter_info->do_remove_column = chain.steps.at(0).can_remove_required_output.at(0); + } + if (hasWhere()) + remove_where_filter = chain.steps.at(where_step_num).can_remove_required_output.at(0); +} + +void ExpressionAnalysisResult::removeExtraColumns() +{ + if (hasFilter()) + filter_info->actions->prependProjectInput(); + if (hasWhere()) + before_where->prependProjectInput(); + if (hasHaving()) + before_having->prependProjectInput(); +} + +void ExpressionAnalysisResult::checkActions() +{ + /// Check that PREWHERE doesn't contain unusual actions. Unusual actions are that can change number of rows. + if (hasPrewhere()) + { + auto check_actions = [](const ExpressionActionsPtr & actions) + { + if (actions) + for (const auto & action : actions->getActions()) + if (action.type == ExpressionAction::Type::JOIN || action.type == ExpressionAction::Type::ARRAY_JOIN) + throw Exception("PREWHERE cannot contain ARRAY JOIN or JOIN action", ErrorCodes::ILLEGAL_PREWHERE); + }; + + check_actions(prewhere_info->prewhere_actions); + check_actions(prewhere_info->alias_actions); + check_actions(prewhere_info->remove_columns_actions); + } +} + } diff --git a/dbms/src/Interpreters/ExpressionAnalyzer.h b/dbms/src/Interpreters/ExpressionAnalyzer.h index 0b077901c669a2fc18374bbd3622ecfec7f58261..e762cdf4d33ce3b564e21301d9647035ecd058e2 100644 --- a/dbms/src/Interpreters/ExpressionAnalyzer.h +++ b/dbms/src/Interpreters/ExpressionAnalyzer.h @@ -2,11 +2,13 @@ #include #include +#include #include #include #include #include #include +#include namespace DB @@ -29,6 +31,9 @@ class ASTExpressionList; class ASTSelectQuery; struct ASTTablesInSelectQueryElement; +/// Create columns in block or return false if not possible +bool sanitizeBlock(Block & block); + /// ExpressionAnalyzer sources, intermediates and results. It splits data and logic, allows to test them separately. struct ExpressionAnalyzerData { @@ -156,10 +161,73 @@ protected: bool isRemoteStorage() const; }; +class SelectQueryExpressionAnalyzer; + +/// Result of SelectQueryExpressionAnalyzer: expressions for InterpreterSelectQuery +struct ExpressionAnalysisResult +{ + bool need_aggregate = false; + bool has_order_by = false; + + bool remove_where_filter = false; + bool optimize_read_in_order = false; + + ExpressionActionsPtr before_join; /// including JOIN + ExpressionActionsPtr before_where; + ExpressionActionsPtr before_aggregation; + ExpressionActionsPtr before_having; + ExpressionActionsPtr before_order_and_select; + ExpressionActionsPtr before_limit_by; + ExpressionActionsPtr final_projection; + + /// Columns from the SELECT list, before renaming them to aliases. + Names selected_columns; + + /// Columns will be removed after prewhere actions execution. + Names columns_to_remove_after_prewhere; + + /// Do I need to perform the first part of the pipeline - running on remote servers during distributed processing. + bool first_stage = false; + /// Do I need to execute the second part of the pipeline - running on the initiating server during distributed processing. + bool second_stage = false; + + SubqueriesForSets subqueries_for_sets; + PrewhereInfoPtr prewhere_info; + FilterInfoPtr filter_info; + ConstantFilterDescription prewhere_constant_filter_description; + ConstantFilterDescription where_constant_filter_description; + + ExpressionAnalysisResult() = default; + + ExpressionAnalysisResult( + const ASTSelectQuery & query, + SelectQueryExpressionAnalyzer & query_analyzer, + bool first_stage, + bool second_stage, + const Context & context, + const StoragePtr & storage, + bool only_types, + const FilterInfoPtr & filter_info, + const Block & source_header); + + bool hasFilter() const { return filter_info.get(); } + bool hasJoin() const { return before_join.get(); } + bool hasPrewhere() const { return prewhere_info.get(); } + bool hasWhere() const { return before_where.get(); } + bool hasHaving() const { return before_having.get(); } + bool hasLimitBy() const { return before_limit_by.get(); } + + void removeExtraColumns(); + void checkActions(); + void finalize(const ExpressionActionsChain & chain, const Context & context, size_t where_step_num); +}; + /// SelectQuery specific ExpressionAnalyzer part. class SelectQueryExpressionAnalyzer : public ExpressionAnalyzer { public: + friend struct ExpressionAnalysisResult; + SelectQueryExpressionAnalyzer( const ASTPtr & query_, const SyntaxAnalyzerResultPtr & syntax_analyzer_result_, @@ -185,37 +253,10 @@ public: /// Tables that will need to be sent to remote servers for distributed query processing. const Tables & getExternalTables() const { return external_tables; } - /** These methods allow you to build a chain of transformations over a block, that receives values in the desired sections of the query. - * - * Example usage: - * ExpressionActionsChain chain; - * analyzer.appendWhere(chain); - * chain.addStep(); - * analyzer.appendSelect(chain); - * analyzer.appendOrderBy(chain); - * chain.finalize(); - * - * If only_types = true set, does not execute subqueries in the relevant parts of the query. The actions got this way - * shouldn't be executed, they are only needed to get a list of columns with their types. - */ + ExpressionActionsPtr simpleSelectActions(); - /// Before aggregation: - bool appendArrayJoin(ExpressionActionsChain & chain, bool only_types); - bool appendJoin(ExpressionActionsChain & chain, bool only_types); - /// Add preliminary rows filtration. Actions are created in other expression analyzer to prevent any possible alias injection. - void appendPreliminaryFilter(ExpressionActionsChain & chain, ExpressionActionsPtr actions, String column_name); - /// remove_filter is set in ExpressionActionsChain::finalize(); - /// Columns in `additional_required_columns` will not be removed (they can be used for e.g. sampling or FINAL modifier). - bool appendPrewhere(ExpressionActionsChain & chain, bool only_types, const Names & additional_required_columns); - bool appendWhere(ExpressionActionsChain & chain, bool only_types); - bool appendGroupBy(ExpressionActionsChain & chain, bool only_types); - void appendAggregateFunctionsArguments(ExpressionActionsChain & chain, bool only_types); - - /// After aggregation: - bool appendHaving(ExpressionActionsChain & chain, bool only_types); + /// These appends are public only for tests void appendSelect(ExpressionActionsChain & chain, bool only_types); - bool appendOrderBy(ExpressionActionsChain & chain, bool only_types, bool optimize_read_in_order); - bool appendLimitBy(ExpressionActionsChain & chain, bool only_types); /// Deletes all columns except mentioned by SELECT, arranges the remaining columns and renames them to aliases. void appendProjectResult(ExpressionActionsChain & chain) const; @@ -244,6 +285,39 @@ private: SubqueryForSet & subquery_for_set) const; const ASTSelectQuery * getAggregatingQuery() const; + + /** These methods allow you to build a chain of transformations over a block, that receives values in the desired sections of the query. + * + * Example usage: + * ExpressionActionsChain chain; + * analyzer.appendWhere(chain); + * chain.addStep(); + * analyzer.appendSelect(chain); + * analyzer.appendOrderBy(chain); + * chain.finalize(); + * + * If only_types = true set, does not execute subqueries in the relevant parts of the query. The actions got this way + * shouldn't be executed, they are only needed to get a list of columns with their types. + */ + + /// Before aggregation: + bool appendArrayJoin(ExpressionActionsChain & chain, bool only_types); + bool appendJoin(ExpressionActionsChain & chain, bool only_types); + /// Add preliminary rows filtration. Actions are created in other expression analyzer to prevent any possible alias injection. + void appendPreliminaryFilter(ExpressionActionsChain & chain, ExpressionActionsPtr actions, String column_name); + /// remove_filter is set in ExpressionActionsChain::finalize(); + /// Columns in `additional_required_columns` will not be removed (they can be used for e.g. sampling or FINAL modifier). + bool appendPrewhere(ExpressionActionsChain & chain, bool only_types, const Names & additional_required_columns); + bool appendWhere(ExpressionActionsChain & chain, bool only_types); + bool appendGroupBy(ExpressionActionsChain & chain, bool only_types); + void appendAggregateFunctionsArguments(ExpressionActionsChain & chain, bool only_types); + + /// After aggregation: + bool appendHaving(ExpressionActionsChain & chain, bool only_types); + /// appendSelect + bool appendOrderBy(ExpressionActionsChain & chain, bool only_types, bool optimize_read_in_order); + bool appendLimitBy(ExpressionActionsChain & chain, bool only_types); + /// appendProjectResult }; } diff --git a/dbms/src/Interpreters/InterpreterCreateQuery.h b/dbms/src/Interpreters/InterpreterCreateQuery.h index ae8125b84d6cc15b5deaa4666e1bd016e79d23b7..0ff5d8c60e742b70dad3caeb0abea707a8d93738 100644 --- a/dbms/src/Interpreters/InterpreterCreateQuery.h +++ b/dbms/src/Interpreters/InterpreterCreateQuery.h @@ -6,6 +6,7 @@ #include #include #include +#include namespace DB diff --git a/dbms/src/Interpreters/InterpreterSelectQuery.cpp b/dbms/src/Interpreters/InterpreterSelectQuery.cpp index ac7ea12d898b03ecd9fa9c39ca6c4859d79d0dc9..cf2ecf360567fa1120eff2e9530d3b50c0f102d8 100644 --- a/dbms/src/Interpreters/InterpreterSelectQuery.cpp +++ b/dbms/src/Interpreters/InterpreterSelectQuery.cpp @@ -154,9 +154,7 @@ String InterpreterSelectQuery::generateFilterActions(ExpressionActionsPtr & acti /// Using separate expression analyzer to prevent any possible alias injection auto syntax_result = SyntaxAnalyzer(*context).analyze(query_ast, storage->getColumns().getAllPhysical()); SelectQueryExpressionAnalyzer analyzer(query_ast, syntax_result, *context); - ExpressionActionsChain new_chain(*context); - analyzer.appendSelect(new_chain, false); - actions = new_chain.getLastActions(); + actions = analyzer.simpleSelectActions(); return expr_list->children.at(0)->getColumnName(); } @@ -212,22 +210,6 @@ static Context getSubqueryContext(const Context & context) return subquery_context; } -static bool sanitizeBlock(Block & block) -{ - for (auto & col : block) - { - if (!col.column) - { - if (isNotCreatable(col.type->getTypeId())) - return false; - col.column = col.type->createColumn(); - } - else if (isColumnConst(*col.column) && !col.column->empty()) - col.column = col.column->cloneEmpty(); - } - return true; -} - InterpreterSelectQuery::InterpreterSelectQuery( const ASTPtr & query_ptr_, const Context & context_, @@ -556,11 +538,18 @@ Block InterpreterSelectQuery::getSampleBlockImpl(bool try_move_to_prewhere) if (storage && !options.only_analyze) from_stage = storage->getQueryProcessingStage(*context); - analysis_result = analyzeExpressions( + /// Do I need to perform the first part of the pipeline - running on remote servers during distributed processing. + bool first_stage = from_stage < QueryProcessingStage::WithMergeableState + && options.to_stage >= QueryProcessingStage::WithMergeableState; + /// Do I need to execute the second part of the pipeline - running on the initiating server during distributed processing. + bool second_stage = from_stage <= QueryProcessingStage::WithMergeableState + && options.to_stage > QueryProcessingStage::WithMergeableState; + + analysis_result = ExpressionAnalysisResult( getSelectQuery(), *query_analyzer, - from_stage, - options.to_stage, + first_stage, + second_stage, *context, storage, options.only_analyze, @@ -616,253 +605,6 @@ Block InterpreterSelectQuery::getSampleBlockImpl(bool try_move_to_prewhere) return analysis_result.final_projection->getSampleBlock(); } -/// Check if there is an ignore function. It's used for disabling constant folding in query -/// predicates because some performance tests use ignore function as a non-optimize guard. -static bool allowEarlyConstantFolding(const ExpressionActions & actions, const Context & context) -{ - if (!context.getSettingsRef().enable_early_constant_folding) - return false; - - for (auto & action : actions.getActions()) - { - if (action.type == action.APPLY_FUNCTION && action.function_base) - { - auto name = action.function_base->getName(); - if (name == "ignore") - return false; - } - } - return true; -} - -InterpreterSelectQuery::AnalysisResult -InterpreterSelectQuery::analyzeExpressions( - const ASTSelectQuery & query, - SelectQueryExpressionAnalyzer & query_analyzer, - QueryProcessingStage::Enum from_stage, - QueryProcessingStage::Enum to_stage, - const Context & context, - const StoragePtr & storage, - bool only_types, - const FilterInfoPtr & filter_info, - const Block & source_header) -{ - AnalysisResult res; - - /// Do I need to perform the first part of the pipeline - running on remote servers during distributed processing. - res.first_stage = from_stage < QueryProcessingStage::WithMergeableState - && to_stage >= QueryProcessingStage::WithMergeableState; - /// Do I need to execute the second part of the pipeline - running on the initiating server during distributed processing. - res.second_stage = from_stage <= QueryProcessingStage::WithMergeableState - && to_stage > QueryProcessingStage::WithMergeableState; - - /** First we compose a chain of actions and remember the necessary steps from it. - * Regardless of from_stage and to_stage, we will compose a complete sequence of actions to perform optimization and - * throw out unnecessary columns based on the entire query. In unnecessary parts of the query, we will not execute subqueries. - */ - - bool has_filter = false; - bool has_prewhere = false; - bool has_where = false; - size_t where_step_num; - - auto finalizeChain = [&](ExpressionActionsChain & chain) - { - chain.finalize(); - - if (has_prewhere) - { - const ExpressionActionsChain::Step & step = chain.steps.at(0); - res.prewhere_info->remove_prewhere_column = step.can_remove_required_output.at(0); - - Names columns_to_remove; - for (size_t i = 1; i < step.required_output.size(); ++i) - { - if (step.can_remove_required_output[i]) - columns_to_remove.push_back(step.required_output[i]); - } - - if (!columns_to_remove.empty()) - { - auto columns = res.prewhere_info->prewhere_actions->getSampleBlock().getNamesAndTypesList(); - ExpressionActionsPtr actions = std::make_shared(columns, context); - for (const auto & column : columns_to_remove) - actions->add(ExpressionAction::removeColumn(column)); - - res.prewhere_info->remove_columns_actions = std::move(actions); - } - - res.columns_to_remove_after_prewhere = std::move(columns_to_remove); - } - else if (has_filter) - { - /// Can't have prewhere and filter set simultaneously - res.filter_info->do_remove_column = chain.steps.at(0).can_remove_required_output.at(0); - } - if (has_where) - res.remove_where_filter = chain.steps.at(where_step_num).can_remove_required_output.at(0); - - has_filter = has_prewhere = has_where = false; - - chain.clear(); - }; - - { - ExpressionActionsChain chain(context); - Names additional_required_columns_after_prewhere; - - if (storage && (query.sample_size() || context.getSettingsRef().parallel_replicas_count > 1)) - { - Names columns_for_sampling = storage->getColumnsRequiredForSampling(); - additional_required_columns_after_prewhere.insert(additional_required_columns_after_prewhere.end(), - columns_for_sampling.begin(), columns_for_sampling.end()); - } - - if (storage && query.final()) - { - Names columns_for_final = storage->getColumnsRequiredForFinal(); - additional_required_columns_after_prewhere.insert(additional_required_columns_after_prewhere.end(), - columns_for_final.begin(), columns_for_final.end()); - } - - if (storage && filter_info) - { - has_filter = true; - res.filter_info = filter_info; - query_analyzer.appendPreliminaryFilter(chain, filter_info->actions, filter_info->column_name); - } - - if (query_analyzer.appendPrewhere(chain, !res.first_stage, additional_required_columns_after_prewhere)) - { - has_prewhere = true; - - res.prewhere_info = std::make_shared( - chain.steps.front().actions, query.prewhere()->getColumnName()); - - if (allowEarlyConstantFolding(*res.prewhere_info->prewhere_actions, context)) - { - Block before_prewhere_sample = source_header; - if (sanitizeBlock(before_prewhere_sample)) - { - res.prewhere_info->prewhere_actions->execute(before_prewhere_sample); - auto & column_elem = before_prewhere_sample.getByName(query.prewhere()->getColumnName()); - /// If the filter column is a constant, record it. - if (column_elem.column) - res.prewhere_constant_filter_description = ConstantFilterDescription(*column_elem.column); - } - } - chain.addStep(); - } - - res.need_aggregate = query_analyzer.hasAggregation(); - - query_analyzer.appendArrayJoin(chain, only_types || !res.first_stage); - - if (query_analyzer.appendJoin(chain, only_types || !res.first_stage)) - { - res.before_join = chain.getLastActions(); - if (!res.hasJoin()) - throw Exception("No expected JOIN", ErrorCodes::LOGICAL_ERROR); - chain.addStep(); - } - - if (query_analyzer.appendWhere(chain, only_types || !res.first_stage)) - { - where_step_num = chain.steps.size() - 1; - has_where = res.has_where = true; - res.before_where = chain.getLastActions(); - if (allowEarlyConstantFolding(*res.before_where, context)) - { - Block before_where_sample; - if (chain.steps.size() > 1) - before_where_sample = chain.steps[chain.steps.size() - 2].actions->getSampleBlock(); - else - before_where_sample = source_header; - if (sanitizeBlock(before_where_sample)) - { - res.before_where->execute(before_where_sample); - auto & column_elem = before_where_sample.getByName(query.where()->getColumnName()); - /// If the filter column is a constant, record it. - if (column_elem.column) - res.where_constant_filter_description = ConstantFilterDescription(*column_elem.column); - } - } - chain.addStep(); - } - - if (res.need_aggregate) - { - query_analyzer.appendGroupBy(chain, only_types || !res.first_stage); - query_analyzer.appendAggregateFunctionsArguments(chain, only_types || !res.first_stage); - res.before_aggregation = chain.getLastActions(); - - finalizeChain(chain); - - if (query_analyzer.appendHaving(chain, only_types || !res.second_stage)) - { - res.has_having = true; - res.before_having = chain.getLastActions(); - chain.addStep(); - } - } - - bool has_stream_with_non_joned_rows = (res.before_join && res.before_join->getTableJoinAlgo()->hasStreamWithNonJoinedRows()); - res.optimize_read_in_order = - context.getSettingsRef().optimize_read_in_order - && storage && query.orderBy() - && !query_analyzer.hasAggregation() - && !query.final() - && !has_stream_with_non_joned_rows; - - /// If there is aggregation, we execute expressions in SELECT and ORDER BY on the initiating server, otherwise on the source servers. - query_analyzer.appendSelect(chain, only_types || (res.need_aggregate ? !res.second_stage : !res.first_stage)); - res.selected_columns = chain.getLastStep().required_output; - res.has_order_by = query_analyzer.appendOrderBy(chain, only_types || (res.need_aggregate ? !res.second_stage : !res.first_stage), res.optimize_read_in_order); - res.before_order_and_select = chain.getLastActions(); - chain.addStep(); - - if (query_analyzer.appendLimitBy(chain, only_types || !res.second_stage)) - { - res.has_limit_by = true; - res.before_limit_by = chain.getLastActions(); - chain.addStep(); - } - - query_analyzer.appendProjectResult(chain); - res.final_projection = chain.getLastActions(); - - finalizeChain(chain); - } - - /// Before executing WHERE and HAVING, remove the extra columns from the block (mostly the aggregation keys). - if (res.filter_info) - res.filter_info->actions->prependProjectInput(); - if (res.has_where) - res.before_where->prependProjectInput(); - if (res.has_having) - res.before_having->prependProjectInput(); - - res.subqueries_for_sets = query_analyzer.getSubqueriesForSets(); - - /// Check that PREWHERE doesn't contain unusual actions. Unusual actions are that can change number of rows. - if (res.prewhere_info) - { - auto check_actions = [](const ExpressionActionsPtr & actions) - { - if (actions) - for (const auto & action : actions->getActions()) - if (action.type == ExpressionAction::Type::JOIN || action.type == ExpressionAction::Type::ARRAY_JOIN) - throw Exception("PREWHERE cannot contain ARRAY JOIN or JOIN action", ErrorCodes::ILLEGAL_PREWHERE); - }; - - check_actions(res.prewhere_info->prewhere_actions); - check_actions(res.prewhere_info->alias_actions); - check_actions(res.prewhere_info->remove_columns_actions); - } - - return res; -} - static Field getWithFillFieldValue(const ASTPtr & node, const Context & context) { const auto & [field, type] = evaluateConstantExpression(node, context); @@ -1094,7 +836,7 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS if (expressions.first_stage) { - if (expressions.filter_info) + if (expressions.hasFilter()) { if constexpr (pipeline_with_processors) { @@ -1176,7 +918,7 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS } } - if (expressions.has_where) + if (expressions.hasWhere()) executeWhere(pipeline, expressions.before_where, expressions.remove_where_filter); if (expressions.need_aggregate) @@ -1192,7 +934,7 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS * but there is an ORDER or LIMIT, * then we will perform the preliminary sorting and LIMIT on the remote server. */ - if (!expressions.second_stage && !expressions.need_aggregate && !expressions.has_having) + if (!expressions.second_stage && !expressions.need_aggregate && !expressions.hasHaving()) { if (expressions.has_order_by) executeOrder(pipeline, query_info.input_sorting_info); @@ -1200,7 +942,7 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS if (expressions.has_order_by && query.limitLength()) executeDistinct(pipeline, false, expressions.selected_columns); - if (expressions.has_limit_by) + if (expressions.hasLimitBy()) { executeExpression(pipeline, expressions.before_limit_by); executeLimitBy(pipeline); @@ -1230,7 +972,7 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS if (query.group_by_with_totals) { bool final = !query.group_by_with_rollup && !query.group_by_with_cube; - executeTotalsAndHaving(pipeline, expressions.has_having, expressions.before_having, aggregate_overflow_row, final); + executeTotalsAndHaving(pipeline, expressions.hasHaving(), expressions.before_having, aggregate_overflow_row, final); } if (query.group_by_with_rollup) @@ -1238,14 +980,14 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS else if (query.group_by_with_cube) executeRollupOrCube(pipeline, Modificator::CUBE); - if ((query.group_by_with_rollup || query.group_by_with_cube) && expressions.has_having) + if ((query.group_by_with_rollup || query.group_by_with_cube) && expressions.hasHaving()) { if (query.group_by_with_totals) throw Exception("WITH TOTALS and WITH ROLLUP or CUBE are not supported together in presence of HAVING", ErrorCodes::NOT_IMPLEMENTED); executeHaving(pipeline, expressions.before_having); } } - else if (expressions.has_having) + else if (expressions.hasHaving()) executeHaving(pipeline, expressions.before_having); executeExpression(pipeline, expressions.before_order_and_select); @@ -1273,7 +1015,8 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS /** Optimization - if there are several sources and there is LIMIT, then first apply the preliminary LIMIT, * limiting the number of rows in each up to `offset + limit`. */ - if (query.limitLength() && !query.limit_with_ties && pipeline.hasMoreThanOneStream() && !query.distinct && !expressions.has_limit_by && !settings.extremes) + if (query.limitLength() && !query.limit_with_ties && pipeline.hasMoreThanOneStream() && + !query.distinct && !expressions.hasLimitBy() && !settings.extremes) { executePreLimit(pipeline); } @@ -1298,7 +1041,7 @@ void InterpreterSelectQuery::executeImpl(TPipeline & pipeline, const BlockInputS if (need_second_distinct_pass) executeDistinct(pipeline, false, expressions.selected_columns); - if (expressions.has_limit_by) + if (expressions.hasLimitBy()) { executeExpression(pipeline, expressions.before_limit_by); executeLimitBy(pipeline); diff --git a/dbms/src/Interpreters/InterpreterSelectQuery.h b/dbms/src/Interpreters/InterpreterSelectQuery.h index 77a6f1ca3acc0b35334dde685732f455831bea8f..0e4e3256332b3cf9a965ee15d4176d9a70a301b8 100644 --- a/dbms/src/Interpreters/InterpreterSelectQuery.h +++ b/dbms/src/Interpreters/InterpreterSelectQuery.h @@ -152,55 +152,6 @@ private: template void executeImpl(TPipeline & pipeline, const BlockInputStreamPtr & prepared_input, std::optional prepared_pipe, QueryPipeline & save_context_and_storage); - struct AnalysisResult - { - bool hasJoin() const { return before_join.get(); } - bool has_where = false; - bool need_aggregate = false; - bool has_having = false; - bool has_order_by = false; - bool has_limit_by = false; - - bool remove_where_filter = false; - bool optimize_read_in_order = false; - - ExpressionActionsPtr before_join; /// including JOIN - ExpressionActionsPtr before_where; - ExpressionActionsPtr before_aggregation; - ExpressionActionsPtr before_having; - ExpressionActionsPtr before_order_and_select; - ExpressionActionsPtr before_limit_by; - ExpressionActionsPtr final_projection; - - /// Columns from the SELECT list, before renaming them to aliases. - Names selected_columns; - - /// Columns will be removed after prewhere actions execution. - Names columns_to_remove_after_prewhere; - - /// Do I need to perform the first part of the pipeline - running on remote servers during distributed processing. - bool first_stage = false; - /// Do I need to execute the second part of the pipeline - running on the initiating server during distributed processing. - bool second_stage = false; - - SubqueriesForSets subqueries_for_sets; - PrewhereInfoPtr prewhere_info; - FilterInfoPtr filter_info; - ConstantFilterDescription prewhere_constant_filter_description; - ConstantFilterDescription where_constant_filter_description; - }; - - static AnalysisResult analyzeExpressions( - const ASTSelectQuery & query, - SelectQueryExpressionAnalyzer & query_analyzer, - QueryProcessingStage::Enum from_stage, - QueryProcessingStage::Enum to_stage, - const Context & context, - const StoragePtr & storage, - bool only_types, - const FilterInfoPtr & filter_info, - const Block & source_header); - /** From which table to read. With JOIN, the "left" table is returned. */ static void getDatabaseAndTableNames(const ASTSelectQuery & query, String & database_name, String & table_name, const Context & context); @@ -284,7 +235,7 @@ private: SelectQueryInfo query_info; /// Is calculated in getSampleBlock. Is used later in readImpl. - AnalysisResult analysis_result; + ExpressionAnalysisResult analysis_result; FilterInfoPtr filter_info; QueryProcessingStage::Enum from_stage = QueryProcessingStage::FetchColumns; diff --git a/dbms/src/Interpreters/SubqueryForSet.cpp b/dbms/src/Interpreters/SubqueryForSet.cpp index 5a2a06cc4113cd1e4c1e34d168fdd4b3fa71b736..47de516d1541e70b1c5ff9cca80561c678737d8e 100644 --- a/dbms/src/Interpreters/SubqueryForSet.cpp +++ b/dbms/src/Interpreters/SubqueryForSet.cpp @@ -2,6 +2,7 @@ #include #include #include +#include #include namespace DB diff --git a/dbms/src/Interpreters/SubqueryForSet.h b/dbms/src/Interpreters/SubqueryForSet.h index aa510faefbc53368e03b4bf91c417a387e2df182..3463f708a46ebbebc74bcbbd8764437208e6cb9f 100644 --- a/dbms/src/Interpreters/SubqueryForSet.h +++ b/dbms/src/Interpreters/SubqueryForSet.h @@ -1,16 +1,18 @@ #pragma once +#include +#include #include #include #include -#include namespace DB { class InterpreterSelectWithUnionQuery; - +class ExpressionActions; +using ExpressionActionsPtr = std::shared_ptr; /// Information on what to do when executing a subquery in the [GLOBAL] IN/JOIN section. struct SubqueryForSet diff --git a/dbms/src/Interpreters/interpretSubquery.cpp b/dbms/src/Interpreters/interpretSubquery.cpp index 82545d4b3be8177c5710e15254268413bad67f7b..e06d7f159ac9b4490160cd8540d91cf45c2ad5a8 100644 --- a/dbms/src/Interpreters/interpretSubquery.cpp +++ b/dbms/src/Interpreters/interpretSubquery.cpp @@ -8,6 +8,7 @@ #include #include #include +#include #include #include diff --git a/dbms/src/Processors/Formats/Impl/ConstantExpressionTemplate.h b/dbms/src/Processors/Formats/Impl/ConstantExpressionTemplate.h index bbac2e2a9998ea4ca613b799cca5816683948adf..931b05673c6bc45e7afea8f6a77b6ea21bc3a1fa 100644 --- a/dbms/src/Processors/Formats/Impl/ConstantExpressionTemplate.h +++ b/dbms/src/Processors/Formats/Impl/ConstantExpressionTemplate.h @@ -1,7 +1,6 @@ #pragma once #include -#include #include #include @@ -12,6 +11,9 @@ struct LiteralInfo; using LiteralsInfo = std::vector; struct SpecialParserType; +class ExpressionActions; +using ExpressionActionsPtr = std::shared_ptr; + /// Deduces template of an expression by replacing literals with dummy columns. /// It allows to parse and evaluate similar expressions without using heavy IParsers and ExpressionAnalyzer. /// Using ConstantExpressionTemplate for one expression is slower then evaluateConstantExpression(...), diff --git a/dbms/src/Processors/Transforms/CreatingSetsTransform.h b/dbms/src/Processors/Transforms/CreatingSetsTransform.h index 00f64440393144b4da66c9d126a64b4d46fc4e03..aeb7a43b61bb9df027d12a685a369e9a519683c2 100644 --- a/dbms/src/Processors/Transforms/CreatingSetsTransform.h +++ b/dbms/src/Processors/Transforms/CreatingSetsTransform.h @@ -1,11 +1,14 @@ #pragma once +#include #include #include #include +#include namespace DB { +class QueryStatus; struct Progress; using ProgressCallback = std::function; diff --git a/dbms/src/Processors/Transforms/ExpressionTransform.cpp b/dbms/src/Processors/Transforms/ExpressionTransform.cpp index 9bd4ba89db6e72c813bdc66ae8a7eb70bed25fb9..7ae2eafa0c6499674336d92430bc8d51d88dea99 100644 --- a/dbms/src/Processors/Transforms/ExpressionTransform.cpp +++ b/dbms/src/Processors/Transforms/ExpressionTransform.cpp @@ -1,5 +1,6 @@ #include #include +#include namespace DB { diff --git a/dbms/src/Processors/Transforms/InflatingExpressionTransform.cpp b/dbms/src/Processors/Transforms/InflatingExpressionTransform.cpp index 6653aa0c5c795693d0401e161ac5028bb12aea8a..837922533905bd149965b5c4715ea55f42d28743 100644 --- a/dbms/src/Processors/Transforms/InflatingExpressionTransform.cpp +++ b/dbms/src/Processors/Transforms/InflatingExpressionTransform.cpp @@ -1,5 +1,6 @@ #include #include +#include namespace DB { diff --git a/dbms/src/Storages/Distributed/DistributedBlockOutputStream.cpp b/dbms/src/Storages/Distributed/DistributedBlockOutputStream.cpp index 348cb74112397862c49e89b934c98df1239c902a..505bc1400438924aa953801fbf95f1eea3d6f333 100644 --- a/dbms/src/Storages/Distributed/DistributedBlockOutputStream.cpp +++ b/dbms/src/Storages/Distributed/DistributedBlockOutputStream.cpp @@ -14,6 +14,7 @@ #include #include #include +#include #include #include diff --git a/dbms/src/Storages/IStorage.cpp b/dbms/src/Storages/IStorage.cpp index 332ecbc8681db7a687242c3c3ca9f0946fb57047..1192107ab32199a75346a2437ae80af1cfe5bb92 100644 --- a/dbms/src/Storages/IStorage.cpp +++ b/dbms/src/Storages/IStorage.cpp @@ -3,6 +3,7 @@ #include #include #include +#include #include #include diff --git a/dbms/src/Storages/IStorage.h b/dbms/src/Storages/IStorage.h index 29cdc0661cd51e1d0a6b2e7e44208800d4d99a7b..65daaa6d77c049556e33030157f50885e4b55a4f 100644 --- a/dbms/src/Storages/IStorage.h +++ b/dbms/src/Storages/IStorage.h @@ -52,6 +52,9 @@ using Processors = std::vector; class Pipe; using Pipes = std::vector; +class StoragePolicy; +using StoragePolicyPtr = std::shared_ptr; + struct ColumnSize { size_t marks = 0; diff --git a/dbms/src/Storages/Kafka/StorageKafka.h b/dbms/src/Storages/Kafka/StorageKafka.h index bf710f58202c413eeb2ad2d05bd2a0d6f84eb782..8f00c111ee4f3729900d6a17a471f29fa2c7ea22 100644 --- a/dbms/src/Storages/Kafka/StorageKafka.h +++ b/dbms/src/Storages/Kafka/StorageKafka.h @@ -3,6 +3,7 @@ #include #include #include +#include #include #include diff --git a/dbms/src/Storages/MergeTree/KeyCondition.h b/dbms/src/Storages/MergeTree/KeyCondition.h index 1971191514c03ee4182a22b40f5c8adc039ccbb6..fd1d11c0ec80213961d6a9643278939fb4bf12f1 100644 --- a/dbms/src/Storages/MergeTree/KeyCondition.h +++ b/dbms/src/Storages/MergeTree/KeyCondition.h @@ -4,7 +4,6 @@ #include #include -#include #include #include #include @@ -19,6 +18,9 @@ namespace DB class IFunction; using FunctionBasePtr = std::shared_ptr; +class ExpressionActions; +using ExpressionActionsPtr = std::shared_ptr; + /** Range with open or closed ends; possibly unbounded. */ struct Range diff --git a/dbms/src/Storages/MergeTree/MergeTreeData.h b/dbms/src/Storages/MergeTree/MergeTreeData.h index ab5644749eed1247fd7ea63ce4815a87d624e8e5..bfb478c7751e9c1403b81281e31f974082785054 100644 --- a/dbms/src/Storages/MergeTree/MergeTreeData.h +++ b/dbms/src/Storages/MergeTree/MergeTreeData.h @@ -2,7 +2,6 @@ #include #include -#include #include #include #include @@ -35,6 +34,9 @@ class MergeListEntry; class AlterCommands; class MergeTreePartsMover; +class ExpressionActions; +using ExpressionActionsPtr = std::shared_ptr; + namespace ErrorCodes { extern const int LOGICAL_ERROR; diff --git a/dbms/src/Storages/MergeTree/MergeTreeRangeReader.h b/dbms/src/Storages/MergeTree/MergeTreeRangeReader.h index 345f537d2aa116ea63dccd5ab417ea50eeaf7b14..6e06768fe8672a993054708c820b82103813021a 100644 --- a/dbms/src/Storages/MergeTree/MergeTreeRangeReader.h +++ b/dbms/src/Storages/MergeTree/MergeTreeRangeReader.h @@ -1,7 +1,6 @@ #pragma once #include #include -#include #include namespace DB diff --git a/dbms/src/Storages/StorageDistributed.h b/dbms/src/Storages/StorageDistributed.h index a3fd6f3c6e11e0d4b05216d7d9d5ba368d0f1477..6b53ce33792c998c980b7713162a7acb9cb93927 100644 --- a/dbms/src/Storages/StorageDistributed.h +++ b/dbms/src/Storages/StorageDistributed.h @@ -8,7 +8,6 @@ #include #include #include -#include #include #include #include @@ -23,6 +22,8 @@ class StorageDistributedDirectoryMonitor; class Volume; using VolumePtr = std::shared_ptr; +class ExpressionActions; +using ExpressionActionsPtr = std::shared_ptr; /** A distributed table that resides on multiple servers. * Uses data from the specified database and tables on each server. diff --git a/dbms/src/Storages/StorageJoin.cpp b/dbms/src/Storages/StorageJoin.cpp index 9288f29e58cfb47ebacdb023ede39a8bb237adaf..9fc791aab2d00e02e055375864f43cd9e3feaaf7 100644 --- a/dbms/src/Storages/StorageJoin.cpp +++ b/dbms/src/Storages/StorageJoin.cpp @@ -1,6 +1,7 @@ #include #include #include +#include #include #include #include diff --git a/dbms/src/Storages/StorageMerge.h b/dbms/src/Storages/StorageMerge.h index c773ff3ae157d600a59474e5d766e99bcc6029cf..35fb8db24380c7779de9a176dcfc39cc0ce52d24 100644 --- a/dbms/src/Storages/StorageMerge.h +++ b/dbms/src/Storages/StorageMerge.h @@ -4,6 +4,7 @@ #include #include +#include namespace DB diff --git a/dbms/src/Storages/StorageMySQL.h b/dbms/src/Storages/StorageMySQL.h index 574221377dc593004f9e7f4e23d575a588cc3a4f..03563b233e145d0aadb6d6b3871d4e2d3dc092c2 100644 --- a/dbms/src/Storages/StorageMySQL.h +++ b/dbms/src/Storages/StorageMySQL.h @@ -6,6 +6,7 @@ #include #include +#include #include diff --git a/dbms/src/Storages/StorageNull.cpp b/dbms/src/Storages/StorageNull.cpp index bbb620132c866133423b5f54eee2df6a4836d53d..c01f463400db97f37292d04262bc7572466d7207 100644 --- a/dbms/src/Storages/StorageNull.cpp +++ b/dbms/src/Storages/StorageNull.cpp @@ -3,6 +3,7 @@ #include #include +#include #include #include diff --git a/dbms/src/Storages/registerStorages.h b/dbms/src/Storages/registerStorages.h index b88b2666a8f067bdf860c03313b6755d83781c8a..c6decff58761ca8424549c0e1a196bc9d66dacf7 100644 --- a/dbms/src/Storages/registerStorages.h +++ b/dbms/src/Storages/registerStorages.h @@ -1,5 +1,6 @@ #pragma once #include +#include "config_core.h" namespace DB { diff --git a/dbms/src/TableFunctions/TableFunctionNumbers.cpp b/dbms/src/TableFunctions/TableFunctionNumbers.cpp index 9b00eb600e8b14058899e485cafdfb5a491821e6..615a54dd1b483f8a02a2d5c6f604f94036997929 100644 --- a/dbms/src/TableFunctions/TableFunctionNumbers.cpp +++ b/dbms/src/TableFunctions/TableFunctionNumbers.cpp @@ -7,6 +7,7 @@ #include #include #include +#include #include "registerTableFunctions.h" diff --git a/dbms/src/TableFunctions/TableFunctionS3.cpp b/dbms/src/TableFunctions/TableFunctionS3.cpp index 019c3ca4f51bd8f342543d1c4b20baf52eb7a707..34a1607178fc483d0835735e9a4eaae7082e060e 100644 --- a/dbms/src/TableFunctions/TableFunctionS3.cpp +++ b/dbms/src/TableFunctions/TableFunctionS3.cpp @@ -6,6 +6,7 @@ #include #include #include +#include #include #include #include diff --git a/dbms/src/TableFunctions/TableFunctionValues.cpp b/dbms/src/TableFunctions/TableFunctionValues.cpp index 96e39f434d808bddacac7fd9926d6f79b9b10a4f..b30867949e9a5b9bb39b41828d2baab1d340a6b8 100644 --- a/dbms/src/TableFunctions/TableFunctionValues.cpp +++ b/dbms/src/TableFunctions/TableFunctionValues.cpp @@ -17,6 +17,7 @@ #include #include #include +#include #include "registerTableFunctions.h" diff --git a/dbms/src/TableFunctions/parseColumnsListForTableFunction.cpp b/dbms/src/TableFunctions/parseColumnsListForTableFunction.cpp index 41aaec9d1fbeb73f179bbba38569e3a4a6ea5e45..7f94236f2391b07e5178c930c3734a9098a2a244 100644 --- a/dbms/src/TableFunctions/parseColumnsListForTableFunction.cpp +++ b/dbms/src/TableFunctions/parseColumnsListForTableFunction.cpp @@ -1,6 +1,7 @@ #include #include #include +#include #include