ClickHouse/src/Interpreters/GlobalSubqueriesVisitor.h

Ignoring revisions in .git-blame-ignore-revs. Click here to bypass and see the normal blame view.

314 lines
13 KiB
C++
Raw Normal View History

#pragma once
#include <Core/Block.h>
#include <Core/NamesAndTypes.h>
#include <Databases/IDatabase.h>
#include <IO/WriteHelpers.h>
#include <Interpreters/Context.h>
#include <Interpreters/IdentifierSemantic.h>
#include <Interpreters/InDepthNodeVisitor.h>
#include <Interpreters/interpretSubquery.h>
2022-07-18 15:53:30 +00:00
#include <Interpreters/PreparedSets.h>
#include <Interpreters/TableJoin.h>
#include <Parsers/ASTFunction.h>
#include <Parsers/ASTIdentifier.h>
#include <Parsers/ASTLiteral.h>
#include <Parsers/ASTSelectQuery.h>
#include <Parsers/ASTSubquery.h>
#include <Parsers/ASTTablesInSelectQuery.h>
#include <Parsers/IAST.h>
2021-09-16 17:40:42 +00:00
#include <Processors/Executors/CompletedPipelineExecutor.h>
2021-07-23 19:33:59 +00:00
#include <Processors/Sinks/SinkToStorage.h>
#include <Processors/QueryPlan/QueryPlan.h>
#include <Common/typeid_cast.h>
#include <Storages/ColumnsDescription.h>
#include <Storages/ConstraintsDescription.h>
#include <Storages/IStorage.h>
namespace DB
{
2020-02-25 18:10:48 +00:00
namespace ErrorCodes
{
extern const int WRONG_GLOBAL_SUBQUERY;
2023-05-26 19:25:33 +00:00
extern const int LOGICAL_ERROR;
extern const int SUPPORT_IS_DISABLED;
2020-02-25 18:10:48 +00:00
}
class GlobalSubqueriesMatcher
{
public:
struct Data : WithContext
{
size_t subquery_depth;
bool is_remote;
bool is_explain;
2020-03-10 19:36:17 +00:00
TemporaryTablesMapping & external_tables;
2022-07-18 15:53:30 +00:00
PreparedSetsPtr prepared_sets;
bool & has_global_subqueries;
TableJoin * table_join;
Data(
ContextPtr context_,
size_t subquery_depth_,
bool is_remote_,
bool is_explain_,
TemporaryTablesMapping & tables,
2022-07-18 15:53:30 +00:00
PreparedSetsPtr prepared_sets_,
bool & has_global_subqueries_,
TableJoin * table_join_)
: WithContext(context_)
, subquery_depth(subquery_depth_)
, is_remote(is_remote_)
, is_explain(is_explain_)
, external_tables(tables)
2022-07-18 15:53:30 +00:00
, prepared_sets(prepared_sets_)
, has_global_subqueries(has_global_subqueries_)
, table_join(table_join_)
{
}
void addExternalStorage(ASTPtr & ast, const Names & required_columns, bool set_alias = false)
{
/// With nondistributed queries, creating temporary tables does not make sense.
if (!is_remote)
return;
bool is_table = false;
ASTPtr subquery_or_table_name; /// ASTTableIdentifier | ASTSubquery | ASTTableExpression
if (const auto * ast_table_expr = ast->as<ASTTableExpression>())
{
subquery_or_table_name = ast_table_expr->subquery;
if (ast_table_expr->database_and_table_name)
{
subquery_or_table_name = ast_table_expr->database_and_table_name;
is_table = true;
}
}
2020-11-13 14:34:47 +00:00
else if (ast->as<ASTTableIdentifier>())
{
subquery_or_table_name = ast;
is_table = true;
}
2021-08-03 12:16:55 +00:00
else if (ast->as<ASTSubquery>())
{
subquery_or_table_name = ast;
}
if (!subquery_or_table_name)
throw Exception(ErrorCodes::WRONG_GLOBAL_SUBQUERY, "Global subquery requires subquery or table name");
if (is_table)
{
/// If this is already an external table, you do not need to add anything. Just remember its presence.
2020-03-11 19:10:55 +00:00
auto temporary_table_name = getIdentifierName(subquery_or_table_name);
bool exists_in_local_map = external_tables.contains(temporary_table_name);
bool exists_in_context = static_cast<bool>(getContext()->tryResolveStorageID(
StorageID("", temporary_table_name), Context::ResolveExternal));
2020-03-11 19:10:55 +00:00
if (exists_in_local_map || exists_in_context)
return;
}
String alias = subquery_or_table_name->tryGetAlias();
String external_table_name;
if (alias.empty())
{
auto hash = subquery_or_table_name->getTreeHash(/*ignore_aliases=*/ true);
2023-07-06 00:35:44 +00:00
external_table_name = fmt::format("_data_{}", toString(hash));
}
else
external_table_name = alias;
/** We replace the subquery with the name of the temporary table.
* It is in this form, the request will go to the remote server.
* This temporary table will go to the remote server, and on its side,
* instead of doing a subquery, you just need to read it.
* TODO We can do better than using alias to name external tables
*/
2020-10-26 15:49:00 +00:00
auto database_and_table_name = std::make_shared<ASTTableIdentifier>(external_table_name);
if (set_alias)
{
2020-10-26 15:49:00 +00:00
if (auto * table_name = subquery_or_table_name->as<ASTTableIdentifier>())
if (alias.empty())
alias = table_name->shortName();
database_and_table_name->setAlias(alias);
}
if (auto * ast_table_expr = ast->as<ASTTableExpression>())
{
ast_table_expr->subquery.reset();
ast_table_expr->database_and_table_name = database_and_table_name;
ast_table_expr->children.clear();
ast_table_expr->children.emplace_back(database_and_table_name);
}
else
ast = database_and_table_name;
if (external_tables.contains(external_table_name))
return;
auto interpreter = interpretSubquery(subquery_or_table_name, getContext(), subquery_depth, required_columns);
Block sample = interpreter->getSampleBlock();
NamesAndTypesList columns = sample.getNamesAndTypesList();
auto external_storage_holder = std::make_shared<TemporaryTableHolder>(
getContext(),
ColumnsDescription{columns},
ConstraintsDescription{},
nullptr,
/*create_for_global_subquery*/ true);
StoragePtr external_storage = external_storage_holder->getTable();
external_tables.emplace(external_table_name, external_storage_holder);
2020-11-21 08:25:45 +00:00
auto set_key = database_and_table_name->getTreeHash(/*ignore_aliases=*/ true);
2023-05-25 19:18:11 +00:00
2023-06-22 14:23:04 +00:00
if (!prepared_sets->findSubquery(set_key))
{
std::unique_ptr<QueryPlan> source = std::make_unique<QueryPlan>();
interpreter->buildQueryPlan(*source);
2023-05-04 17:54:08 +00:00
2023-06-22 14:23:04 +00:00
auto future_set = prepared_sets->addFromSubquery(set_key, std::move(source), std::move(external_storage), nullptr, getContext()->getSettingsRef());
external_storage_holder->future_set = std::move(future_set);
2020-11-21 08:25:45 +00:00
}
2023-06-22 14:23:04 +00:00
else
throw Exception(ErrorCodes::LOGICAL_ERROR, "Set is already created for GLOBAL IN");
/** NOTE If it was written IN tmp_table - the existing temporary (but not external) table,
* then a new temporary table will be created (for example, _data1),
* and the data will then be copied to it.
* Maybe this can be avoided.
*/
}
};
2019-02-22 13:33:56 +00:00
static void visit(ASTPtr & ast, Data & data)
{
2019-03-11 13:22:51 +00:00
if (auto * t = ast->as<ASTFunction>())
visit(*t, ast, data);
2019-03-11 13:22:51 +00:00
if (auto * t = ast->as<ASTTablesInSelectQueryElement>())
visit(*t, ast, data);
}
static bool needChildVisit(ASTPtr &, const ASTPtr & child)
{
/// We do not go into subqueries.
2020-11-13 14:13:27 +00:00
return !child->as<ASTSelectQuery>();
}
private:
/// GLOBAL IN
static void visit(ASTFunction & func, ASTPtr &, Data & data)
{
const Settings & settings = data.getContext()->getSettingsRef();
const bool prefer_global = settings.prefer_global_in_and_join;
const bool enable_parallel_processing_of_joins = data.getContext()->canUseParallelReplicasOnInitiator();
if (((prefer_global || enable_parallel_processing_of_joins)
2021-04-21 13:14:40 +00:00
&& (func.name == "in" || func.name == "notIn" || func.name == "nullIn" || func.name == "notNullIn"))
|| func.name == "globalIn" || func.name == "globalNotIn" || func.name == "globalNullIn" || func.name == "globalNotNullIn")
{
2020-04-11 21:28:04 +00:00
ASTPtr & ast = func.arguments->children[1];
if (enable_parallel_processing_of_joins)
{
/// We don't enable parallel replicas for IN (subquery)
if (ast->as<ASTSubquery>())
{
if (settings.allow_experimental_parallel_reading_from_replicas == 1)
{
LOG_DEBUG(getLogger("GlobalSubqueriesMatcher"), "IN with subquery is not supported with parallel replicas");
data.getContext()->getQueryContext()->setSetting("allow_experimental_parallel_reading_from_replicas", Field(0));
return;
}
else if (settings.allow_experimental_parallel_reading_from_replicas == 2)
throw Exception(ErrorCodes::SUPPORT_IS_DISABLED, "IN with subquery is not supported with parallel replicas");
}
}
2020-04-11 21:28:04 +00:00
2021-07-11 06:09:52 +00:00
/// Literal or function can use regular IN.
/// NOTE: We don't support passing table functions to IN.
if (ast->as<ASTLiteral>() || ast->as<ASTFunction>())
2020-04-11 21:28:04 +00:00
{
if (func.name == "globalIn")
func.name = "in";
2021-04-21 13:14:40 +00:00
else if (func.name == "globalNotIn")
2020-04-11 21:28:04 +00:00
func.name = "notIn";
2021-04-21 13:14:40 +00:00
else if (func.name == "globalNullIn")
func.name = "nullIn";
else if (func.name == "globalNotNullIn")
func.name = "notNullIn";
2020-04-11 21:28:04 +00:00
return;
}
data.addExternalStorage(ast, {});
data.has_global_subqueries = true;
}
}
/// GLOBAL JOIN
static void visit(ASTTablesInSelectQueryElement & table_elem, ASTPtr &, Data & data)
{
const Settings & settings = data.getContext()->getSettingsRef();
const bool prefer_global = settings.prefer_global_in_and_join;
const bool enable_parallel_processing_of_joins = data.getContext()->canUseParallelReplicasOnInitiator();
2021-04-21 13:14:40 +00:00
if (table_elem.table_join
&& (table_elem.table_join->as<ASTTableJoin &>().locality == JoinLocality::Global || prefer_global
|| enable_parallel_processing_of_joins))
{
if (enable_parallel_processing_of_joins)
{
/// For parallel replicas we currently only support JOIN with subqueries
/// Note that tableA join tableB is previously converted into tableA JOIN (Select * FROM tableB) so that's ok
/// We don't support WITH cte as (subquery) Select table JOIN cte because we don't do conversion in AST
bool is_subquery = false;
if (const auto * ast_table_expr = table_elem.table_expression->as<ASTTableExpression>())
2024-02-01 10:46:55 +00:00
{
is_subquery = ast_table_expr->subquery && ast_table_expr->subquery->as<ASTSubquery>() != nullptr
&& ast_table_expr->subquery->as<ASTSubquery>()->cte_name.empty();
2024-02-01 10:46:55 +00:00
}
else if (table_elem.table_expression->as<ASTSubquery>())
is_subquery = true;
if (!is_subquery)
{
if (settings.allow_experimental_parallel_reading_from_replicas == 1)
{
LOG_DEBUG(getLogger("GlobalSubqueriesMatcher"), "JOIN with parallel replicas is only supported with subqueries");
data.getContext()->getQueryContext()->setSetting("allow_experimental_parallel_reading_from_replicas", Field(0));
return;
}
else if (settings.allow_experimental_parallel_reading_from_replicas == 2)
throw Exception(ErrorCodes::SUPPORT_IS_DISABLED, "JOIN with parallel replicas is only supported with subqueries");
}
}
Names required_columns;
/// Fill required columns for GLOBAL JOIN.
/// This code is partial copy-paste from ExpressionAnalyzer.
if (data.table_join)
{
auto joined_block_actions = data.table_join->createJoinedBlockActions(data.getContext());
NamesWithAliases required_columns_with_aliases = data.table_join->getRequiredColumns(
Block(joined_block_actions->getResultColumns()), joined_block_actions->getRequiredColumns().getNames());
for (auto & pr : required_columns_with_aliases)
required_columns.push_back(pr.first);
}
data.addExternalStorage(table_elem.table_expression, required_columns, true);
data.has_global_subqueries = true;
}
}
};
/// Converts GLOBAL subqueries to external tables; Puts them into the external_tables dictionary: name -> StoragePtr.
using GlobalSubqueriesVisitor = InDepthNodeVisitor<GlobalSubqueriesMatcher, false>;
}