2018-03-04 16:15:31 +00:00
|
|
|
#include <Common/typeid_cast.h>
|
|
|
|
#include <Storages/IStorage.h>
|
|
|
|
#include <Interpreters/PredicateExpressionsOptimizer.h>
|
|
|
|
#include <Interpreters/InterpreterSelectQuery.h>
|
|
|
|
#include <AggregateFunctions/AggregateFunctionFactory.h>
|
|
|
|
#include <iostream>
|
2018-08-22 06:42:37 +00:00
|
|
|
#include <Parsers/ASTAsterisk.h>
|
|
|
|
#include <Parsers/ASTQualifiedAsterisk.h>
|
|
|
|
#include <Parsers/queryToString.h>
|
2018-09-12 05:41:09 +00:00
|
|
|
#include <Interpreters/QueryNormalizer.h>
|
2018-09-28 10:52:08 +00:00
|
|
|
#include <Interpreters/QueryAliasesVisitor.h>
|
2018-11-15 15:23:44 +00:00
|
|
|
#include "TranslateQualifiedNamesVisitor.h"
|
2018-03-04 16:15:31 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
static constexpr auto and_function_name = "and";
|
|
|
|
|
|
|
|
PredicateExpressionsOptimizer::PredicateExpressionsOptimizer(
|
2018-10-18 15:03:14 +00:00
|
|
|
ASTSelectQuery * ast_select_, ExtractedSettings && settings_, const Context & context_)
|
2018-08-22 06:42:37 +00:00
|
|
|
: ast_select(ast_select_), settings(settings_), context(context_)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
|
|
|
}
|
|
|
|
|
|
|
|
bool PredicateExpressionsOptimizer::optimize()
|
|
|
|
{
|
2018-08-22 06:42:37 +00:00
|
|
|
if (!settings.enable_optimize_predicate_expression || !ast_select || !ast_select->tables || ast_select->tables->children.empty())
|
2018-03-04 16:15:31 +00:00
|
|
|
return false;
|
|
|
|
|
2018-11-15 15:23:44 +00:00
|
|
|
if (!ast_select->where_expression && !ast_select->prewhere_expression)
|
|
|
|
return false;
|
|
|
|
|
2018-08-20 15:49:39 +00:00
|
|
|
SubqueriesProjectionColumns all_subquery_projection_columns;
|
2018-08-22 06:42:37 +00:00
|
|
|
getAllSubqueryProjectionColumns(all_subquery_projection_columns);
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-20 15:49:39 +00:00
|
|
|
bool is_rewrite_subqueries = false;
|
2018-03-04 16:15:31 +00:00
|
|
|
if (!all_subquery_projection_columns.empty())
|
|
|
|
{
|
2018-08-20 15:49:39 +00:00
|
|
|
is_rewrite_subqueries |= optimizeImpl(ast_select->where_expression, all_subquery_projection_columns, false);
|
|
|
|
is_rewrite_subqueries |= optimizeImpl(ast_select->prewhere_expression, all_subquery_projection_columns, true);
|
2018-03-04 16:15:31 +00:00
|
|
|
}
|
2018-08-20 15:49:39 +00:00
|
|
|
return is_rewrite_subqueries;
|
2018-03-04 16:15:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
bool PredicateExpressionsOptimizer::optimizeImpl(
|
2018-08-20 15:49:39 +00:00
|
|
|
ASTPtr & outer_expression, SubqueriesProjectionColumns & subqueries_projection_columns, bool is_prewhere)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
|
|
|
/// split predicate with `and`
|
|
|
|
PredicateExpressions outer_predicate_expressions = splitConjunctionPredicate(outer_expression);
|
|
|
|
|
2018-10-26 15:13:02 +00:00
|
|
|
std::vector<DatabaseAndTableWithAlias> database_and_table_with_aliases =
|
2018-10-30 16:31:21 +00:00
|
|
|
getDatabaseAndTables(*ast_select, context.getCurrentDatabase());
|
2018-08-22 06:42:37 +00:00
|
|
|
|
2018-03-04 16:15:31 +00:00
|
|
|
bool is_rewrite_subquery = false;
|
|
|
|
for (const auto & outer_predicate : outer_predicate_expressions)
|
|
|
|
{
|
2018-09-12 05:41:09 +00:00
|
|
|
if (isArrayJoinFunction(outer_predicate))
|
|
|
|
continue;
|
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
IdentifiersWithQualifiedNameSet outer_predicate_dependencies;
|
|
|
|
getDependenciesAndQualifiedOfExpression(outer_predicate, outer_predicate_dependencies, database_and_table_with_aliases);
|
2018-03-04 16:15:31 +00:00
|
|
|
|
|
|
|
/// TODO: remove origin expression
|
2018-08-20 15:49:39 +00:00
|
|
|
for (const auto & subquery_projection_columns : subqueries_projection_columns)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
|
|
|
auto subquery = static_cast<ASTSelectQuery *>(subquery_projection_columns.first);
|
|
|
|
const ProjectionsWithAliases projection_columns = subquery_projection_columns.second;
|
|
|
|
|
|
|
|
OptimizeKind optimize_kind = OptimizeKind::NONE;
|
2018-08-22 06:42:37 +00:00
|
|
|
if (!cannotPushDownOuterPredicate(projection_columns, subquery, outer_predicate_dependencies, is_prewhere, optimize_kind))
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
|
|
|
ASTPtr inner_predicate;
|
2018-08-22 06:42:37 +00:00
|
|
|
cloneOuterPredicateForInnerPredicate(outer_predicate, projection_columns, database_and_table_with_aliases, inner_predicate);
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-11-23 18:52:00 +00:00
|
|
|
switch (optimize_kind)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
|
|
|
case OptimizeKind::NONE: continue;
|
|
|
|
case OptimizeKind::PUSH_TO_WHERE: is_rewrite_subquery |= optimizeExpression(inner_predicate, subquery->where_expression, subquery); continue;
|
|
|
|
case OptimizeKind::PUSH_TO_HAVING: is_rewrite_subquery |= optimizeExpression(inner_predicate, subquery->having_expression, subquery); continue;
|
|
|
|
case OptimizeKind::PUSH_TO_PREWHERE: is_rewrite_subquery |= optimizeExpression(inner_predicate, subquery->prewhere_expression, subquery); continue;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return is_rewrite_subquery;
|
|
|
|
}
|
|
|
|
|
|
|
|
PredicateExpressions PredicateExpressionsOptimizer::splitConjunctionPredicate(ASTPtr & predicate_expression)
|
|
|
|
{
|
|
|
|
PredicateExpressions predicate_expressions;
|
|
|
|
|
|
|
|
if (predicate_expression)
|
|
|
|
{
|
|
|
|
predicate_expressions.emplace_back(predicate_expression);
|
|
|
|
|
|
|
|
auto remove_expression_at_index = [&predicate_expressions] (const size_t index)
|
|
|
|
{
|
|
|
|
if (index < predicate_expressions.size() - 1)
|
|
|
|
std::swap(predicate_expressions[index], predicate_expressions.back());
|
|
|
|
predicate_expressions.pop_back();
|
|
|
|
};
|
|
|
|
|
|
|
|
for (size_t idx = 0; idx < predicate_expressions.size();)
|
|
|
|
{
|
|
|
|
const auto expression = predicate_expressions.at(idx);
|
|
|
|
|
|
|
|
if (const auto function = typeid_cast<ASTFunction *>(expression.get()))
|
|
|
|
{
|
|
|
|
if (function->name == and_function_name)
|
|
|
|
{
|
|
|
|
for (auto & child : function->arguments->children)
|
|
|
|
predicate_expressions.emplace_back(child);
|
|
|
|
|
|
|
|
remove_expression_at_index(idx);
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
idx++;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return predicate_expressions;
|
|
|
|
}
|
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
void PredicateExpressionsOptimizer::getDependenciesAndQualifiedOfExpression(const ASTPtr & expression,
|
|
|
|
IdentifiersWithQualifiedNameSet & dependencies_and_qualified,
|
|
|
|
std::vector<DatabaseAndTableWithAlias> & tables_with_aliases)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
2018-08-22 06:42:37 +00:00
|
|
|
if (const auto identifier = typeid_cast<ASTIdentifier *>(expression.get()))
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
2018-08-22 06:42:37 +00:00
|
|
|
if (!identifier->children.empty())
|
|
|
|
dependencies_and_qualified.emplace_back(std::pair(identifier, expression->getAliasOrColumnName()));
|
|
|
|
else
|
|
|
|
{
|
|
|
|
size_t best_table_pos = 0;
|
|
|
|
size_t max_num_qualifiers_to_strip = 0;
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
/// translate qualifiers for dependent columns
|
|
|
|
for (size_t table_pos = 0; table_pos < tables_with_aliases.size(); ++table_pos)
|
|
|
|
{
|
|
|
|
const auto & table = tables_with_aliases[table_pos];
|
|
|
|
auto num_qualifiers_to_strip = getNumComponentsToStripInOrderToTranslateQualifiedName(*identifier, table);
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
if (num_qualifiers_to_strip > max_num_qualifiers_to_strip)
|
|
|
|
{
|
|
|
|
max_num_qualifiers_to_strip = num_qualifiers_to_strip;
|
|
|
|
best_table_pos = table_pos;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
String qualified_name = tables_with_aliases[best_table_pos].getQualifiedNamePrefix() + expression->getAliasOrColumnName();
|
|
|
|
dependencies_and_qualified.emplace_back(std::pair(identifier, qualified_name));
|
|
|
|
}
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
for (const auto & child : expression->children)
|
|
|
|
getDependenciesAndQualifiedOfExpression(child, dependencies_and_qualified, tables_with_aliases);
|
|
|
|
}
|
2018-03-04 16:15:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
bool PredicateExpressionsOptimizer::cannotPushDownOuterPredicate(
|
|
|
|
const ProjectionsWithAliases & subquery_projection_columns, ASTSelectQuery * subquery,
|
2018-08-22 06:42:37 +00:00
|
|
|
IdentifiersWithQualifiedNameSet & outer_predicate_dependencies, bool & is_prewhere, OptimizeKind & optimize_kind)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
2018-08-24 03:32:20 +00:00
|
|
|
if (subquery->final() || subquery->limit_by_expression_list || subquery->limit_length || subquery->with_expression_list)
|
2018-03-04 16:15:31 +00:00
|
|
|
return true;
|
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
for (auto & predicate_dependency : outer_predicate_dependencies)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
|
|
|
bool is_found = false;
|
|
|
|
|
|
|
|
for (auto projection_column : subquery_projection_columns)
|
|
|
|
{
|
2018-08-22 06:42:37 +00:00
|
|
|
if (projection_column.second == predicate_dependency.second)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
|
|
|
is_found = true;
|
|
|
|
optimize_kind = isAggregateFunction(projection_column.first) ? OptimizeKind::PUSH_TO_HAVING : optimize_kind;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (!is_found)
|
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
|
|
|
if (optimize_kind == OptimizeKind::NONE)
|
|
|
|
optimize_kind = is_prewhere ? OptimizeKind::PUSH_TO_PREWHERE : OptimizeKind::PUSH_TO_WHERE;
|
|
|
|
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
|
2018-09-12 05:41:09 +00:00
|
|
|
bool PredicateExpressionsOptimizer::isArrayJoinFunction(const ASTPtr & node)
|
|
|
|
{
|
|
|
|
if (auto function = typeid_cast<ASTFunction *>(node.get()))
|
|
|
|
{
|
|
|
|
if (function->name == "arrayJoin")
|
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
|
|
|
for (auto & child : node->children)
|
|
|
|
if (isArrayJoinFunction(child))
|
|
|
|
return true;
|
|
|
|
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
|
2018-03-04 16:15:31 +00:00
|
|
|
bool PredicateExpressionsOptimizer::isAggregateFunction(ASTPtr & node)
|
|
|
|
{
|
|
|
|
if (auto function = typeid_cast<ASTFunction *>(node.get()))
|
|
|
|
{
|
|
|
|
if (AggregateFunctionFactory::instance().isAggregateFunctionName(function->name))
|
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
|
|
|
for (auto & child : node->children)
|
|
|
|
if (isAggregateFunction(child))
|
|
|
|
return true;
|
|
|
|
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
|
|
|
|
void PredicateExpressionsOptimizer::cloneOuterPredicateForInnerPredicate(
|
2018-08-22 06:42:37 +00:00
|
|
|
const ASTPtr & outer_predicate, const ProjectionsWithAliases & projection_columns,
|
|
|
|
std::vector<DatabaseAndTableWithAlias> & tables, ASTPtr & inner_predicate)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
2018-08-14 21:48:39 +00:00
|
|
|
inner_predicate = outer_predicate->clone();
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-09-27 05:27:45 +00:00
|
|
|
/// clears the alias name contained in the outer predicate
|
|
|
|
cleanExpressionAlias(inner_predicate);
|
2018-08-22 06:42:37 +00:00
|
|
|
IdentifiersWithQualifiedNameSet new_expression_requires;
|
|
|
|
getDependenciesAndQualifiedOfExpression(inner_predicate, new_expression_requires, tables);
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
for (auto & require : new_expression_requires)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
2018-08-22 06:42:37 +00:00
|
|
|
for (auto projection : projection_columns)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
2018-08-22 06:42:37 +00:00
|
|
|
if (require.second == projection.second)
|
2018-09-12 05:41:09 +00:00
|
|
|
{
|
|
|
|
ASTPtr & ast = projection.first;
|
|
|
|
if (!typeid_cast<ASTIdentifier *>(ast.get()) && ast->tryGetAlias().empty())
|
|
|
|
ast->setAlias(ast->getColumnName());
|
|
|
|
require.first->name = ast->getAliasOrColumnName();
|
|
|
|
}
|
2018-03-04 16:15:31 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
bool PredicateExpressionsOptimizer::optimizeExpression(const ASTPtr & outer_expression, ASTPtr & subquery_expression, ASTSelectQuery * subquery)
|
|
|
|
{
|
|
|
|
ASTPtr new_subquery_expression = subquery_expression;
|
|
|
|
new_subquery_expression = new_subquery_expression ? makeASTFunction(and_function_name, outer_expression, subquery_expression) : outer_expression;
|
|
|
|
|
|
|
|
if (!subquery_expression)
|
|
|
|
subquery->children.emplace_back(new_subquery_expression);
|
|
|
|
else
|
|
|
|
for (auto & child : subquery->children)
|
|
|
|
if (child == subquery_expression)
|
|
|
|
child = new_subquery_expression;
|
|
|
|
|
|
|
|
subquery_expression = std::move(new_subquery_expression);
|
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
void PredicateExpressionsOptimizer::getAllSubqueryProjectionColumns(SubqueriesProjectionColumns & all_subquery_projection_columns)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
2018-10-30 16:31:21 +00:00
|
|
|
const auto tables_expression = getSelectTablesExpression(*ast_select);
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
for (const auto & table_expression : tables_expression)
|
|
|
|
{
|
|
|
|
if (table_expression->subquery)
|
|
|
|
{
|
|
|
|
/// Use qualifiers to translate the columns of subqueries
|
2018-10-30 16:31:21 +00:00
|
|
|
DatabaseAndTableWithAlias database_and_table_with_alias(*table_expression, context.getCurrentDatabase());
|
2018-08-22 06:42:37 +00:00
|
|
|
String qualified_name_prefix = database_and_table_with_alias.getQualifiedNamePrefix();
|
|
|
|
getSubqueryProjectionColumns(all_subquery_projection_columns, qualified_name_prefix,
|
|
|
|
static_cast<const ASTSubquery *>(table_expression->subquery.get())->children[0]);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
void PredicateExpressionsOptimizer::getSubqueryProjectionColumns(SubqueriesProjectionColumns & all_subquery_projection_columns,
|
|
|
|
String & qualified_name_prefix, const ASTPtr & subquery)
|
|
|
|
{
|
|
|
|
ASTs select_with_union_projections;
|
|
|
|
auto select_with_union_query = static_cast<ASTSelectWithUnionQuery *>(subquery.get());
|
|
|
|
|
|
|
|
for (auto & select_without_union_query : select_with_union_query->list_of_selects->children)
|
2018-03-04 16:15:31 +00:00
|
|
|
{
|
2018-08-22 06:42:37 +00:00
|
|
|
ProjectionsWithAliases subquery_projections;
|
|
|
|
auto select_projection_columns = getSelectQueryProjectionColumns(select_without_union_query);
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-24 03:32:20 +00:00
|
|
|
if (!select_projection_columns.empty())
|
|
|
|
{
|
|
|
|
if (select_with_union_projections.empty())
|
|
|
|
select_with_union_projections = select_projection_columns;
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-24 03:32:20 +00:00
|
|
|
for (size_t i = 0; i < select_projection_columns.size(); i++)
|
|
|
|
subquery_projections.emplace_back(std::pair(select_projection_columns[i],
|
|
|
|
qualified_name_prefix + select_with_union_projections[i]->getAliasOrColumnName()));
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-24 03:32:20 +00:00
|
|
|
all_subquery_projection_columns.insert(std::pair(select_without_union_query.get(), subquery_projections));
|
|
|
|
}
|
2018-08-22 06:42:37 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
ASTs PredicateExpressionsOptimizer::getSelectQueryProjectionColumns(ASTPtr & ast)
|
|
|
|
{
|
2018-11-15 15:23:44 +00:00
|
|
|
ASTs projection_columns;
|
|
|
|
auto select_query = static_cast<ASTSelectQuery *>(ast.get());
|
|
|
|
|
2018-09-12 05:41:09 +00:00
|
|
|
/// first should normalize query tree.
|
|
|
|
std::unordered_map<String, ASTPtr> aliases;
|
2018-11-15 15:23:44 +00:00
|
|
|
std::vector<DatabaseAndTableWithAlias> tables = getDatabaseAndTables(*select_query, context.getCurrentDatabase());
|
|
|
|
|
2018-12-10 13:02:45 +00:00
|
|
|
TranslateQualifiedNamesVisitor::Data qn_visitor_data{{}, tables};
|
2018-12-06 15:29:55 +00:00
|
|
|
TranslateQualifiedNamesVisitor(qn_visitor_data).visit(ast);
|
2018-12-10 13:02:45 +00:00
|
|
|
QueryAliasesVisitor::Data query_aliases_data{aliases};
|
2018-12-06 19:02:42 +00:00
|
|
|
QueryAliasesVisitor(query_aliases_data).visit(ast);
|
2018-09-24 03:20:22 +00:00
|
|
|
QueryNormalizer(ast, aliases, settings, {}, {}).perform();
|
2018-09-12 05:41:09 +00:00
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
for (const auto & projection_column : select_query->select_expression_list->children)
|
|
|
|
{
|
|
|
|
if (typeid_cast<ASTAsterisk *>(projection_column.get()) || typeid_cast<ASTQualifiedAsterisk *>(projection_column.get()))
|
|
|
|
{
|
|
|
|
ASTs evaluated_columns = evaluateAsterisk(select_query, projection_column);
|
|
|
|
|
|
|
|
for (const auto & column : evaluated_columns)
|
|
|
|
projection_columns.emplace_back(column);
|
|
|
|
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
|
|
|
|
projection_columns.emplace_back(projection_column);
|
|
|
|
}
|
|
|
|
return projection_columns;
|
|
|
|
}
|
|
|
|
|
2018-08-24 03:32:20 +00:00
|
|
|
ASTs PredicateExpressionsOptimizer::evaluateAsterisk(ASTSelectQuery * select_query, const ASTPtr & asterisk)
|
2018-08-22 06:42:37 +00:00
|
|
|
{
|
2018-08-24 05:30:49 +00:00
|
|
|
/// SELECT *, SELECT dummy, SELECT 1 AS id
|
2018-08-22 06:42:37 +00:00
|
|
|
if (!select_query->tables || select_query->tables->children.empty())
|
2018-08-24 03:32:20 +00:00
|
|
|
return {};
|
2018-08-22 06:42:37 +00:00
|
|
|
|
2018-10-30 16:31:21 +00:00
|
|
|
std::vector<const ASTTableExpression *> tables_expression = getSelectTablesExpression(*select_query);
|
2018-08-22 06:42:37 +00:00
|
|
|
|
|
|
|
if (const auto qualified_asterisk = typeid_cast<ASTQualifiedAsterisk *>(asterisk.get()))
|
|
|
|
{
|
|
|
|
if (qualified_asterisk->children.size() != 1)
|
|
|
|
throw Exception("Logical error: qualified asterisk must have exactly one child", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
|
|
|
ASTIdentifier * ident = typeid_cast<ASTIdentifier *>(qualified_asterisk->children[0].get());
|
|
|
|
if (!ident)
|
|
|
|
throw Exception("Logical error: qualified asterisk must have identifier as its child", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
|
|
|
size_t num_components = ident->children.size();
|
|
|
|
if (num_components > 2)
|
|
|
|
throw Exception("Qualified asterisk cannot have more than two qualifiers", ErrorCodes::UNKNOWN_ELEMENT_IN_AST);
|
|
|
|
|
|
|
|
for (auto it = tables_expression.begin(); it != tables_expression.end(); ++it)
|
|
|
|
{
|
|
|
|
const ASTTableExpression * table_expression = *it;
|
2018-10-30 16:31:21 +00:00
|
|
|
DatabaseAndTableWithAlias database_and_table_with_alias(*table_expression, context.getCurrentDatabase());
|
2018-08-22 06:42:37 +00:00
|
|
|
/// database.table.*
|
|
|
|
if (num_components == 2 && !database_and_table_with_alias.database.empty()
|
|
|
|
&& static_cast<const ASTIdentifier &>(*ident->children[0]).name == database_and_table_with_alias.database
|
|
|
|
&& static_cast<const ASTIdentifier &>(*ident->children[1]).name == database_and_table_with_alias.table)
|
|
|
|
continue;
|
|
|
|
/// table.* or alias.*
|
|
|
|
else if (num_components == 0
|
|
|
|
&& ((!database_and_table_with_alias.table.empty() && ident->name == database_and_table_with_alias.table)
|
|
|
|
|| (!database_and_table_with_alias.alias.empty() && ident->name == database_and_table_with_alias.alias)))
|
|
|
|
continue;
|
|
|
|
else
|
|
|
|
/// It's not a required table
|
|
|
|
tables_expression.erase(it);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
ASTs projection_columns;
|
|
|
|
for (auto & table_expression : tables_expression)
|
|
|
|
{
|
|
|
|
if (table_expression->subquery)
|
|
|
|
{
|
|
|
|
const auto subquery = static_cast<const ASTSubquery *>(table_expression->subquery.get());
|
|
|
|
const auto select_with_union_query = static_cast<ASTSelectWithUnionQuery *>(subquery->children[0].get());
|
|
|
|
const auto subquery_projections = getSelectQueryProjectionColumns(select_with_union_query->list_of_selects->children[0]);
|
|
|
|
projection_columns.insert(projection_columns.end(), subquery_projections.begin(), subquery_projections.end());
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
StoragePtr storage;
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
if (table_expression->table_function)
|
2018-08-24 07:40:09 +00:00
|
|
|
{
|
|
|
|
auto query_context = const_cast<Context *>(&context.getQueryContext());
|
|
|
|
storage = query_context->executeTableFunction(table_expression->table_function);
|
|
|
|
}
|
2018-08-22 06:42:37 +00:00
|
|
|
else if (table_expression->database_and_table_name)
|
|
|
|
{
|
|
|
|
const auto database_and_table_ast = static_cast<ASTIdentifier*>(table_expression->database_and_table_name.get());
|
2018-10-30 16:31:21 +00:00
|
|
|
DatabaseAndTableWithAlias database_and_table_name(*database_and_table_ast);
|
|
|
|
storage = context.getTable(database_and_table_name.database, database_and_table_name.table);
|
2018-08-22 06:42:37 +00:00
|
|
|
}
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-08-22 06:42:37 +00:00
|
|
|
const auto block = storage->getSampleBlock();
|
|
|
|
for (size_t idx = 0; idx < block.columns(); idx++)
|
|
|
|
projection_columns.emplace_back(std::make_shared<ASTIdentifier>(block.getByPosition(idx).name));
|
|
|
|
}
|
2018-03-04 16:15:31 +00:00
|
|
|
}
|
2018-08-22 06:42:37 +00:00
|
|
|
return projection_columns;
|
|
|
|
}
|
|
|
|
|
2018-09-27 05:27:45 +00:00
|
|
|
void PredicateExpressionsOptimizer::cleanExpressionAlias(ASTPtr & expression)
|
|
|
|
{
|
|
|
|
const auto my_alias = expression->tryGetAlias();
|
|
|
|
if (!my_alias.empty())
|
|
|
|
expression->setAlias("");
|
|
|
|
|
|
|
|
for (auto & child : expression->children)
|
|
|
|
cleanExpressionAlias(child);
|
|
|
|
}
|
|
|
|
|
2018-03-04 16:15:31 +00:00
|
|
|
}
|