2020-07-22 17:13:05 +00:00
|
|
|
#include <Core/Settings.h>
|
|
|
|
|
|
|
|
#include <Interpreters/TreeOptimizer.h>
|
2021-05-21 18:48:19 +00:00
|
|
|
#include <Interpreters/TreeRewriter.h>
|
2020-07-22 17:13:05 +00:00
|
|
|
#include <Interpreters/OptimizeIfChains.h>
|
|
|
|
#include <Interpreters/OptimizeIfWithConstantConditionVisitor.h>
|
|
|
|
#include <Interpreters/ArithmeticOperationsInAgrFuncOptimize.h>
|
|
|
|
#include <Interpreters/DuplicateOrderByVisitor.h>
|
|
|
|
#include <Interpreters/GroupByFunctionKeysVisitor.h>
|
|
|
|
#include <Interpreters/AggregateFunctionOfGroupByKeysVisitor.h>
|
2020-07-23 15:15:22 +00:00
|
|
|
#include <Interpreters/RewriteAnyFunctionVisitor.h>
|
2020-07-22 17:13:05 +00:00
|
|
|
#include <Interpreters/RemoveInjectiveFunctionsVisitor.h>
|
|
|
|
#include <Interpreters/RedundantFunctionsInOrderByVisitor.h>
|
2021-02-07 07:22:51 +00:00
|
|
|
#include <Interpreters/RewriteCountVariantsVisitor.h>
|
2020-07-22 17:13:05 +00:00
|
|
|
#include <Interpreters/MonotonicityCheckVisitor.h>
|
|
|
|
#include <Interpreters/ConvertStringsToEnumVisitor.h>
|
|
|
|
#include <Interpreters/PredicateExpressionsOptimizer.h>
|
2021-05-21 18:48:19 +00:00
|
|
|
#include <Interpreters/RewriteFunctionToSubcolumnVisitor.h>
|
2020-07-22 17:13:05 +00:00
|
|
|
#include <Interpreters/Context.h>
|
|
|
|
#include <Interpreters/ExternalDictionariesLoader.h>
|
2021-07-21 06:43:40 +00:00
|
|
|
#include <Interpreters/GatherFunctionQuantileVisitor.h>
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
#include <Parsers/ASTExpressionList.h>
|
|
|
|
#include <Parsers/ASTFunction.h>
|
|
|
|
#include <Parsers/ASTLiteral.h>
|
|
|
|
#include <Parsers/ASTOrderByElement.h>
|
|
|
|
#include <Parsers/ASTSelectQuery.h>
|
2020-08-20 17:04:42 +00:00
|
|
|
#include <Parsers/ASTSubquery.h>
|
|
|
|
#include <Parsers/ASTSelectWithUnionQuery.h>
|
2020-07-22 17:13:05 +00:00
|
|
|
#include <Parsers/ASTTablesInSelectQuery.h>
|
|
|
|
|
|
|
|
#include <Functions/FunctionFactory.h>
|
2021-05-21 18:48:19 +00:00
|
|
|
#include <Storages/IStorage.h>
|
2020-07-22 17:13:05 +00:00
|
|
|
|
2021-01-21 09:01:35 +00:00
|
|
|
#include <Interpreters/RewriteSumIfFunctionVisitor.h>
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
|
|
|
extern const int LOGICAL_ERROR;
|
2021-09-11 22:41:37 +00:00
|
|
|
extern const int UNKNOWN_TYPE_OF_AST_NODE;
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
namespace
|
|
|
|
{
|
|
|
|
|
|
|
|
const std::unordered_set<String> possibly_injective_function_names
|
|
|
|
{
|
|
|
|
"dictGet",
|
|
|
|
"dictGetString",
|
|
|
|
"dictGetUInt8",
|
|
|
|
"dictGetUInt16",
|
|
|
|
"dictGetUInt32",
|
|
|
|
"dictGetUInt64",
|
|
|
|
"dictGetInt8",
|
|
|
|
"dictGetInt16",
|
|
|
|
"dictGetInt32",
|
|
|
|
"dictGetInt64",
|
|
|
|
"dictGetFloat32",
|
|
|
|
"dictGetFloat64",
|
|
|
|
"dictGetDate",
|
|
|
|
"dictGetDateTime"
|
|
|
|
};
|
|
|
|
|
|
|
|
/** You can not completely remove GROUP BY. Because if there were no aggregate functions, then it turns out that there will be no aggregation.
|
|
|
|
* Instead, leave `GROUP BY const`.
|
|
|
|
* Next, see deleting the constants in the analyzeAggregation method.
|
|
|
|
*/
|
|
|
|
void appendUnusedGroupByColumn(ASTSelectQuery * select_query, const NameSet & source_columns)
|
|
|
|
{
|
|
|
|
/// You must insert a constant that is not the name of the column in the table. Such a case is rare, but it happens.
|
2021-08-09 14:16:44 +00:00
|
|
|
/// Also start unused_column integer from source_columns.size() + 1, because lower numbers ([1, source_columns.size()])
|
|
|
|
/// might be in positional GROUP BY.
|
|
|
|
UInt64 unused_column = source_columns.size() + 1;
|
2020-07-22 17:13:05 +00:00
|
|
|
String unused_column_name = toString(unused_column);
|
|
|
|
|
|
|
|
while (source_columns.count(unused_column_name))
|
|
|
|
{
|
|
|
|
++unused_column;
|
|
|
|
unused_column_name = toString(unused_column);
|
|
|
|
}
|
|
|
|
|
|
|
|
select_query->setExpression(ASTSelectQuery::Expression::GROUP_BY, std::make_shared<ASTExpressionList>());
|
|
|
|
select_query->groupBy()->children.emplace_back(std::make_shared<ASTLiteral>(UInt64(unused_column)));
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Eliminates injective function calls and constant expressions from group by statement.
|
2021-06-01 12:20:52 +00:00
|
|
|
void optimizeGroupBy(ASTSelectQuery * select_query, const NameSet & source_columns, ContextPtr context)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
|
|
|
const FunctionFactory & function_factory = FunctionFactory::instance();
|
|
|
|
|
|
|
|
if (!select_query->groupBy())
|
|
|
|
{
|
|
|
|
// If there is a HAVING clause without GROUP BY, make sure we have some aggregation happen.
|
|
|
|
if (select_query->having())
|
|
|
|
appendUnusedGroupByColumn(select_query, source_columns);
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
|
|
|
const auto is_literal = [] (const ASTPtr & ast) -> bool
|
|
|
|
{
|
|
|
|
return ast->as<ASTLiteral>();
|
|
|
|
};
|
|
|
|
|
|
|
|
auto & group_exprs = select_query->groupBy()->children;
|
|
|
|
|
|
|
|
/// removes expression at index idx by making it last one and calling .pop_back()
|
|
|
|
const auto remove_expr_at_index = [&group_exprs] (const size_t idx)
|
|
|
|
{
|
|
|
|
if (idx < group_exprs.size() - 1)
|
|
|
|
std::swap(group_exprs[idx], group_exprs.back());
|
|
|
|
|
|
|
|
group_exprs.pop_back();
|
|
|
|
};
|
|
|
|
|
2021-08-09 14:16:44 +00:00
|
|
|
const auto & settings = context->getSettingsRef();
|
|
|
|
|
2020-07-22 17:13:05 +00:00
|
|
|
/// iterate over each GROUP BY expression, eliminate injective function calls and literals
|
|
|
|
for (size_t i = 0; i < group_exprs.size();)
|
|
|
|
{
|
|
|
|
if (const auto * function = group_exprs[i]->as<ASTFunction>())
|
|
|
|
{
|
|
|
|
/// assert function is injective
|
|
|
|
if (possibly_injective_function_names.count(function->name))
|
|
|
|
{
|
|
|
|
/// do not handle semantic errors here
|
|
|
|
if (function->arguments->children.size() < 2)
|
|
|
|
{
|
|
|
|
++i;
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
|
|
|
|
const auto * dict_name_ast = function->arguments->children[0]->as<ASTLiteral>();
|
|
|
|
const auto * attr_name_ast = function->arguments->children[1]->as<ASTLiteral>();
|
|
|
|
if (!dict_name_ast || !attr_name_ast)
|
|
|
|
{
|
|
|
|
++i;
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
|
|
|
|
const auto & dict_name = dict_name_ast->value.safeGet<String>();
|
|
|
|
const auto & attr_name = attr_name_ast->value.safeGet<String>();
|
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
const auto & dict_ptr = context->getExternalDictionariesLoader().getDictionary(dict_name, context);
|
2020-07-22 17:13:05 +00:00
|
|
|
if (!dict_ptr->isInjective(attr_name))
|
|
|
|
{
|
|
|
|
++i;
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
}
|
2020-10-09 07:41:28 +00:00
|
|
|
else if (!function_factory.get(function->name, context)->isInjective({}))
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
|
|
|
++i;
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
|
|
|
|
/// copy shared pointer to args in order to ensure lifetime
|
|
|
|
auto args_ast = function->arguments;
|
|
|
|
|
|
|
|
/** remove function call and take a step back to ensure
|
|
|
|
* next iteration does not skip not yet processed data
|
|
|
|
*/
|
|
|
|
remove_expr_at_index(i);
|
|
|
|
|
|
|
|
/// copy non-literal arguments
|
|
|
|
std::remove_copy_if(
|
|
|
|
std::begin(args_ast->children), std::end(args_ast->children),
|
|
|
|
std::back_inserter(group_exprs), is_literal
|
|
|
|
);
|
|
|
|
}
|
|
|
|
else if (is_literal(group_exprs[i]))
|
|
|
|
{
|
2021-08-19 15:47:26 +00:00
|
|
|
bool keep_position = false;
|
|
|
|
if (settings.enable_positional_arguments)
|
|
|
|
{
|
2021-08-20 09:08:39 +00:00
|
|
|
const auto & value = group_exprs[i]->as<ASTLiteral>()->value;
|
2021-08-19 15:47:26 +00:00
|
|
|
if (value.getType() == Field::Types::UInt64)
|
|
|
|
{
|
|
|
|
auto pos = value.get<UInt64>();
|
|
|
|
if (pos > 0 && pos <= select_query->children.size())
|
|
|
|
keep_position = true;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (keep_position)
|
|
|
|
++i;
|
|
|
|
else
|
|
|
|
remove_expr_at_index(i);
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
/// if neither a function nor literal - advance to next expression
|
|
|
|
++i;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (group_exprs.empty())
|
|
|
|
appendUnusedGroupByColumn(select_query, source_columns);
|
|
|
|
}
|
|
|
|
|
|
|
|
struct GroupByKeysInfo
|
|
|
|
{
|
2020-11-12 19:50:01 +00:00
|
|
|
NameSet key_names; ///set of keys' short names
|
2020-07-22 17:13:05 +00:00
|
|
|
bool has_function = false;
|
|
|
|
};
|
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
GroupByKeysInfo getGroupByKeysInfo(const ASTs & group_by_keys)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
|
|
|
GroupByKeysInfo data;
|
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
/// filling set with short names of keys
|
2020-11-13 16:30:10 +00:00
|
|
|
for (const auto & group_key : group_by_keys)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
|
|
|
if (group_key->as<ASTFunction>())
|
|
|
|
data.has_function = true;
|
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
data.key_names.insert(group_key->getColumnName());
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
return data;
|
|
|
|
}
|
|
|
|
|
|
|
|
///eliminate functions of other GROUP BY keys
|
|
|
|
void optimizeGroupByFunctionKeys(ASTSelectQuery * select_query)
|
|
|
|
{
|
|
|
|
if (!select_query->groupBy())
|
|
|
|
return;
|
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
auto group_by = select_query->groupBy();
|
|
|
|
const auto & group_by_keys = group_by->children;
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
ASTs modified; ///result
|
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
GroupByKeysInfo group_by_keys_data = getGroupByKeysInfo(group_by_keys);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
if (!group_by_keys_data.has_function)
|
2020-07-22 17:13:05 +00:00
|
|
|
return;
|
|
|
|
|
|
|
|
GroupByFunctionKeysVisitor::Data visitor_data{group_by_keys_data.key_names};
|
2020-11-12 19:50:01 +00:00
|
|
|
GroupByFunctionKeysVisitor(visitor_data).visit(group_by);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
modified.reserve(group_by_keys.size());
|
2020-07-22 17:13:05 +00:00
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
/// filling the result
|
2020-11-12 21:01:53 +00:00
|
|
|
for (const auto & group_key : group_by_keys)
|
2020-11-12 19:50:01 +00:00
|
|
|
if (group_by_keys_data.key_names.count(group_key->getColumnName()))
|
|
|
|
modified.push_back(group_key);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
2020-11-12 19:50:01 +00:00
|
|
|
/// modifying the input
|
|
|
|
group_by->children = modified;
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
/// Eliminates min/max/any-aggregators of functions of GROUP BY keys
|
2020-10-13 05:30:56 +00:00
|
|
|
void optimizeAggregateFunctionsOfGroupByKeys(ASTSelectQuery * select_query, ASTPtr & node)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
|
|
|
if (!select_query->groupBy())
|
|
|
|
return;
|
|
|
|
|
2020-11-13 16:30:10 +00:00
|
|
|
const auto & group_by_keys = select_query->groupBy()->children;
|
2020-11-12 19:50:01 +00:00
|
|
|
GroupByKeysInfo group_by_keys_data = getGroupByKeysInfo(group_by_keys);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
SelectAggregateFunctionOfGroupByKeysVisitor::Data visitor_data{group_by_keys_data.key_names};
|
2020-10-13 05:30:56 +00:00
|
|
|
SelectAggregateFunctionOfGroupByKeysVisitor(visitor_data).visit(node);
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
/// Remove duplicate items from ORDER BY.
|
|
|
|
void optimizeDuplicatesInOrderBy(const ASTSelectQuery * select_query)
|
|
|
|
{
|
|
|
|
if (!select_query->orderBy())
|
|
|
|
return;
|
|
|
|
|
|
|
|
/// Make unique sorting conditions.
|
|
|
|
using NameAndLocale = std::pair<String, String>;
|
|
|
|
std::set<NameAndLocale> elems_set;
|
|
|
|
|
|
|
|
ASTs & elems = select_query->orderBy()->children;
|
|
|
|
ASTs unique_elems;
|
|
|
|
unique_elems.reserve(elems.size());
|
|
|
|
|
|
|
|
for (const auto & elem : elems)
|
|
|
|
{
|
|
|
|
String name = elem->children.front()->getColumnName();
|
|
|
|
const auto & order_by_elem = elem->as<ASTOrderByElement &>();
|
|
|
|
|
2021-09-11 23:55:53 +00:00
|
|
|
if (order_by_elem.with_fill /// Always keep elements WITH FILL as they affects other.
|
|
|
|
|| elems_set.emplace(name, order_by_elem.collation ? order_by_elem.collation->getColumnName() : "").second)
|
2020-07-22 17:13:05 +00:00
|
|
|
unique_elems.emplace_back(elem);
|
|
|
|
}
|
|
|
|
|
|
|
|
if (unique_elems.size() < elems.size())
|
|
|
|
elems = std::move(unique_elems);
|
|
|
|
}
|
|
|
|
|
2020-08-20 17:04:42 +00:00
|
|
|
/// Optimize duplicate ORDER BY
|
2021-06-01 12:20:52 +00:00
|
|
|
void optimizeDuplicateOrderBy(ASTPtr & query, ContextPtr context)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
|
|
|
DuplicateOrderByVisitor::Data order_by_data{context};
|
|
|
|
DuplicateOrderByVisitor(order_by_data).visit(query);
|
2020-08-20 17:04:42 +00:00
|
|
|
}
|
|
|
|
|
2020-08-20 20:50:53 +00:00
|
|
|
/// Return simple subselect (without UNIONs or JOINs or SETTINGS) if any
|
2020-08-20 17:04:42 +00:00
|
|
|
const ASTSelectQuery * getSimpleSubselect(const ASTSelectQuery & select)
|
|
|
|
{
|
2020-08-20 18:09:48 +00:00
|
|
|
if (!select.tables())
|
|
|
|
return nullptr;
|
|
|
|
|
2020-08-20 17:04:42 +00:00
|
|
|
const auto & tables = select.tables()->children;
|
|
|
|
if (tables.empty() || tables.size() != 1)
|
|
|
|
return nullptr;
|
|
|
|
|
|
|
|
const auto & ast_table_expression = tables[0]->as<ASTTablesInSelectQueryElement>()->table_expression;
|
|
|
|
if (!ast_table_expression)
|
|
|
|
return nullptr;
|
|
|
|
|
|
|
|
const auto & table_expression = ast_table_expression->as<ASTTableExpression>();
|
|
|
|
if (!table_expression->subquery)
|
|
|
|
return nullptr;
|
|
|
|
|
|
|
|
const auto & subquery = table_expression->subquery->as<ASTSubquery>();
|
|
|
|
if (!subquery || subquery->children.size() != 1)
|
|
|
|
return nullptr;
|
|
|
|
|
|
|
|
const auto & subselect_union = subquery->children[0]->as<ASTSelectWithUnionQuery>();
|
|
|
|
if (!subselect_union || !subselect_union->list_of_selects ||
|
|
|
|
subselect_union->list_of_selects->children.size() != 1)
|
|
|
|
return nullptr;
|
|
|
|
|
2020-08-20 20:50:53 +00:00
|
|
|
const auto & subselect = subselect_union->list_of_selects->children[0]->as<ASTSelectQuery>();
|
|
|
|
if (subselect && subselect->settings())
|
|
|
|
return nullptr;
|
|
|
|
|
|
|
|
return subselect;
|
2020-08-20 17:04:42 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
std::unordered_set<String> getDistinctNames(const ASTSelectQuery & select)
|
|
|
|
{
|
|
|
|
if (!select.select() || select.select()->children.empty())
|
|
|
|
return {};
|
|
|
|
|
|
|
|
std::unordered_set<String> names;
|
|
|
|
std::unordered_set<String> implicit_distinct;
|
|
|
|
|
|
|
|
if (!select.distinct)
|
|
|
|
{
|
|
|
|
/// SELECT a, b FROM (SELECT DISTINCT a FROM ...)
|
|
|
|
if (const ASTSelectQuery * subselect = getSimpleSubselect(select))
|
|
|
|
implicit_distinct = getDistinctNames(*subselect);
|
|
|
|
|
|
|
|
if (implicit_distinct.empty())
|
|
|
|
return {};
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Extract result column names (prefer aliases, ignore table name)
|
|
|
|
for (const auto & id : select.select()->children)
|
|
|
|
{
|
|
|
|
String alias = id->tryGetAlias();
|
|
|
|
|
|
|
|
if (const auto * identifier = id->as<ASTIdentifier>())
|
|
|
|
{
|
2020-08-20 22:05:06 +00:00
|
|
|
const String & name = identifier->shortName();
|
2020-08-20 17:04:42 +00:00
|
|
|
|
|
|
|
if (select.distinct || implicit_distinct.count(name))
|
|
|
|
{
|
|
|
|
if (alias.empty())
|
|
|
|
names.insert(name);
|
|
|
|
else
|
|
|
|
names.insert(alias);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
else if (select.distinct && !alias.empty())
|
|
|
|
{
|
|
|
|
/// It's not possible to use getAliasOrColumnName() cause name is context specific (function arguments)
|
|
|
|
names.insert(alias);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-08-20 18:51:22 +00:00
|
|
|
/// SELECT a FROM (SELECT DISTINCT a, b FROM ...)
|
2020-08-20 19:04:46 +00:00
|
|
|
if (!select.distinct && names.size() != implicit_distinct.size())
|
2020-08-20 18:51:22 +00:00
|
|
|
return {};
|
|
|
|
|
2020-08-20 17:04:42 +00:00
|
|
|
return names;
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Remove DISTINCT from query if columns are known as DISTINCT from subquery
|
|
|
|
void optimizeDuplicateDistinct(ASTSelectQuery & select)
|
|
|
|
{
|
|
|
|
if (!select.select() || select.select()->children.empty())
|
|
|
|
return;
|
|
|
|
|
|
|
|
const ASTSelectQuery * subselect = getSimpleSubselect(select);
|
|
|
|
if (!subselect)
|
|
|
|
return;
|
|
|
|
|
|
|
|
std::unordered_set<String> distinct_names = getDistinctNames(*subselect);
|
|
|
|
std::unordered_set<String> selected_names;
|
|
|
|
|
|
|
|
/// Check source column names from select list (ignore aliases and table names)
|
|
|
|
for (const auto & id : select.select()->children)
|
|
|
|
{
|
|
|
|
const auto * identifier = id->as<ASTIdentifier>();
|
|
|
|
if (!identifier)
|
|
|
|
return;
|
|
|
|
|
|
|
|
String name = identifier->shortName();
|
|
|
|
if (!distinct_names.count(name))
|
|
|
|
return; /// Not a distinct column, keep DISTINCT for it.
|
|
|
|
|
|
|
|
selected_names.insert(name);
|
|
|
|
}
|
|
|
|
|
|
|
|
/// select columns list != distinct columns list
|
|
|
|
/// SELECT DISTINCT a FROM (SELECT DISTINCT a, b FROM ...)) -- cannot remove DISTINCT
|
|
|
|
if (selected_names.size() != distinct_names.size())
|
|
|
|
return;
|
|
|
|
|
|
|
|
select.distinct = false;
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
/// Replace monotonous functions in ORDER BY if they don't participate in GROUP BY expression,
|
|
|
|
/// has a single argument and not an aggregate functions.
|
2021-06-01 12:20:52 +00:00
|
|
|
void optimizeMonotonousFunctionsInOrderBy(ASTSelectQuery * select_query, ContextPtr context,
|
2020-11-12 23:33:26 +00:00
|
|
|
const TablesWithColumns & tables_with_columns,
|
|
|
|
const Names & sorting_key_columns)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
|
|
|
auto order_by = select_query->orderBy();
|
|
|
|
if (!order_by)
|
|
|
|
return;
|
|
|
|
|
2021-09-11 22:41:37 +00:00
|
|
|
for (const auto & child : order_by->children)
|
|
|
|
{
|
|
|
|
auto * order_by_element = child->as<ASTOrderByElement>();
|
|
|
|
|
|
|
|
if (!order_by_element || order_by_element->children.empty())
|
|
|
|
throw Exception("Bad ORDER BY expression AST", ErrorCodes::UNKNOWN_TYPE_OF_AST_NODE);
|
|
|
|
|
|
|
|
if (order_by_element->with_fill)
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
2020-07-22 17:13:05 +00:00
|
|
|
std::unordered_set<String> group_by_hashes;
|
|
|
|
if (auto group_by = select_query->groupBy())
|
|
|
|
{
|
|
|
|
for (auto & elem : group_by->children)
|
|
|
|
{
|
|
|
|
auto hash = elem->getTreeHash();
|
|
|
|
String key = toString(hash.first) + '_' + toString(hash.second);
|
|
|
|
group_by_hashes.insert(key);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-11-12 23:33:26 +00:00
|
|
|
bool is_sorting_key_prefix = true;
|
|
|
|
for (size_t i = 0; i < order_by->children.size(); ++i)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
2020-11-12 23:33:26 +00:00
|
|
|
auto * order_by_element = order_by->children[i]->as<ASTOrderByElement>();
|
2021-09-11 22:41:37 +00:00
|
|
|
|
2020-07-22 17:13:05 +00:00
|
|
|
auto & ast_func = order_by_element->children[0];
|
|
|
|
if (!ast_func->as<ASTFunction>())
|
|
|
|
continue;
|
|
|
|
|
2020-11-12 23:33:26 +00:00
|
|
|
if (i >= sorting_key_columns.size() || ast_func->getColumnName() != sorting_key_columns[i])
|
|
|
|
is_sorting_key_prefix = false;
|
|
|
|
|
|
|
|
/// If order by expression matches the sorting key, do not remove
|
|
|
|
/// functions to allow execute reading in order of key.
|
|
|
|
if (is_sorting_key_prefix)
|
|
|
|
continue;
|
|
|
|
|
2020-07-22 17:13:05 +00:00
|
|
|
MonotonicityCheckVisitor::Data data{tables_with_columns, context, group_by_hashes};
|
|
|
|
MonotonicityCheckVisitor(data).visit(ast_func);
|
|
|
|
|
|
|
|
if (!data.isRejected())
|
|
|
|
{
|
|
|
|
ast_func = data.identifier->clone();
|
|
|
|
ast_func->setAlias("");
|
|
|
|
if (!data.monotonicity.is_positive)
|
|
|
|
order_by_element->direction *= -1;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-08-08 01:01:47 +00:00
|
|
|
/// If ORDER BY has argument x followed by f(x) transforms it to ORDER BY x.
|
2020-07-22 17:13:05 +00:00
|
|
|
/// Optimize ORDER BY x, y, f(x), g(x, y), f(h(x)), t(f(x), g(x)) into ORDER BY x, y
|
|
|
|
/// in case if f(), g(), h(), t() are deterministic (in scope of query).
|
|
|
|
/// Don't optimize ORDER BY f(x), g(x), x even if f(x) is bijection for x or g(x).
|
2021-06-01 12:20:52 +00:00
|
|
|
void optimizeRedundantFunctionsInOrderBy(const ASTSelectQuery * select_query, ContextPtr context)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
|
|
|
const auto & order_by = select_query->orderBy();
|
|
|
|
if (!order_by)
|
|
|
|
return;
|
|
|
|
|
2021-09-11 22:41:37 +00:00
|
|
|
for (const auto & child : order_by->children)
|
|
|
|
{
|
|
|
|
auto * order_by_element = child->as<ASTOrderByElement>();
|
|
|
|
|
|
|
|
if (!order_by_element || order_by_element->children.empty())
|
|
|
|
throw Exception("Bad ORDER BY expression AST", ErrorCodes::UNKNOWN_TYPE_OF_AST_NODE);
|
|
|
|
|
|
|
|
if (order_by_element->with_fill)
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
2020-07-22 17:13:05 +00:00
|
|
|
std::unordered_set<String> prev_keys;
|
|
|
|
ASTs modified;
|
|
|
|
modified.reserve(order_by->children.size());
|
|
|
|
|
|
|
|
for (auto & order_by_element : order_by->children)
|
|
|
|
{
|
|
|
|
/// Order by contains ASTOrderByElement as children and meaning item only as a grand child.
|
|
|
|
ASTPtr & name_or_function = order_by_element->children[0];
|
|
|
|
|
|
|
|
if (name_or_function->as<ASTFunction>())
|
|
|
|
{
|
|
|
|
if (!prev_keys.empty())
|
|
|
|
{
|
|
|
|
RedundantFunctionsInOrderByVisitor::Data data{prev_keys, context};
|
|
|
|
RedundantFunctionsInOrderByVisitor(data).visit(name_or_function);
|
|
|
|
if (data.redundant)
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
/// @note Leave duplicate keys unchanged. They would be removed in optimizeDuplicatesInOrderBy()
|
|
|
|
if (auto * identifier = name_or_function->as<ASTIdentifier>())
|
|
|
|
prev_keys.emplace(getIdentifierName(identifier));
|
|
|
|
|
|
|
|
modified.push_back(order_by_element);
|
|
|
|
}
|
|
|
|
|
|
|
|
if (modified.size() < order_by->children.size())
|
|
|
|
order_by->children = std::move(modified);
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Remove duplicate items from LIMIT BY.
|
|
|
|
void optimizeLimitBy(const ASTSelectQuery * select_query)
|
|
|
|
{
|
|
|
|
if (!select_query->limitBy())
|
|
|
|
return;
|
|
|
|
|
|
|
|
std::set<String> elems_set;
|
|
|
|
|
|
|
|
ASTs & elems = select_query->limitBy()->children;
|
|
|
|
ASTs unique_elems;
|
|
|
|
unique_elems.reserve(elems.size());
|
|
|
|
|
|
|
|
for (const auto & elem : elems)
|
|
|
|
{
|
|
|
|
if (elems_set.emplace(elem->getColumnName()).second)
|
|
|
|
unique_elems.emplace_back(elem);
|
|
|
|
}
|
|
|
|
|
|
|
|
if (unique_elems.size() < elems.size())
|
|
|
|
elems = std::move(unique_elems);
|
|
|
|
}
|
|
|
|
|
|
|
|
/// Remove duplicated columns from USING(...).
|
|
|
|
void optimizeUsing(const ASTSelectQuery * select_query)
|
|
|
|
{
|
|
|
|
if (!select_query->join())
|
|
|
|
return;
|
|
|
|
|
|
|
|
const auto * table_join = select_query->join()->table_join->as<ASTTableJoin>();
|
|
|
|
if (!(table_join && table_join->using_expression_list))
|
|
|
|
return;
|
|
|
|
|
|
|
|
ASTs & expression_list = table_join->using_expression_list->children;
|
|
|
|
ASTs uniq_expressions_list;
|
|
|
|
|
|
|
|
std::set<String> expressions_names;
|
|
|
|
|
|
|
|
for (const auto & expression : expression_list)
|
|
|
|
{
|
|
|
|
auto expression_name = expression->getAliasOrColumnName();
|
|
|
|
if (expressions_names.find(expression_name) == expressions_names.end())
|
|
|
|
{
|
|
|
|
uniq_expressions_list.push_back(expression);
|
|
|
|
expressions_names.insert(expression_name);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (uniq_expressions_list.size() < expression_list.size())
|
|
|
|
expression_list = uniq_expressions_list;
|
|
|
|
}
|
|
|
|
|
|
|
|
void optimizeAggregationFunctions(ASTPtr & query)
|
|
|
|
{
|
|
|
|
/// Move arithmetic operations out of aggregation functions
|
|
|
|
ArithmeticOperationsInAgrFuncVisitor::Data data;
|
|
|
|
ArithmeticOperationsInAgrFuncVisitor(data).visit(query);
|
|
|
|
}
|
|
|
|
|
2020-07-23 15:15:22 +00:00
|
|
|
void optimizeAnyFunctions(ASTPtr & query)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
2020-07-23 15:15:22 +00:00
|
|
|
RewriteAnyFunctionVisitor::Data data = {};
|
|
|
|
RewriteAnyFunctionVisitor(data).visit(query);
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
|
2021-01-21 09:01:35 +00:00
|
|
|
void optimizeSumIfFunctions(ASTPtr & query)
|
|
|
|
{
|
|
|
|
RewriteSumIfFunctionVisitor::Data data = {};
|
|
|
|
RewriteSumIfFunctionVisitor(data).visit(query);
|
|
|
|
}
|
|
|
|
|
2021-02-07 07:22:51 +00:00
|
|
|
void optimizeCountConstantAndSumOne(ASTPtr & query)
|
|
|
|
{
|
|
|
|
RewriteCountVariantsVisitor::visit(query);
|
|
|
|
}
|
|
|
|
|
2021-01-21 09:01:35 +00:00
|
|
|
|
2021-06-01 12:20:52 +00:00
|
|
|
void optimizeInjectiveFunctionsInsideUniq(ASTPtr & query, ContextPtr context)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
RemoveInjectiveFunctionsVisitor::Data data(context);
|
2020-07-22 17:13:05 +00:00
|
|
|
RemoveInjectiveFunctionsVisitor(data).visit(query);
|
|
|
|
}
|
|
|
|
|
|
|
|
void transformIfStringsIntoEnum(ASTPtr & query)
|
|
|
|
{
|
|
|
|
std::unordered_set<String> function_names = {"if", "transform"};
|
|
|
|
std::unordered_set<String> used_as_argument;
|
|
|
|
|
|
|
|
FindUsedFunctionsVisitor::Data used_data{function_names, used_as_argument};
|
|
|
|
FindUsedFunctionsVisitor(used_data).visit(query);
|
|
|
|
|
|
|
|
ConvertStringsToEnumVisitor::Data convert_data{used_as_argument};
|
|
|
|
ConvertStringsToEnumVisitor(convert_data).visit(query);
|
|
|
|
}
|
|
|
|
|
2021-05-21 23:22:22 +00:00
|
|
|
void optimizeFunctionsToSubcolumns(ASTPtr & query, const StorageMetadataPtr & metadata_snapshot)
|
2021-05-21 18:48:19 +00:00
|
|
|
{
|
2021-05-21 23:22:22 +00:00
|
|
|
RewriteFunctionToSubcolumnVisitor::Data data{metadata_snapshot};
|
2021-05-21 18:48:19 +00:00
|
|
|
RewriteFunctionToSubcolumnVisitor(data).visit(query);
|
|
|
|
}
|
|
|
|
|
2021-07-21 06:43:40 +00:00
|
|
|
/// Rewrites multi quantile()() functions with the same arguments to quantiles()()[]
|
|
|
|
/// eg:SELECT quantile(0.5)(x), quantile(0.9)(x), quantile(0.95)(x) FROM...
|
|
|
|
/// rewrite to : SELECT quantiles(0.5, 0.9, 0.95)(x)[1], quantiles(0.5, 0.9, 0.95)(x)[2], quantiles(0.5, 0.9, 0.95)(x)[3] FROM ...
|
2021-09-14 12:12:21 +00:00
|
|
|
void fuseCandidate(std::unordered_map<String, GatherFunctionQuantileData::FuseQuantileAggregatesData> & fuse_quantile)
|
2021-07-21 06:43:40 +00:00
|
|
|
{
|
2021-09-15 12:15:42 +00:00
|
|
|
for (auto & candidate : fuse_quantile)
|
2021-07-21 06:43:40 +00:00
|
|
|
{
|
|
|
|
String func_name = candidate.first;
|
2021-09-15 12:15:42 +00:00
|
|
|
auto & args_to_functions = candidate.second;
|
2021-07-21 06:43:40 +00:00
|
|
|
|
2021-08-25 08:24:53 +00:00
|
|
|
// Try to fuse multiply `quantile*` Function to plural
|
2021-08-30 03:02:02 +00:00
|
|
|
for (auto it : args_to_functions.arg_map_function)
|
2021-07-21 06:43:40 +00:00
|
|
|
{
|
2021-08-25 08:24:53 +00:00
|
|
|
std::vector<ASTPtr *> & functions = it.second;
|
2021-09-15 12:15:42 +00:00
|
|
|
if (functions.size() < 2)
|
|
|
|
continue;
|
|
|
|
|
|
|
|
const auto & common_arguments = (*functions[0])->as<ASTFunction>()->arguments->children;
|
|
|
|
auto func_base = makeASTFunction(GatherFunctionQuantileData::getFusedName(func_name));
|
|
|
|
func_base->arguments->children = common_arguments;
|
|
|
|
func_base->parameters = std::make_shared<ASTExpressionList>();
|
|
|
|
|
|
|
|
for (const auto * ast : functions)
|
2021-07-21 06:43:40 +00:00
|
|
|
{
|
2021-09-15 12:15:42 +00:00
|
|
|
assert(ast && *ast);
|
|
|
|
const auto * func = (*ast)->as<ASTFunction>();
|
|
|
|
assert(func && func->parameters->as<ASTExpressionList>());
|
|
|
|
const ASTs & parameters = func->parameters->as<ASTExpressionList &>().children;
|
|
|
|
if (parameters.size() != 1)
|
|
|
|
throw Exception(ErrorCodes::LOGICAL_ERROR, "Aggregate function '{}' requires one parameter", func_name);
|
|
|
|
func_base->parameters->children.push_back(parameters[0]);
|
|
|
|
}
|
2021-07-21 06:43:40 +00:00
|
|
|
|
2021-09-15 12:15:42 +00:00
|
|
|
for (size_t i = 0; i < functions.size(); ++i)
|
|
|
|
{
|
|
|
|
std::shared_ptr<ASTFunction> ast_new = makeASTFunction("arrayElement", func_base, std::make_shared<ASTLiteral>(i + 1));
|
|
|
|
if (const auto & alias = (*functions[i])->tryGetAlias(); !alias.empty())
|
|
|
|
ast_new->setAlias(alias);
|
|
|
|
*functions[i] = ast_new;
|
2021-07-21 06:43:40 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
2021-09-15 12:15:42 +00:00
|
|
|
|
2021-07-21 06:43:40 +00:00
|
|
|
void optimizeFuseQuantileFunctions(ASTPtr & query)
|
|
|
|
{
|
|
|
|
GatherFunctionQuantileVisitor::Data data{};
|
|
|
|
GatherFunctionQuantileVisitor(data).visit(query);
|
2021-09-14 12:12:21 +00:00
|
|
|
fuseCandidate(data.fuse_quantile);
|
2021-07-21 06:43:40 +00:00
|
|
|
}
|
|
|
|
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void TreeOptimizer::optimizeIf(ASTPtr & query, Aliases & aliases, bool if_chain_to_multiif)
|
|
|
|
{
|
|
|
|
/// Optimize if with constant condition after constants was substituted instead of scalar subqueries.
|
|
|
|
OptimizeIfWithConstantConditionVisitor(aliases).visit(query);
|
|
|
|
|
|
|
|
if (if_chain_to_multiif)
|
|
|
|
OptimizeIfChainsVisitor().visit(query);
|
|
|
|
}
|
|
|
|
|
2021-05-21 18:48:19 +00:00
|
|
|
void TreeOptimizer::apply(ASTPtr & query, TreeRewriterResult & result,
|
2021-06-12 00:25:09 +00:00
|
|
|
const std::vector<TableWithColumnNamesAndTypes> & tables_with_columns, ContextPtr context)
|
2020-07-22 17:13:05 +00:00
|
|
|
{
|
2021-04-10 23:33:54 +00:00
|
|
|
const auto & settings = context->getSettingsRef();
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
auto * select_query = query->as<ASTSelectQuery>();
|
|
|
|
if (!select_query)
|
|
|
|
throw Exception("Select analyze for not select asts.", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
2021-05-21 23:22:22 +00:00
|
|
|
if (settings.optimize_functions_to_subcolumns && result.storage
|
|
|
|
&& result.storage->supportsSubcolumns() && result.metadata_snapshot)
|
|
|
|
optimizeFunctionsToSubcolumns(query, result.metadata_snapshot);
|
2021-05-21 18:48:19 +00:00
|
|
|
|
|
|
|
optimizeIf(query, result.aliases, settings.optimize_if_chain_to_multiif);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
/// Move arithmetic operations out of aggregation functions
|
|
|
|
if (settings.optimize_arithmetic_operations_in_aggregate_functions)
|
|
|
|
optimizeAggregationFunctions(query);
|
|
|
|
|
|
|
|
/// Push the predicate expression down to the subqueries.
|
2021-05-21 18:48:19 +00:00
|
|
|
result.rewrite_subqueries = PredicateExpressionsOptimizer(context, tables_with_columns, settings).optimize(*select_query);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
/// GROUP BY injective function elimination.
|
2021-05-21 18:48:19 +00:00
|
|
|
optimizeGroupBy(select_query, result.source_columns_set, context);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
/// GROUP BY functions of other keys elimination.
|
|
|
|
if (settings.optimize_group_by_function_keys)
|
|
|
|
optimizeGroupByFunctionKeys(select_query);
|
|
|
|
|
2020-07-23 15:15:22 +00:00
|
|
|
/// Move all operations out of any function
|
2020-07-22 17:13:05 +00:00
|
|
|
if (settings.optimize_move_functions_out_of_any)
|
2020-07-23 15:15:22 +00:00
|
|
|
optimizeAnyFunctions(query);
|
2021-01-21 09:01:35 +00:00
|
|
|
|
2021-02-07 07:22:51 +00:00
|
|
|
if (settings.optimize_normalize_count_variants)
|
|
|
|
optimizeCountConstantAndSumOne(query);
|
|
|
|
|
2021-01-21 09:01:35 +00:00
|
|
|
if (settings.optimize_rewrite_sum_if_to_count_if)
|
|
|
|
optimizeSumIfFunctions(query);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
/// Remove injective functions inside uniq
|
|
|
|
if (settings.optimize_injective_functions_inside_uniq)
|
|
|
|
optimizeInjectiveFunctionsInsideUniq(query, context);
|
|
|
|
|
|
|
|
/// Eliminate min/max/any aggregators of functions of GROUP BY keys
|
2020-10-26 13:44:46 +00:00
|
|
|
if (settings.optimize_aggregators_of_group_by_keys
|
|
|
|
&& !select_query->group_by_with_totals
|
|
|
|
&& !select_query->group_by_with_rollup
|
|
|
|
&& !select_query->group_by_with_cube)
|
|
|
|
{
|
2020-10-13 05:30:56 +00:00
|
|
|
optimizeAggregateFunctionsOfGroupByKeys(select_query, query);
|
2020-10-26 13:44:46 +00:00
|
|
|
}
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
/// Remove duplicate ORDER BY and DISTINCT from subqueries.
|
|
|
|
if (settings.optimize_duplicate_order_by_and_distinct)
|
2020-08-20 17:04:42 +00:00
|
|
|
{
|
|
|
|
optimizeDuplicateOrderBy(query, context);
|
2020-08-20 20:50:53 +00:00
|
|
|
|
|
|
|
/// DISTINCT has special meaning in Distributed query with enabled distributed_group_by_no_merge
|
|
|
|
/// TODO: disable Distributed/remote() tables only
|
|
|
|
if (!settings.distributed_group_by_no_merge)
|
|
|
|
optimizeDuplicateDistinct(*select_query);
|
2020-08-20 17:04:42 +00:00
|
|
|
}
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
/// Remove functions from ORDER BY if its argument is also in ORDER BY
|
|
|
|
if (settings.optimize_redundant_functions_in_order_by)
|
|
|
|
optimizeRedundantFunctionsInOrderBy(select_query, context);
|
|
|
|
|
|
|
|
/// Replace monotonous functions with its argument
|
|
|
|
if (settings.optimize_monotonous_functions_in_order_by)
|
2020-11-12 23:33:26 +00:00
|
|
|
optimizeMonotonousFunctionsInOrderBy(select_query, context, tables_with_columns,
|
2021-05-21 18:48:19 +00:00
|
|
|
result.metadata_snapshot ? result.metadata_snapshot->getSortingKeyColumns() : Names{});
|
2020-11-12 23:33:26 +00:00
|
|
|
|
|
|
|
/// Remove duplicate items from ORDER BY.
|
|
|
|
/// Execute it after all order by optimizations,
|
|
|
|
/// because they can produce duplicated columns.
|
|
|
|
optimizeDuplicatesInOrderBy(select_query);
|
2020-07-22 17:13:05 +00:00
|
|
|
|
|
|
|
/// If function "if" has String-type arguments, transform them into enum
|
|
|
|
if (settings.optimize_if_transform_strings_to_enum)
|
|
|
|
transformIfStringsIntoEnum(query);
|
|
|
|
|
|
|
|
/// Remove duplicated elements from LIMIT BY clause.
|
|
|
|
optimizeLimitBy(select_query);
|
|
|
|
|
|
|
|
/// Remove duplicated columns from USING(...).
|
|
|
|
optimizeUsing(select_query);
|
2021-07-21 06:43:40 +00:00
|
|
|
|
2021-09-14 12:27:12 +00:00
|
|
|
if (settings.optimize_syntax_fuse_functions)
|
2021-07-21 06:43:40 +00:00
|
|
|
optimizeFuseQuantileFunctions(query);
|
2020-07-22 17:13:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
}
|