Merge pull request #36055 from helifu/master2

Don't materialize external tables if it's explain statement
This commit is contained in:
Kruglov Pavel 2022-04-19 20:26:27 +02:00 committed by GitHub
commit 894b1b163e
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
5 changed files with 28 additions and 8 deletions

View File

@ -210,6 +210,7 @@ ExpressionAnalyzer::ExpressionAnalyzer(
ContextPtr context_,
size_t subquery_depth_,
bool do_global,
bool is_explain,
SubqueriesForSets subqueries_for_sets_,
PreparedSets prepared_sets_)
: WithContext(context_)
@ -223,7 +224,7 @@ ExpressionAnalyzer::ExpressionAnalyzer(
/// external_tables, subqueries_for_sets for global subqueries.
/// Replaces global subqueries with the generated names of temporary tables that will be sent to remote servers.
initGlobalSubqueriesAndExternalTables(do_global);
initGlobalSubqueriesAndExternalTables(do_global, is_explain);
auto temp_actions = std::make_shared<ActionsDAG>(sourceColumns());
columns_after_array_join = getColumnsAfterArrayJoin(temp_actions, sourceColumns());
@ -391,12 +392,12 @@ void ExpressionAnalyzer::analyzeAggregation(ActionsDAGPtr & temp_actions)
}
void ExpressionAnalyzer::initGlobalSubqueriesAndExternalTables(bool do_global)
void ExpressionAnalyzer::initGlobalSubqueriesAndExternalTables(bool do_global, bool is_explain)
{
if (do_global)
{
GlobalSubqueriesVisitor::Data subqueries_data(
getContext(), subquery_depth, isRemoteStorage(), external_tables, subqueries_for_sets, has_global_subqueries);
getContext(), subquery_depth, isRemoteStorage(), is_explain, external_tables, subqueries_for_sets, has_global_subqueries);
GlobalSubqueriesVisitor(subqueries_data).visit(query);
}
}

View File

@ -98,7 +98,7 @@ public:
/// Ctor for non-select queries. Generally its usage is:
/// auto actions = ExpressionAnalyzer(query, syntax, context).getActions();
ExpressionAnalyzer(const ASTPtr & query_, const TreeRewriterResultPtr & syntax_analyzer_result_, ContextPtr context_)
: ExpressionAnalyzer(query_, syntax_analyzer_result_, context_, 0, false, {}, {})
: ExpressionAnalyzer(query_, syntax_analyzer_result_, context_, 0, false, false, {}, {})
{
}
@ -154,6 +154,7 @@ protected:
ContextPtr context_,
size_t subquery_depth_,
bool do_global_,
bool is_explain_,
SubqueriesForSets subqueries_for_sets_,
PreparedSets prepared_sets_);
@ -168,7 +169,7 @@ protected:
const NamesAndTypesList & sourceColumns() const { return syntax->required_source_columns; }
const std::vector<const ASTFunction *> & aggregates() const { return syntax->aggregates; }
/// Find global subqueries in the GLOBAL IN/JOIN sections. Fills in external_tables.
void initGlobalSubqueriesAndExternalTables(bool do_global);
void initGlobalSubqueriesAndExternalTables(bool do_global, bool is_explain);
ArrayJoinActionPtr addMultipleArrayJoinAction(ActionsDAGPtr & actions, bool is_left) const;
@ -305,6 +306,7 @@ public:
context_,
options_.subquery_depth,
do_global_,
options_.is_explain,
std::move(subqueries_for_sets_),
std::move(prepared_sets_))
, metadata_snapshot(metadata_snapshot_)

View File

@ -39,6 +39,7 @@ public:
{
size_t subquery_depth;
bool is_remote;
bool is_explain;
TemporaryTablesMapping & external_tables;
SubqueriesForSets & subqueries_for_sets;
bool & has_global_subqueries;
@ -47,12 +48,14 @@ public:
ContextPtr context_,
size_t subquery_depth_,
bool is_remote_,
bool is_explain_,
TemporaryTablesMapping & tables,
SubqueriesForSets & subqueries_for_sets_,
bool & has_global_subqueries_)
: WithContext(context_)
, subquery_depth(subquery_depth_)
, is_remote(is_remote_)
, is_explain(is_explain_)
, external_tables(tables)
, subqueries_for_sets(subqueries_for_sets_)
, has_global_subqueries(has_global_subqueries_)
@ -160,7 +163,11 @@ public:
/// We need to materialize external tables immediately because reading from distributed
/// tables might generate local plans which can refer to external tables during index
/// analysis. It's too late to populate the external table via CreatingSetsTransform.
if (getContext()->getSettingsRef().use_index_for_in_with_subqueries)
if (is_explain)
{
/// Do not materialize external tables if it's explain statement.
}
else if (getContext()->getSettingsRef().use_index_for_in_with_subqueries)
{
auto external_table = external_storage_holder->getTable();
auto table_out = external_table->write({}, external_table->getInMemoryMetadataPtr(), getContext());

View File

@ -269,6 +269,9 @@ QueryPipeline InterpreterExplainQuery::executeImpl()
bool single_line = false;
bool insert_buf = true;
SelectQueryOptions options;
options.setExplain();
switch (ast.getKind())
{
case ASTExplainQuery::ParsedAST:
@ -299,7 +302,7 @@ QueryPipeline InterpreterExplainQuery::executeImpl()
auto settings = checkAndGetSettings<QueryPlanSettings>(ast.getSettings());
QueryPlan plan;
InterpreterSelectWithUnionQuery interpreter(ast.getExplainedQuery(), getContext(), SelectQueryOptions());
InterpreterSelectWithUnionQuery interpreter(ast.getExplainedQuery(), getContext(), options);
interpreter.buildQueryPlan(plan);
if (settings.optimize)
@ -334,7 +337,7 @@ QueryPipeline InterpreterExplainQuery::executeImpl()
auto settings = checkAndGetSettings<QueryPipelineSettings>(ast.getSettings());
QueryPlan plan;
InterpreterSelectWithUnionQuery interpreter(ast.getExplainedQuery(), getContext(), SelectQueryOptions());
InterpreterSelectWithUnionQuery interpreter(ast.getExplainedQuery(), getContext(), options);
interpreter.buildQueryPlan(plan);
auto pipeline = plan.buildQueryPipeline(
QueryPlanOptimizationSettings::fromContext(getContext()),

View File

@ -49,6 +49,7 @@ struct SelectQueryOptions
bool is_subquery = false; // non-subquery can also have subquery_depth > 0, e.g. insert select
bool with_all_cols = false; /// asterisk include materialized and aliased columns
bool settings_limit_offset_done = false;
bool is_explain = false; /// The value is true if it's explain statement.
/// These two fields are used to evaluate shardNum() and shardCount() function when
/// prefer_localhost_replica == 1 and local instance is selected. They are needed because local
@ -150,6 +151,12 @@ struct SelectQueryOptions
shard_count = shard_count_;
return *this;
}
SelectQueryOptions & setExplain(bool value = true)
{
is_explain = value;
return *this;
}
};
}