mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-10 09:32:06 +00:00
Merge branch 'master' of github.com:yandex/ClickHouse
This commit is contained in:
commit
1af718677a
130
dbms/src/Interpreters/ExecuteScalarSubqueriesVisitor.cpp
Normal file
130
dbms/src/Interpreters/ExecuteScalarSubqueriesVisitor.cpp
Normal file
@ -0,0 +1,130 @@
|
||||
#include <Parsers/ASTFunction.h>
|
||||
#include <Parsers/ASTLiteral.h>
|
||||
#include <Parsers/ASTSubquery.h>
|
||||
#include <Parsers/ASTTablesInSelectQuery.h>
|
||||
#include <Parsers/ASTExpressionList.h>
|
||||
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Interpreters/QueryNormalizer.h>
|
||||
#include <Interpreters/InterpreterSelectWithUnionQuery.h>
|
||||
#include <Interpreters/ExecuteScalarSubqueriesVisitor.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
namespace ErrorCodes
|
||||
{
|
||||
extern const int INCORRECT_RESULT_OF_SCALAR_SUBQUERY;
|
||||
extern const int TOO_MANY_ROWS;
|
||||
}
|
||||
|
||||
|
||||
static ASTPtr addTypeConversion(std::unique_ptr<ASTLiteral> && ast, const String & type_name)
|
||||
{
|
||||
auto func = std::make_shared<ASTFunction>();
|
||||
ASTPtr res = func;
|
||||
func->alias = ast->alias;
|
||||
func->prefer_alias_to_column_name = ast->prefer_alias_to_column_name;
|
||||
ast->alias.clear();
|
||||
func->name = "CAST";
|
||||
auto exp_list = std::make_shared<ASTExpressionList>();
|
||||
func->arguments = exp_list;
|
||||
func->children.push_back(func->arguments);
|
||||
exp_list->children.emplace_back(ast.release());
|
||||
exp_list->children.emplace_back(std::make_shared<ASTLiteral>(type_name));
|
||||
return res;
|
||||
}
|
||||
|
||||
void ExecuteScalarSubqueriesVisitor::visit(ASTSubquery * subquery, ASTPtr & ast, const DumpASTNode &) const
|
||||
{
|
||||
Context subquery_context = context;
|
||||
Settings subquery_settings = context.getSettings();
|
||||
subquery_settings.max_result_rows = 1;
|
||||
subquery_settings.extremes = 0;
|
||||
subquery_context.setSettings(subquery_settings);
|
||||
|
||||
ASTPtr subquery_select = subquery->children.at(0);
|
||||
BlockIO res = InterpreterSelectWithUnionQuery(
|
||||
subquery_select, subquery_context, {}, QueryProcessingStage::Complete, subquery_depth + 1).execute();
|
||||
|
||||
Block block;
|
||||
try
|
||||
{
|
||||
block = res.in->read();
|
||||
|
||||
if (!block)
|
||||
{
|
||||
/// Interpret subquery with empty result as Null literal
|
||||
auto ast_new = std::make_unique<ASTLiteral>(Null());
|
||||
ast_new->setAlias(ast->tryGetAlias());
|
||||
ast = std::move(ast_new);
|
||||
return;
|
||||
}
|
||||
|
||||
if (block.rows() != 1 || res.in->read())
|
||||
throw Exception("Scalar subquery returned more than one row", ErrorCodes::INCORRECT_RESULT_OF_SCALAR_SUBQUERY);
|
||||
}
|
||||
catch (const Exception & e)
|
||||
{
|
||||
if (e.code() == ErrorCodes::TOO_MANY_ROWS)
|
||||
throw Exception("Scalar subquery returned more than one row", ErrorCodes::INCORRECT_RESULT_OF_SCALAR_SUBQUERY);
|
||||
else
|
||||
throw;
|
||||
}
|
||||
|
||||
size_t columns = block.columns();
|
||||
if (columns == 1)
|
||||
{
|
||||
auto lit = std::make_unique<ASTLiteral>((*block.safeGetByPosition(0).column)[0]);
|
||||
lit->alias = subquery->alias;
|
||||
lit->prefer_alias_to_column_name = subquery->prefer_alias_to_column_name;
|
||||
ast = addTypeConversion(std::move(lit), block.safeGetByPosition(0).type->getName());
|
||||
}
|
||||
else
|
||||
{
|
||||
auto tuple = std::make_shared<ASTFunction>();
|
||||
tuple->alias = subquery->alias;
|
||||
ast = tuple;
|
||||
tuple->name = "tuple";
|
||||
auto exp_list = std::make_shared<ASTExpressionList>();
|
||||
tuple->arguments = exp_list;
|
||||
tuple->children.push_back(tuple->arguments);
|
||||
|
||||
exp_list->children.resize(columns);
|
||||
for (size_t i = 0; i < columns; ++i)
|
||||
{
|
||||
exp_list->children[i] = addTypeConversion(
|
||||
std::make_unique<ASTLiteral>((*block.safeGetByPosition(i).column)[0]),
|
||||
block.safeGetByPosition(i).type->getName());
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
|
||||
void ExecuteScalarSubqueriesVisitor::visit(ASTTableExpression *, ASTPtr &, const DumpASTNode &) const
|
||||
{
|
||||
/// Don't descend into subqueries in FROM section.
|
||||
}
|
||||
|
||||
void ExecuteScalarSubqueriesVisitor::visit(ASTFunction * func, ASTPtr & ast, const DumpASTNode &) const
|
||||
{
|
||||
/// Don't descend into subqueries in arguments of IN operator.
|
||||
/// But if an argument is not subquery, than deeper may be scalar subqueries and we need to descend in them.
|
||||
|
||||
if (functionIsInOrGlobalInOperator(func->name))
|
||||
{
|
||||
for (auto & child : ast->children)
|
||||
{
|
||||
if (child != func->arguments)
|
||||
visit(child);
|
||||
else
|
||||
for (size_t i = 0, size = func->arguments->children.size(); i < size; ++i)
|
||||
if (i != 1 || !typeid_cast<ASTSubquery *>(func->arguments->children[i].get()))
|
||||
visit(func->arguments->children[i]);
|
||||
}
|
||||
}
|
||||
else
|
||||
visitChildren(ast);
|
||||
}
|
||||
|
||||
}
|
79
dbms/src/Interpreters/ExecuteScalarSubqueriesVisitor.h
Normal file
79
dbms/src/Interpreters/ExecuteScalarSubqueriesVisitor.h
Normal file
@ -0,0 +1,79 @@
|
||||
#pragma once
|
||||
|
||||
#include <Common/typeid_cast.h>
|
||||
#include <Parsers/DumpASTNode.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
class Context;
|
||||
class ASTSubquery;
|
||||
class ASTFunction;
|
||||
struct ASTTableExpression;
|
||||
|
||||
|
||||
/** Replace subqueries that return exactly one row
|
||||
* ("scalar" subqueries) to the corresponding constants.
|
||||
*
|
||||
* If the subquery returns more than one column, it is replaced by a tuple of constants.
|
||||
*
|
||||
* Features
|
||||
*
|
||||
* A replacement occurs during query analysis, and not during the main runtime.
|
||||
* This means that the progress indicator will not work during the execution of these requests,
|
||||
* and also such queries can not be aborted.
|
||||
*
|
||||
* But the query result can be used for the index in the table.
|
||||
*
|
||||
* Scalar subqueries are executed on the request-initializer server.
|
||||
* The request is sent to remote servers with already substituted constants.
|
||||
*/
|
||||
class ExecuteScalarSubqueriesVisitor
|
||||
{
|
||||
public:
|
||||
ExecuteScalarSubqueriesVisitor(const Context & context_, size_t subquery_depth_, std::ostream * ostr_ = nullptr)
|
||||
: context(context_),
|
||||
subquery_depth(subquery_depth_),
|
||||
visit_depth(0),
|
||||
ostr(ostr_)
|
||||
{}
|
||||
|
||||
void visit(ASTPtr & ast) const
|
||||
{
|
||||
DumpASTNode dump(*ast, ostr, visit_depth, "executeScalarSubqueries");
|
||||
|
||||
if (!tryVisit<ASTSubquery>(ast, dump) &&
|
||||
!tryVisit<ASTTableExpression>(ast, dump) &&
|
||||
!tryVisit<ASTFunction>(ast, dump))
|
||||
visitChildren(ast);
|
||||
}
|
||||
|
||||
private:
|
||||
const Context & context;
|
||||
size_t subquery_depth;
|
||||
mutable size_t visit_depth;
|
||||
std::ostream * ostr;
|
||||
|
||||
void visit(ASTSubquery * subquery, ASTPtr & ast, const DumpASTNode & dump) const;
|
||||
void visit(ASTFunction * func, ASTPtr & ast, const DumpASTNode &) const;
|
||||
void visit(ASTTableExpression *, ASTPtr &, const DumpASTNode &) const;
|
||||
|
||||
void visitChildren(ASTPtr & ast) const
|
||||
{
|
||||
for (auto & child : ast->children)
|
||||
visit(child);
|
||||
}
|
||||
|
||||
template <typename T>
|
||||
bool tryVisit(ASTPtr & ast, const DumpASTNode & dump) const
|
||||
{
|
||||
if (T * t = typeid_cast<T *>(ast.get()))
|
||||
{
|
||||
visit(t, ast, dump);
|
||||
return true;
|
||||
}
|
||||
return false;
|
||||
}
|
||||
};
|
||||
|
||||
}
|
@ -37,6 +37,7 @@
|
||||
#include <Interpreters/ProjectionManipulation.h>
|
||||
#include <Interpreters/evaluateConstantExpression.h>
|
||||
#include <Interpreters/TranslateQualifiedNamesVisitor.h>
|
||||
#include <Interpreters/ExecuteScalarSubqueriesVisitor.h>
|
||||
|
||||
#include <AggregateFunctions/AggregateFunctionFactory.h>
|
||||
#include <AggregateFunctions/parseAggregateFunctionParameters.h>
|
||||
@ -86,8 +87,6 @@ namespace ErrorCodes
|
||||
extern const int MULTIPLE_EXPRESSIONS_FOR_ALIAS;
|
||||
extern const int UNKNOWN_IDENTIFIER;
|
||||
extern const int CYCLIC_ALIASES;
|
||||
extern const int INCORRECT_RESULT_OF_SCALAR_SUBQUERY;
|
||||
extern const int TOO_MANY_ROWS;
|
||||
extern const int NOT_FOUND_COLUMN_IN_BLOCK;
|
||||
extern const int INCORRECT_ELEMENT_OF_SET;
|
||||
extern const int ALIAS_REQUIRED;
|
||||
@ -150,16 +149,6 @@ const std::unordered_set<String> possibly_injective_function_names
|
||||
namespace
|
||||
{
|
||||
|
||||
bool functionIsInOperator(const String & name)
|
||||
{
|
||||
return name == "in" || name == "notIn";
|
||||
}
|
||||
|
||||
bool functionIsInOrGlobalInOperator(const String & name)
|
||||
{
|
||||
return name == "in" || name == "notIn" || name == "globalIn" || name == "globalNotIn";
|
||||
}
|
||||
|
||||
void removeDuplicateColumns(NamesAndTypesList & columns)
|
||||
{
|
||||
std::set<String> names;
|
||||
@ -904,8 +893,13 @@ void ExpressionAnalyzer::addAliasColumns()
|
||||
|
||||
void ExpressionAnalyzer::executeScalarSubqueries()
|
||||
{
|
||||
LogAST log;
|
||||
|
||||
if (!select_query)
|
||||
executeScalarSubqueriesImpl(query);
|
||||
{
|
||||
ExecuteScalarSubqueriesVisitor execute_scalar_subqueries_visitor(context, subquery_depth, log.stream());
|
||||
execute_scalar_subqueries_visitor.visit(query);
|
||||
}
|
||||
else
|
||||
{
|
||||
for (auto & child : query->children)
|
||||
@ -914,143 +908,14 @@ void ExpressionAnalyzer::executeScalarSubqueries()
|
||||
if (!typeid_cast<const ASTTableExpression *>(child.get())
|
||||
&& !typeid_cast<const ASTSelectQuery *>(child.get()))
|
||||
{
|
||||
executeScalarSubqueriesImpl(child);
|
||||
ExecuteScalarSubqueriesVisitor execute_scalar_subqueries_visitor(context, subquery_depth, log.stream());
|
||||
execute_scalar_subqueries_visitor.visit(child);
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
|
||||
static ASTPtr addTypeConversion(std::unique_ptr<ASTLiteral> && ast, const String & type_name)
|
||||
{
|
||||
auto func = std::make_shared<ASTFunction>();
|
||||
ASTPtr res = func;
|
||||
func->alias = ast->alias;
|
||||
func->prefer_alias_to_column_name = ast->prefer_alias_to_column_name;
|
||||
ast->alias.clear();
|
||||
func->name = "CAST";
|
||||
auto exp_list = std::make_shared<ASTExpressionList>();
|
||||
func->arguments = exp_list;
|
||||
func->children.push_back(func->arguments);
|
||||
exp_list->children.emplace_back(ast.release());
|
||||
exp_list->children.emplace_back(std::make_shared<ASTLiteral>(type_name));
|
||||
return res;
|
||||
}
|
||||
|
||||
|
||||
void ExpressionAnalyzer::executeScalarSubqueriesImpl(ASTPtr & ast)
|
||||
{
|
||||
/** Replace subqueries that return exactly one row
|
||||
* ("scalar" subqueries) to the corresponding constants.
|
||||
*
|
||||
* If the subquery returns more than one column, it is replaced by a tuple of constants.
|
||||
*
|
||||
* Features
|
||||
*
|
||||
* A replacement occurs during query analysis, and not during the main runtime.
|
||||
* This means that the progress indicator will not work during the execution of these requests,
|
||||
* and also such queries can not be aborted.
|
||||
*
|
||||
* But the query result can be used for the index in the table.
|
||||
*
|
||||
* Scalar subqueries are executed on the request-initializer server.
|
||||
* The request is sent to remote servers with already substituted constants.
|
||||
*/
|
||||
|
||||
if (ASTSubquery * subquery = typeid_cast<ASTSubquery *>(ast.get()))
|
||||
{
|
||||
Context subquery_context = context;
|
||||
Settings subquery_settings = context.getSettings();
|
||||
subquery_settings.max_result_rows = 1;
|
||||
subquery_settings.extremes = 0;
|
||||
subquery_context.setSettings(subquery_settings);
|
||||
|
||||
ASTPtr subquery_select = subquery->children.at(0);
|
||||
BlockIO res = InterpreterSelectWithUnionQuery(subquery_select, subquery_context, {}, QueryProcessingStage::Complete, subquery_depth + 1).execute();
|
||||
|
||||
Block block;
|
||||
try
|
||||
{
|
||||
block = res.in->read();
|
||||
|
||||
if (!block)
|
||||
{
|
||||
/// Interpret subquery with empty result as Null literal
|
||||
auto ast_new = std::make_unique<ASTLiteral>(Null());
|
||||
ast_new->setAlias(ast->tryGetAlias());
|
||||
ast = std::move(ast_new);
|
||||
return;
|
||||
}
|
||||
|
||||
if (block.rows() != 1 || res.in->read())
|
||||
throw Exception("Scalar subquery returned more than one row", ErrorCodes::INCORRECT_RESULT_OF_SCALAR_SUBQUERY);
|
||||
}
|
||||
catch (const Exception & e)
|
||||
{
|
||||
if (e.code() == ErrorCodes::TOO_MANY_ROWS)
|
||||
throw Exception("Scalar subquery returned more than one row", ErrorCodes::INCORRECT_RESULT_OF_SCALAR_SUBQUERY);
|
||||
else
|
||||
throw;
|
||||
}
|
||||
|
||||
size_t columns = block.columns();
|
||||
if (columns == 1)
|
||||
{
|
||||
auto lit = std::make_unique<ASTLiteral>((*block.safeGetByPosition(0).column)[0]);
|
||||
lit->alias = subquery->alias;
|
||||
lit->prefer_alias_to_column_name = subquery->prefer_alias_to_column_name;
|
||||
ast = addTypeConversion(std::move(lit), block.safeGetByPosition(0).type->getName());
|
||||
}
|
||||
else
|
||||
{
|
||||
auto tuple = std::make_shared<ASTFunction>();
|
||||
tuple->alias = subquery->alias;
|
||||
ast = tuple;
|
||||
tuple->name = "tuple";
|
||||
auto exp_list = std::make_shared<ASTExpressionList>();
|
||||
tuple->arguments = exp_list;
|
||||
tuple->children.push_back(tuple->arguments);
|
||||
|
||||
exp_list->children.resize(columns);
|
||||
for (size_t i = 0; i < columns; ++i)
|
||||
{
|
||||
exp_list->children[i] = addTypeConversion(
|
||||
std::make_unique<ASTLiteral>((*block.safeGetByPosition(i).column)[0]),
|
||||
block.safeGetByPosition(i).type->getName());
|
||||
}
|
||||
}
|
||||
}
|
||||
else
|
||||
{
|
||||
/** Don't descend into subqueries in FROM section.
|
||||
*/
|
||||
if (!typeid_cast<ASTTableExpression *>(ast.get()))
|
||||
{
|
||||
/** Don't descend into subqueries in arguments of IN operator.
|
||||
* But if an argument is not subquery, than deeper may be scalar subqueries and we need to descend in them.
|
||||
*/
|
||||
ASTFunction * func = typeid_cast<ASTFunction *>(ast.get());
|
||||
|
||||
if (func && functionIsInOrGlobalInOperator(func->name))
|
||||
{
|
||||
for (auto & child : ast->children)
|
||||
{
|
||||
if (child != func->arguments)
|
||||
executeScalarSubqueriesImpl(child);
|
||||
else
|
||||
for (size_t i = 0, size = func->arguments->children.size(); i < size; ++i)
|
||||
if (i != 1 || !typeid_cast<ASTSubquery *>(func->arguments->children[i].get()))
|
||||
executeScalarSubqueriesImpl(func->arguments->children[i]);
|
||||
}
|
||||
}
|
||||
else
|
||||
for (auto & child : ast->children)
|
||||
executeScalarSubqueriesImpl(child);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
|
||||
void ExpressionAnalyzer::optimizeGroupBy()
|
||||
{
|
||||
if (!(select_query && select_query->group_expression_list))
|
||||
|
@ -21,16 +21,6 @@ namespace ErrorCodes
|
||||
}
|
||||
|
||||
|
||||
namespace
|
||||
{
|
||||
|
||||
bool functionIsInOrGlobalInOperator(const String & name)
|
||||
{
|
||||
return name == "in" || name == "notIn" || name == "globalIn" || name == "globalNotIn";
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
QueryNormalizer::QueryNormalizer(ASTPtr & query, const QueryNormalizer::Aliases & aliases,
|
||||
const Settings & settings, const Names & all_column_names,
|
||||
const TableNamesAndColumnNames & table_names_and_column_names)
|
||||
|
@ -1,5 +1,6 @@
|
||||
#pragma once
|
||||
|
||||
#include <Core/Names.h>
|
||||
#include <Parsers/IAST.h>
|
||||
#include <Interpreters/Settings.h>
|
||||
#include <Interpreters/evaluateQualified.h>
|
||||
@ -7,6 +8,17 @@
|
||||
namespace DB
|
||||
{
|
||||
|
||||
inline bool functionIsInOperator(const String & name)
|
||||
{
|
||||
return name == "in" || name == "notIn";
|
||||
}
|
||||
|
||||
inline bool functionIsInOrGlobalInOperator(const String & name)
|
||||
{
|
||||
return functionIsInOperator(name) || name == "globalIn" || name == "globalNotIn";
|
||||
}
|
||||
|
||||
|
||||
using TableNameAndColumnNames = std::pair<DatabaseAndTableWithAlias, Names>;
|
||||
using TableNamesAndColumnNames = std::vector<TableNameAndColumnNames>;
|
||||
|
||||
|
Loading…
Reference in New Issue
Block a user