2015-09-04 21:39:10 +00:00
|
|
|
#include <Poco/Util/Application.h>
|
2016-11-20 12:43:20 +00:00
|
|
|
#include <Poco/String.h>
|
2015-09-04 21:39:10 +00:00
|
|
|
|
2018-10-18 15:03:14 +00:00
|
|
|
#include <Core/Block.h>
|
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Parsers/ASTFunction.h>
|
|
|
|
#include <Parsers/ASTIdentifier.h>
|
|
|
|
#include <Parsers/ASTLiteral.h>
|
|
|
|
#include <Parsers/ASTAsterisk.h>
|
2017-06-19 03:05:05 +00:00
|
|
|
#include <Parsers/ASTQualifiedAsterisk.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Parsers/ASTExpressionList.h>
|
|
|
|
#include <Parsers/ASTSelectQuery.h>
|
2018-02-26 09:05:06 +00:00
|
|
|
#include <Parsers/ASTSelectWithUnionQuery.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Parsers/ASTSubquery.h>
|
|
|
|
#include <Parsers/ASTOrderByElement.h>
|
2018-02-05 14:03:01 +00:00
|
|
|
#include <Parsers/formatAST.h>
|
2018-10-09 14:32:11 +00:00
|
|
|
#include <Parsers/DumpASTNode.h>
|
2013-05-24 10:49:19 +00:00
|
|
|
|
2017-12-25 00:38:45 +00:00
|
|
|
#include <DataTypes/DataTypeNullable.h>
|
2017-12-25 18:58:39 +00:00
|
|
|
#include <DataTypes/NestedUtils.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <DataTypes/DataTypesNumber.h>
|
2018-08-28 14:40:07 +00:00
|
|
|
|
2018-10-12 15:41:28 +00:00
|
|
|
#include <Columns/IColumn.h>
|
2013-05-24 10:49:19 +00:00
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
#include <Interpreters/InterpreterSelectWithUnionQuery.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/ExpressionAnalyzer.h>
|
|
|
|
#include <Interpreters/ExpressionActions.h>
|
|
|
|
#include <Interpreters/InJoinSubqueriesPreprocessor.h>
|
|
|
|
#include <Interpreters/LogicalExpressionsOptimizer.h>
|
2018-03-04 16:15:31 +00:00
|
|
|
#include <Interpreters/PredicateExpressionsOptimizer.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/ExternalDictionaries.h>
|
|
|
|
#include <Interpreters/Set.h>
|
|
|
|
#include <Interpreters/Join.h>
|
2018-09-27 19:25:18 +00:00
|
|
|
#include <Interpreters/TranslateQualifiedNamesVisitor.h>
|
2018-10-11 19:28:59 +00:00
|
|
|
#include <Interpreters/ExecuteScalarSubqueriesVisitor.h>
|
2013-05-24 10:49:19 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <AggregateFunctions/AggregateFunctionFactory.h>
|
2017-07-10 23:30:17 +00:00
|
|
|
#include <AggregateFunctions/parseAggregateFunctionParameters.h>
|
2015-04-16 06:12:35 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Storages/StorageDistributed.h>
|
|
|
|
#include <Storages/StorageMemory.h>
|
|
|
|
#include <Storages/StorageJoin.h>
|
2014-03-14 14:52:48 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <DataStreams/LazyBlockInputStream.h>
|
|
|
|
#include <DataStreams/copyData.h>
|
2014-03-14 14:52:48 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Dictionaries/IDictionary.h>
|
2015-05-13 16:11:07 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/typeid_cast.h>
|
2018-01-15 19:07:47 +00:00
|
|
|
#include <Common/StringUtils/StringUtils.h>
|
2014-06-26 00:58:14 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Parsers/formatAST.h>
|
2013-05-24 10:49:19 +00:00
|
|
|
|
2017-06-06 17:18:32 +00:00
|
|
|
#include <ext/range.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <DataTypes/DataTypeFactory.h>
|
2018-02-06 19:34:53 +00:00
|
|
|
#include <Functions/FunctionsMiscellaneous.h>
|
2018-07-23 15:54:10 +00:00
|
|
|
#include <Parsers/queryToString.h>
|
2018-07-26 12:58:23 +00:00
|
|
|
#include <Parsers/ExpressionListParsers.h>
|
|
|
|
#include <Parsers/parseQuery.h>
|
2018-07-27 16:21:43 +00:00
|
|
|
#include <Parsers/queryToString.h>
|
2018-10-12 15:41:28 +00:00
|
|
|
#include <Interpreters/interpretSubquery.h>
|
2018-10-30 16:31:21 +00:00
|
|
|
#include <Interpreters/DatabaseAndTableWithAlias.h>
|
2018-09-12 05:41:09 +00:00
|
|
|
#include <Interpreters/QueryNormalizer.h>
|
2018-10-16 19:00:05 +00:00
|
|
|
|
2018-09-28 10:52:08 +00:00
|
|
|
#include <Interpreters/QueryAliasesVisitor.h>
|
2018-10-12 15:41:28 +00:00
|
|
|
#include <Interpreters/ActionsVisitor.h>
|
2018-10-16 19:00:05 +00:00
|
|
|
#include <Interpreters/ExternalTablesVisitor.h>
|
|
|
|
#include <Interpreters/GlobalSubqueriesVisitor.h>
|
|
|
|
#include <Interpreters/ArrayJoinedColumnsVisitor.h>
|
|
|
|
#include <Interpreters/RequiredSourceColumnsVisitor.h>
|
2013-05-24 10:49:19 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2018-10-10 16:23:27 +00:00
|
|
|
using LogAST = DebugASTLog<false>; /// set to true to enable logs
|
2018-10-09 14:32:11 +00:00
|
|
|
|
|
|
|
|
2016-01-11 21:46:36 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2018-04-05 20:52:01 +00:00
|
|
|
extern const int BAD_ARGUMENTS;
|
2017-04-01 07:20:54 +00:00
|
|
|
extern const int MULTIPLE_EXPRESSIONS_FOR_ALIAS;
|
|
|
|
extern const int UNKNOWN_IDENTIFIER;
|
|
|
|
extern const int CYCLIC_ALIASES;
|
|
|
|
extern const int NOT_FOUND_COLUMN_IN_BLOCK;
|
|
|
|
extern const int INCORRECT_ELEMENT_OF_SET;
|
|
|
|
extern const int ALIAS_REQUIRED;
|
|
|
|
extern const int EMPTY_NESTED_TABLE;
|
|
|
|
extern const int DUPLICATE_COLUMN;
|
|
|
|
extern const int FUNCTION_CANNOT_HAVE_PARAMETERS;
|
|
|
|
extern const int ILLEGAL_AGGREGATION;
|
|
|
|
extern const int SUPPORT_IS_DISABLED;
|
|
|
|
extern const int TOO_DEEP_AST;
|
2018-03-02 06:33:17 +00:00
|
|
|
extern const int TOO_BIG_AST;
|
2017-12-20 07:36:30 +00:00
|
|
|
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
|
2018-04-24 07:16:39 +00:00
|
|
|
extern const int CONDITIONAL_TREE_PARENT_NOT_FOUND;
|
2018-07-23 15:54:10 +00:00
|
|
|
extern const int INVALID_JOIN_ON_EXPRESSION;
|
2018-08-29 15:15:42 +00:00
|
|
|
extern const int EXPECTED_ALL_OR_ANY;
|
2016-01-11 21:46:36 +00:00
|
|
|
}
|
|
|
|
|
2013-05-24 10:49:19 +00:00
|
|
|
|
2018-11-02 18:53:23 +00:00
|
|
|
//namespace
|
|
|
|
//{
|
|
|
|
//
|
|
|
|
//void removeDuplicateColumns(NamesAndTypesList & columns)
|
|
|
|
//{
|
|
|
|
// std::set<String> names;
|
|
|
|
// for (auto it = columns.begin(); it != columns.end();)
|
|
|
|
// {
|
|
|
|
// if (names.emplace(it->name).second)
|
|
|
|
// ++it;
|
|
|
|
// else
|
|
|
|
// columns.erase(it++);
|
|
|
|
// }
|
|
|
|
//}
|
|
|
|
//
|
|
|
|
//}
|
2015-07-26 07:08:46 +00:00
|
|
|
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2017-01-14 09:00:19 +00:00
|
|
|
ExpressionAnalyzer::ExpressionAnalyzer(
|
2018-08-27 17:58:43 +00:00
|
|
|
const ASTPtr & query_,
|
2017-04-01 07:20:54 +00:00
|
|
|
const Context & context_,
|
2017-09-08 03:47:27 +00:00
|
|
|
const StoragePtr & storage_,
|
2018-02-26 09:05:06 +00:00
|
|
|
const NamesAndTypesList & source_columns_,
|
|
|
|
const Names & required_result_columns_,
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t subquery_depth_,
|
2018-02-08 17:10:35 +00:00
|
|
|
bool do_global_,
|
2018-10-18 15:03:14 +00:00
|
|
|
const SubqueriesForSets & subqueries_for_sets_)
|
|
|
|
: ExpressionAnalyzerData(source_columns_, required_result_columns_, subqueries_for_sets_),
|
|
|
|
query(query_), context(context_), settings(context.getSettings()), storage(storage_),
|
|
|
|
subquery_depth(subquery_depth_), do_global(do_global_)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2018-11-02 18:53:23 +00:00
|
|
|
auto syntax_analyzer_result = SyntaxAnalyzer()
|
|
|
|
.analyze(query, context, storage, source_columns, required_result_columns_, subquery_depth);
|
|
|
|
query = syntax_analyzer_result.query;
|
|
|
|
source_columns = syntax_analyzer_result.source_columns;
|
|
|
|
aliases = syntax_analyzer_result.aliases;
|
|
|
|
array_join_result_to_source = syntax_analyzer_result.array_join_result_to_source;
|
|
|
|
array_join_alias_to_name = syntax_analyzer_result.array_join_alias_to_name;
|
|
|
|
array_join_name_to_alias = syntax_analyzer_result.array_join_name_to_alias;
|
|
|
|
analyzed_join = syntax_analyzer_result.analyzed_join;
|
|
|
|
rewrite_subqueries = syntax_analyzer_result.rewrite_subqueries;
|
|
|
|
storage = syntax_analyzer_result.storage;
|
2017-06-19 02:14:18 +00:00
|
|
|
|
2018-11-02 18:53:23 +00:00
|
|
|
select_query = typeid_cast<ASTSelectQuery *>(query.get());
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-11-02 18:53:23 +00:00
|
|
|
// removeDuplicateColumns(source_columns);
|
2018-03-04 16:15:31 +00:00
|
|
|
|
2018-02-26 21:00:42 +00:00
|
|
|
/// Delete the unnecessary from `source_columns` list. Create `unknown_required_source_columns`. Form `columns_added_by_join`.
|
2017-04-01 07:20:54 +00:00
|
|
|
collectUsedColumns();
|
2014-07-04 19:13:38 +00:00
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// 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.
|
2017-04-01 07:20:54 +00:00
|
|
|
initGlobalSubqueriesAndExternalTables();
|
2015-10-15 11:07:39 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/// has_aggregation, aggregation_keys, aggregate_descriptions, aggregated_columns.
|
2017-04-02 17:37:49 +00:00
|
|
|
/// This analysis should be performed after processing global subqueries, because otherwise,
|
|
|
|
/// if the aggregate function contains a global subquery, then `analyzeAggregation` method will save
|
|
|
|
/// in `aggregate_descriptions` the information about the parameters of this aggregate function, among which
|
|
|
|
/// global subquery. Then, when you call `initGlobalSubqueriesAndExternalTables` method, this
|
|
|
|
/// the global subquery will be replaced with a temporary table, resulting in aggregate_descriptions
|
|
|
|
/// will contain out-of-date information, which will lead to an error when the query is executed.
|
2017-04-01 07:20:54 +00:00
|
|
|
analyzeAggregation();
|
2014-07-04 19:13:38 +00:00
|
|
|
}
|
|
|
|
|
2018-10-17 11:06:46 +00:00
|
|
|
bool ExpressionAnalyzer::isRemoteStorage() const
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2018-10-17 11:06:46 +00:00
|
|
|
return storage && storage->isRemote();
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-07-04 19:13:38 +00:00
|
|
|
void ExpressionAnalyzer::analyzeAggregation()
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/** Find aggregation keys (aggregation_keys), information about aggregate functions (aggregate_descriptions),
|
|
|
|
* as well as a set of columns obtained after the aggregation, if any,
|
|
|
|
* or after all the actions that are usually performed before aggregation (aggregated_columns).
|
2017-04-01 07:20:54 +00:00
|
|
|
*
|
2017-04-02 17:37:49 +00:00
|
|
|
* Everything below (compiling temporary ExpressionActions) - only for the purpose of query analysis (type output).
|
2017-04-01 07:20:54 +00:00
|
|
|
*/
|
|
|
|
|
|
|
|
if (select_query && (select_query->group_expression_list || select_query->having_expression))
|
|
|
|
has_aggregation = true;
|
|
|
|
|
2018-08-30 16:31:20 +00:00
|
|
|
ExpressionActionsPtr temp_actions = std::make_shared<ExpressionActions>(source_columns, context);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
if (select_query && select_query->array_join_expression_list())
|
|
|
|
{
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->array_join_expression_list(), true, temp_actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
addMultipleArrayJoinAction(temp_actions);
|
2018-04-04 18:56:30 +00:00
|
|
|
array_join_columns = temp_actions->getSampleBlock().getNamesAndTypesList();
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if (select_query)
|
|
|
|
{
|
|
|
|
const ASTTablesInSelectQueryElement * join = select_query->join();
|
|
|
|
if (join)
|
|
|
|
{
|
2018-07-30 13:57:50 +00:00
|
|
|
const auto table_join = static_cast<const ASTTableJoin &>(*join->table_join);
|
|
|
|
if (table_join.using_expression_list)
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(table_join.using_expression_list, true, temp_actions);
|
2018-07-30 13:57:50 +00:00
|
|
|
if (table_join.on_expression)
|
|
|
|
for (const auto & key_ast : analyzed_join.key_asts_left)
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(key_ast, true, temp_actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
addJoinAction(temp_actions, true);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-08-27 17:58:43 +00:00
|
|
|
getAggregates(query, temp_actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
if (has_aggregation)
|
|
|
|
{
|
|
|
|
assertSelect();
|
|
|
|
|
|
|
|
/// Find out aggregation keys.
|
|
|
|
if (select_query->group_expression_list)
|
|
|
|
{
|
|
|
|
NameSet unique_keys;
|
|
|
|
ASTs & group_asts = select_query->group_expression_list->children;
|
|
|
|
for (ssize_t i = 0; i < ssize_t(group_asts.size()); ++i)
|
|
|
|
{
|
|
|
|
ssize_t size = group_asts.size();
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(group_asts[i], true, temp_actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
const auto & column_name = group_asts[i]->getColumnName();
|
|
|
|
const auto & block = temp_actions->getSampleBlock();
|
|
|
|
|
|
|
|
if (!block.has(column_name))
|
|
|
|
throw Exception("Unknown identifier (in GROUP BY): " + column_name, ErrorCodes::UNKNOWN_IDENTIFIER);
|
|
|
|
|
|
|
|
const auto & col = block.getByName(column_name);
|
|
|
|
|
|
|
|
/// Constant expressions have non-null column pointer at this stage.
|
2018-07-27 18:14:58 +00:00
|
|
|
if (col.column && col.column->isColumnConst())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
/// But don't remove last key column if no aggregate functions, otherwise aggregation will not work.
|
|
|
|
if (!aggregate_descriptions.empty() || size > 1)
|
|
|
|
{
|
|
|
|
if (i + 1 < static_cast<ssize_t>(size))
|
|
|
|
group_asts[i] = std::move(group_asts.back());
|
|
|
|
|
|
|
|
group_asts.pop_back();
|
|
|
|
|
|
|
|
--i;
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2017-12-25 21:57:29 +00:00
|
|
|
NameAndTypePair key{column_name, col.type};
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Aggregation keys are uniqued.
|
|
|
|
if (!unique_keys.count(key.name))
|
|
|
|
{
|
|
|
|
unique_keys.insert(key.name);
|
2018-04-04 18:56:30 +00:00
|
|
|
aggregation_keys.push_back(key);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Key is no longer needed, therefore we can save a little by moving it.
|
2018-04-04 18:56:30 +00:00
|
|
|
aggregated_columns.push_back(std::move(key));
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (group_asts.empty())
|
|
|
|
{
|
|
|
|
select_query->group_expression_list = nullptr;
|
2018-04-04 18:56:30 +00:00
|
|
|
has_aggregation = select_query->having_expression || aggregate_descriptions.size();
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
for (size_t i = 0; i < aggregate_descriptions.size(); ++i)
|
|
|
|
{
|
|
|
|
AggregateDescription & desc = aggregate_descriptions[i];
|
2018-04-04 18:56:30 +00:00
|
|
|
aggregated_columns.emplace_back(desc.column_name, desc.function->getReturnType());
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
2018-04-04 18:56:30 +00:00
|
|
|
else
|
|
|
|
{
|
|
|
|
aggregated_columns = temp_actions->getSampleBlock().getNamesAndTypesList();
|
|
|
|
}
|
2013-05-24 10:49:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-07-04 20:30:06 +00:00
|
|
|
void ExpressionAnalyzer::initGlobalSubqueriesAndExternalTables()
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Adds existing external tables (not subqueries) to the external_tables dictionary.
|
2018-10-16 19:00:05 +00:00
|
|
|
ExternalTablesVisitor tables_visitor(context, external_tables);
|
|
|
|
tables_visitor.visit(query);
|
2014-07-04 20:30:06 +00:00
|
|
|
|
2018-10-17 10:59:05 +00:00
|
|
|
if (do_global)
|
|
|
|
{
|
2018-10-17 11:06:46 +00:00
|
|
|
GlobalSubqueriesVisitor subqueries_visitor(context, subquery_depth, isRemoteStorage(),
|
2018-10-17 10:59:05 +00:00
|
|
|
external_tables, subqueries_for_sets, has_global_subqueries);
|
|
|
|
subqueries_visitor.visit(query);
|
|
|
|
}
|
2015-11-04 22:02:52 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-04-10 08:57:01 +00:00
|
|
|
void ExpressionAnalyzer::makeSetsForIndex()
|
2014-03-31 14:49:43 +00:00
|
|
|
{
|
2018-03-16 06:33:23 +00:00
|
|
|
if (storage && select_query && storage->supportsIndexForIn())
|
|
|
|
{
|
|
|
|
if (select_query->where_expression)
|
|
|
|
makeSetsForIndexImpl(select_query->where_expression, storage->getSampleBlock());
|
|
|
|
if (select_query->prewhere_expression)
|
|
|
|
makeSetsForIndexImpl(select_query->prewhere_expression, storage->getSampleBlock());
|
|
|
|
}
|
2014-03-31 14:49:43 +00:00
|
|
|
}
|
|
|
|
|
2018-01-21 07:30:07 +00:00
|
|
|
|
2018-07-02 19:37:04 +00:00
|
|
|
void ExpressionAnalyzer::tryMakeSetForIndexFromSubquery(const ASTPtr & subquery_or_table_name)
|
2018-01-21 07:30:07 +00:00
|
|
|
{
|
|
|
|
BlockIO res = interpretSubquery(subquery_or_table_name, context, subquery_depth + 1, {})->execute();
|
|
|
|
|
2018-10-18 15:03:14 +00:00
|
|
|
SetPtr set = std::make_shared<Set>(settings.size_limits_for_set, true);
|
2018-01-21 07:30:07 +00:00
|
|
|
|
2018-04-19 21:34:04 +00:00
|
|
|
set->setHeader(res.in->getHeader());
|
2018-01-21 07:30:07 +00:00
|
|
|
while (Block block = res.in->read())
|
|
|
|
{
|
|
|
|
/// If the limits have been exceeded, give up and let the default subquery processing actions take place.
|
2018-07-02 18:57:14 +00:00
|
|
|
if (!set->insertFromBlock(block))
|
2018-01-21 07:30:07 +00:00
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
2018-05-16 10:25:10 +00:00
|
|
|
prepared_sets[subquery_or_table_name->range] = std::move(set);
|
2018-01-21 07:30:07 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-07-15 03:48:36 +00:00
|
|
|
void ExpressionAnalyzer::makeSetsForIndexImpl(const ASTPtr & node, const Block & sample_block)
|
2014-03-31 14:49:43 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
for (auto & child : node->children)
|
2018-02-14 17:39:16 +00:00
|
|
|
{
|
2018-03-16 14:01:00 +00:00
|
|
|
/// Don't descent into subqueries.
|
2018-03-16 06:33:23 +00:00
|
|
|
if (typeid_cast<ASTSubquery *>(child.get()))
|
|
|
|
continue;
|
|
|
|
|
|
|
|
/// Don't dive into lambda functions
|
|
|
|
const ASTFunction * func = typeid_cast<const ASTFunction *>(child.get());
|
|
|
|
if (func && func->name == "lambda")
|
|
|
|
continue;
|
|
|
|
|
|
|
|
makeSetsForIndexImpl(child, sample_block);
|
2018-02-14 17:39:16 +00:00
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-07-14 00:33:37 +00:00
|
|
|
const ASTFunction * func = typeid_cast<const ASTFunction *>(node.get());
|
2018-02-24 01:31:42 +00:00
|
|
|
if (func && functionIsInOperator(func->name))
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2017-07-14 00:33:37 +00:00
|
|
|
const IAST & args = *func->arguments;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-03-16 06:39:32 +00:00
|
|
|
if (storage && storage->mayBenefitFromIndexForIn(args.children.at(0)))
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2018-03-16 06:39:32 +00:00
|
|
|
const ASTPtr & arg = args.children.at(1);
|
|
|
|
|
2018-05-16 10:25:10 +00:00
|
|
|
if (!prepared_sets.count(arg->range)) /// Not already prepared.
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2018-03-16 06:39:32 +00:00
|
|
|
if (typeid_cast<ASTSubquery *>(arg.get()) || typeid_cast<ASTIdentifier *>(arg.get()))
|
|
|
|
{
|
|
|
|
if (settings.use_index_for_in_with_subqueries)
|
2018-07-02 19:37:04 +00:00
|
|
|
tryMakeSetForIndexFromSubquery(arg);
|
2018-03-16 06:39:32 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2018-04-03 19:16:24 +00:00
|
|
|
NamesAndTypesList temp_columns = source_columns;
|
2018-04-04 18:56:30 +00:00
|
|
|
temp_columns.insert(temp_columns.end(), array_join_columns.begin(), array_join_columns.end());
|
2018-07-31 11:31:18 +00:00
|
|
|
for (const auto & joined_column : analyzed_join.columns_added_by_join)
|
|
|
|
temp_columns.push_back(joined_column.name_and_type);
|
2018-08-30 16:31:20 +00:00
|
|
|
ExpressionActionsPtr temp_actions = std::make_shared<ExpressionActions>(temp_columns, context);
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(func->arguments->children.at(0), true, temp_actions);
|
2018-03-16 04:37:21 +00:00
|
|
|
|
2018-03-16 06:39:32 +00:00
|
|
|
Block sample_block_with_calculated_columns = temp_actions->getSampleBlock();
|
|
|
|
if (sample_block_with_calculated_columns.has(args.children.at(0)->getColumnName()))
|
2018-10-18 15:03:14 +00:00
|
|
|
makeExplicitSet(func, sample_block_with_calculated_columns, true, context,
|
|
|
|
settings.size_limits_for_set, prepared_sets);
|
2018-03-16 06:39:32 +00:00
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
2014-03-31 14:49:43 +00:00
|
|
|
}
|
2013-05-24 10:49:19 +00:00
|
|
|
|
2018-05-01 13:31:38 +00:00
|
|
|
bool ExpressionAnalyzer::isThereArrayJoin(const ASTPtr & ast)
|
|
|
|
{
|
2018-04-24 07:16:39 +00:00
|
|
|
if (typeid_cast<ASTIdentifier *>(ast.get()))
|
|
|
|
{
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
else if (ASTFunction * node = typeid_cast<ASTFunction *>(ast.get()))
|
|
|
|
{
|
|
|
|
if (node->name == "arrayJoin")
|
|
|
|
{
|
|
|
|
return true;
|
|
|
|
}
|
|
|
|
if (functionIsInOrGlobalInOperator(node->name))
|
|
|
|
{
|
|
|
|
return isThereArrayJoin(node->arguments->children.at(0));
|
|
|
|
}
|
|
|
|
if (node->name == "indexHint")
|
|
|
|
{
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
if (AggregateFunctionFactory::instance().isAggregateFunctionName(node->name))
|
|
|
|
{
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
for (auto & child : node->arguments->children)
|
|
|
|
{
|
2018-05-01 13:31:38 +00:00
|
|
|
if (isThereArrayJoin(child))
|
|
|
|
{
|
2018-04-24 07:16:39 +00:00
|
|
|
return true;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
else if (typeid_cast<ASTLiteral *>(ast.get()))
|
|
|
|
{
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
for (auto & child : ast->children)
|
|
|
|
{
|
2018-05-01 13:31:38 +00:00
|
|
|
if (isThereArrayJoin(child))
|
|
|
|
{
|
2018-04-24 07:16:39 +00:00
|
|
|
return true;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
}
|
2013-07-26 16:11:31 +00:00
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
|
|
|
|
void ExpressionAnalyzer::getRootActions(const ASTPtr & ast, bool no_subqueries, ExpressionActionsPtr & actions, bool only_consts)
|
2018-07-24 12:41:35 +00:00
|
|
|
{
|
2018-10-16 12:34:20 +00:00
|
|
|
bool is_conditional_tree = !isThereArrayJoin(ast) && settings.enable_conditional_computation && !only_consts;
|
2018-07-24 12:41:35 +00:00
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
LogAST log;
|
2018-10-18 15:03:14 +00:00
|
|
|
ActionsVisitor actions_visitor(context, settings.size_limits_for_set, is_conditional_tree, subquery_depth,
|
2018-10-16 12:34:20 +00:00
|
|
|
source_columns, actions, prepared_sets, subqueries_for_sets,
|
2018-10-17 11:06:46 +00:00
|
|
|
no_subqueries, only_consts, !isRemoteStorage(), log.stream());
|
2018-10-16 12:34:20 +00:00
|
|
|
actions_visitor.visit(ast);
|
|
|
|
actions = actions_visitor.popActionsLevel();
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void ExpressionAnalyzer::getActionsFromJoinKeys(const ASTTableJoin & table_join, bool no_subqueries, ExpressionActionsPtr & actions)
|
|
|
|
{
|
|
|
|
bool only_consts = false;
|
|
|
|
bool is_conditional_tree = !isThereArrayJoin(query) && settings.enable_conditional_computation && !only_consts;
|
2018-07-24 12:41:35 +00:00
|
|
|
|
2018-10-12 15:41:28 +00:00
|
|
|
LogAST log;
|
2018-10-18 15:03:14 +00:00
|
|
|
ActionsVisitor actions_visitor(context, settings.size_limits_for_set, is_conditional_tree, subquery_depth,
|
2018-10-16 12:34:20 +00:00
|
|
|
source_columns, actions, prepared_sets, subqueries_for_sets,
|
2018-10-17 11:06:46 +00:00
|
|
|
no_subqueries, only_consts, !isRemoteStorage(), log.stream());
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2018-07-24 12:41:35 +00:00
|
|
|
if (table_join.using_expression_list)
|
2018-10-16 12:34:20 +00:00
|
|
|
actions_visitor.visit(table_join.using_expression_list);
|
2018-07-24 12:41:35 +00:00
|
|
|
else if (table_join.on_expression)
|
|
|
|
{
|
2018-07-30 13:57:50 +00:00
|
|
|
for (const auto & ast : analyzed_join.key_asts_left)
|
2018-10-16 12:34:20 +00:00
|
|
|
actions_visitor.visit(ast);
|
2018-07-24 12:41:35 +00:00
|
|
|
}
|
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
actions = actions_visitor.popActionsLevel();
|
2018-07-24 12:41:35 +00:00
|
|
|
}
|
|
|
|
|
2013-05-24 10:49:19 +00:00
|
|
|
|
2016-03-05 02:30:20 +00:00
|
|
|
void ExpressionAnalyzer::getAggregates(const ASTPtr & ast, ExpressionActionsPtr & actions)
|
2013-05-24 10:49:19 +00:00
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// There can not be aggregate functions inside the WHERE and PREWHERE.
|
2017-04-01 07:20:54 +00:00
|
|
|
if (select_query && (ast.get() == select_query->where_expression.get() || ast.get() == select_query->prewhere_expression.get()))
|
|
|
|
{
|
|
|
|
assertNoAggregates(ast, "in WHERE or PREWHERE");
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// If we are not analyzing a SELECT query, but a separate expression, then there can not be aggregate functions in it.
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!select_query)
|
|
|
|
{
|
|
|
|
assertNoAggregates(ast, "in wrong place");
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
|
|
|
const ASTFunction * node = typeid_cast<const ASTFunction *>(ast.get());
|
2018-02-24 01:31:42 +00:00
|
|
|
if (node && AggregateFunctionFactory::instance().isAggregateFunctionName(node->name))
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
has_aggregation = true;
|
|
|
|
AggregateDescription aggregate;
|
|
|
|
aggregate.column_name = node->getColumnName();
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Make unique aggregate functions.
|
2017-04-01 07:20:54 +00:00
|
|
|
for (size_t i = 0; i < aggregate_descriptions.size(); ++i)
|
|
|
|
if (aggregate_descriptions[i].column_name == aggregate.column_name)
|
|
|
|
return;
|
|
|
|
|
|
|
|
const ASTs & arguments = node->arguments->children;
|
|
|
|
aggregate.argument_names.resize(arguments.size());
|
|
|
|
DataTypes types(arguments.size());
|
|
|
|
|
|
|
|
for (size_t i = 0; i < arguments.size(); ++i)
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// There can not be other aggregate functions within the aggregate functions.
|
2017-04-01 07:20:54 +00:00
|
|
|
assertNoAggregates(arguments[i], "inside another aggregate function");
|
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(arguments[i], true, actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
const std::string & name = arguments[i]->getColumnName();
|
|
|
|
types[i] = actions->getSampleBlock().getByName(name).type;
|
|
|
|
aggregate.argument_names[i] = name;
|
|
|
|
}
|
|
|
|
|
2017-07-10 23:30:17 +00:00
|
|
|
aggregate.parameters = (node->parameters) ? getAggregateFunctionParametersArray(node->parameters) : Array();
|
|
|
|
aggregate.function = AggregateFunctionFactory::instance().get(node->name, types, aggregate.parameters);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
aggregate_descriptions.push_back(aggregate);
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
for (const auto & child : ast->children)
|
|
|
|
if (!typeid_cast<const ASTSubquery *>(child.get())
|
|
|
|
&& !typeid_cast<const ASTSelectQuery *>(child.get()))
|
|
|
|
getAggregates(child, actions);
|
|
|
|
}
|
2013-05-24 10:49:19 +00:00
|
|
|
}
|
|
|
|
|
2016-03-05 02:30:20 +00:00
|
|
|
|
|
|
|
void ExpressionAnalyzer::assertNoAggregates(const ASTPtr & ast, const char * description)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
const ASTFunction * node = typeid_cast<const ASTFunction *>(ast.get());
|
2016-03-05 02:30:20 +00:00
|
|
|
|
2018-02-24 01:31:42 +00:00
|
|
|
if (node && AggregateFunctionFactory::instance().isAggregateFunctionName(node->name))
|
2017-04-01 07:20:54 +00:00
|
|
|
throw Exception("Aggregate function " + node->getColumnName()
|
|
|
|
+ " is found " + String(description) + " in query", ErrorCodes::ILLEGAL_AGGREGATION);
|
2016-03-05 02:30:20 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
for (const auto & child : ast->children)
|
|
|
|
if (!typeid_cast<const ASTSubquery *>(child.get())
|
|
|
|
&& !typeid_cast<const ASTSelectQuery *>(child.get()))
|
|
|
|
assertNoAggregates(child, description);
|
2016-03-05 02:30:20 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2015-04-18 22:30:43 +00:00
|
|
|
void ExpressionAnalyzer::assertSelect() const
|
2013-05-28 11:54:37 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!select_query)
|
|
|
|
throw Exception("Not a select query", ErrorCodes::LOGICAL_ERROR);
|
2013-05-28 11:54:37 +00:00
|
|
|
}
|
2013-06-14 16:38:54 +00:00
|
|
|
|
2015-04-18 22:30:43 +00:00
|
|
|
void ExpressionAnalyzer::assertAggregation() const
|
2013-05-24 10:49:19 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!has_aggregation)
|
|
|
|
throw Exception("No aggregation", ErrorCodes::LOGICAL_ERROR);
|
2013-05-28 11:54:37 +00:00
|
|
|
}
|
2013-06-14 16:38:54 +00:00
|
|
|
|
2017-12-25 21:57:29 +00:00
|
|
|
void ExpressionAnalyzer::initChain(ExpressionActionsChain & chain, const NamesAndTypesList & columns) const
|
2013-05-28 14:24:20 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (chain.steps.empty())
|
|
|
|
{
|
2018-08-30 16:31:20 +00:00
|
|
|
chain.steps.emplace_back(std::make_shared<ExpressionActions>(columns, context));
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
2013-05-28 14:24:20 +00:00
|
|
|
}
|
2013-05-28 11:54:37 +00:00
|
|
|
|
2016-07-23 02:25:09 +00:00
|
|
|
/// "Big" ARRAY JOIN.
|
2015-04-18 22:30:43 +00:00
|
|
|
void ExpressionAnalyzer::addMultipleArrayJoinAction(ExpressionActionsPtr & actions) const
|
2013-10-17 13:32:32 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
NameSet result_columns;
|
|
|
|
for (const auto & result_source : array_join_result_to_source)
|
|
|
|
{
|
|
|
|
/// Assign new names to columns, if needed.
|
|
|
|
if (result_source.first != result_source.second)
|
|
|
|
actions->add(ExpressionAction::copyColumn(result_source.second, result_source.first));
|
2015-10-22 20:56:52 +00:00
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Make ARRAY JOIN (replace arrays with their insides) for the columns in these new names.
|
2017-04-01 07:20:54 +00:00
|
|
|
result_columns.insert(result_source.first);
|
|
|
|
}
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
actions->add(ExpressionAction::arrayJoin(result_columns, select_query->array_join_is_left(), context));
|
2013-10-17 13:32:32 +00:00
|
|
|
}
|
|
|
|
|
2014-03-28 12:13:58 +00:00
|
|
|
bool ExpressionAnalyzer::appendArrayJoin(ExpressionActionsChain & chain, bool only_types)
|
2013-07-26 16:33:05 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertSelect();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!select_query->array_join_expression_list())
|
|
|
|
return false;
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
initChain(chain, source_columns);
|
2017-04-01 07:20:54 +00:00
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->array_join_expression_list(), only_types, step.actions);
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
addMultipleArrayJoinAction(step.actions);
|
2013-10-21 11:33:25 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return true;
|
2013-07-26 16:33:05 +00:00
|
|
|
}
|
|
|
|
|
2015-04-18 22:30:43 +00:00
|
|
|
void ExpressionAnalyzer::addJoinAction(ExpressionActionsPtr & actions, bool only_types) const
|
2014-06-13 02:05:05 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (only_types)
|
2018-07-30 13:57:50 +00:00
|
|
|
actions->add(ExpressionAction::ordinaryJoin(nullptr, analyzed_join.key_names_left,
|
2018-07-31 11:31:18 +00:00
|
|
|
analyzed_join.getColumnsAddedByJoin()));
|
2017-04-01 07:20:54 +00:00
|
|
|
else
|
|
|
|
for (auto & subquery_for_set : subqueries_for_sets)
|
|
|
|
if (subquery_for_set.second.join)
|
2018-07-30 13:57:50 +00:00
|
|
|
actions->add(ExpressionAction::ordinaryJoin(subquery_for_set.second.join, analyzed_join.key_names_left,
|
2018-07-31 11:31:18 +00:00
|
|
|
analyzed_join.getColumnsAddedByJoin()));
|
2014-06-13 02:05:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
bool ExpressionAnalyzer::appendJoin(ExpressionActionsChain & chain, bool only_types)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertSelect();
|
|
|
|
|
|
|
|
if (!select_query->join())
|
|
|
|
return false;
|
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
initChain(chain, source_columns);
|
2017-04-01 07:20:54 +00:00
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
|
|
|
|
2018-07-26 15:11:53 +00:00
|
|
|
const auto & join_element = static_cast<const ASTTablesInSelectQueryElement &>(*select_query->join());
|
2018-08-28 13:57:31 +00:00
|
|
|
auto & join_params = static_cast<ASTTableJoin &>(*join_element.table_join);
|
|
|
|
|
2018-08-29 13:51:07 +00:00
|
|
|
if (join_params.strictness == ASTTableJoin::Strictness::Unspecified && join_params.kind != ASTTableJoin::Kind::Cross)
|
2018-08-28 13:57:31 +00:00
|
|
|
{
|
2018-10-18 15:03:14 +00:00
|
|
|
if (settings.join_default_strictness == "ANY")
|
2018-08-28 13:57:31 +00:00
|
|
|
join_params.strictness = ASTTableJoin::Strictness::Any;
|
2018-10-18 15:03:14 +00:00
|
|
|
else if (settings.join_default_strictness == "ALL")
|
2018-08-28 13:57:31 +00:00
|
|
|
join_params.strictness = ASTTableJoin::Strictness::All;
|
|
|
|
else
|
2018-08-29 15:15:42 +00:00
|
|
|
throw Exception("Expected ANY or ALL in JOIN section, because setting (join_default_strictness) is empty", DB::ErrorCodes::EXPECTED_ALL_OR_ANY);
|
2018-08-28 13:57:31 +00:00
|
|
|
}
|
|
|
|
|
2018-07-26 15:11:53 +00:00
|
|
|
const auto & table_to_join = static_cast<const ASTTableExpression &>(*join_element.table_expression);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
getActionsFromJoinKeys(join_params, only_types, step.actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Two JOINs are not supported with the same subquery, but different USINGs.
|
2018-02-23 08:05:21 +00:00
|
|
|
auto join_hash = join_element.getTreeHash();
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-02-23 08:05:21 +00:00
|
|
|
SubqueryForSet & subquery_for_set = subqueries_for_sets[toString(join_hash.first) + "_" + toString(join_hash.second)];
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Special case - if table name is specified on the right of JOIN, then the table has the type Join (the previously prepared mapping).
|
|
|
|
/// TODO This syntax does not support specifying a database name.
|
2017-04-01 07:20:54 +00:00
|
|
|
if (table_to_join.database_and_table_name)
|
|
|
|
{
|
2018-07-27 20:01:54 +00:00
|
|
|
const auto & identifier = static_cast<const ASTIdentifier &>(*table_to_join.database_and_table_name);
|
2018-10-30 16:31:21 +00:00
|
|
|
DatabaseAndTableWithAlias database_table(identifier);
|
|
|
|
StoragePtr table = context.tryGetTable(database_table.database, database_table.table);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
if (table)
|
|
|
|
{
|
2017-11-04 16:46:14 +00:00
|
|
|
StorageJoin * storage_join = dynamic_cast<StorageJoin *>(table.get());
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
if (storage_join)
|
|
|
|
{
|
|
|
|
storage_join->assertCompatible(join_params.kind, join_params.strictness);
|
2017-04-02 17:37:49 +00:00
|
|
|
/// TODO Check the set of keys.
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
JoinPtr & join = storage_join->getJoin();
|
|
|
|
subquery_for_set.join = join;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (!subquery_for_set.join)
|
|
|
|
{
|
|
|
|
JoinPtr join = std::make_shared<Join>(
|
2018-07-30 13:57:50 +00:00
|
|
|
analyzed_join.key_names_left, analyzed_join.key_names_right, analyzed_join.columns_added_by_join_from_right_keys,
|
2018-10-18 15:03:14 +00:00
|
|
|
settings.join_use_nulls, settings.size_limits_for_join,
|
2017-04-01 07:20:54 +00:00
|
|
|
join_params.kind, join_params.strictness);
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/** For GLOBAL JOINs (in the case, for example, of the push method for executing GLOBAL subqueries), the following occurs
|
|
|
|
* - in the addExternalStorage function, the JOIN (SELECT ...) subquery is replaced with JOIN _data1,
|
|
|
|
* in the subquery_for_set object this subquery is exposed as source and the temporary table _data1 as the `table`.
|
|
|
|
* - this function shows the expression JOIN _data1.
|
2017-04-01 07:20:54 +00:00
|
|
|
*/
|
|
|
|
if (!subquery_for_set.source)
|
|
|
|
{
|
|
|
|
ASTPtr table;
|
2018-07-30 13:57:50 +00:00
|
|
|
|
2018-08-14 09:44:56 +00:00
|
|
|
if (table_to_join.subquery)
|
2017-04-01 07:20:54 +00:00
|
|
|
table = table_to_join.subquery;
|
2018-08-14 09:44:56 +00:00
|
|
|
else if (table_to_join.table_function)
|
|
|
|
table = table_to_join.table_function;
|
|
|
|
else if (table_to_join.database_and_table_name)
|
|
|
|
table = table_to_join.database_and_table_name;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-10-01 14:18:47 +00:00
|
|
|
Names original_columns;
|
2018-11-02 18:53:23 +00:00
|
|
|
for (const auto & column : analyzed_join.columns_from_joined_table)
|
2018-10-01 14:18:47 +00:00
|
|
|
if (analyzed_join.required_columns_from_joined_table.count(column.name_and_type.name))
|
|
|
|
original_columns.emplace_back(column.original_name);
|
|
|
|
|
|
|
|
auto interpreter = interpretSubquery(table, context, subquery_depth, original_columns);
|
2018-02-15 19:48:41 +00:00
|
|
|
subquery_for_set.source = std::make_shared<LazyBlockInputStream>(
|
|
|
|
interpreter->getSampleBlock(),
|
|
|
|
[interpreter]() mutable { return interpreter->execute().in; });
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
2018-10-01 14:18:47 +00:00
|
|
|
/// Alias duplicating columns as qualified.
|
2018-11-02 18:53:23 +00:00
|
|
|
for (const auto & column : analyzed_join.columns_from_joined_table)
|
2018-10-01 14:18:47 +00:00
|
|
|
if (analyzed_join.required_columns_from_joined_table.count(column.name_and_type.name))
|
|
|
|
subquery_for_set.joined_block_aliases.emplace_back(column.original_name, column.name_and_type.name);
|
2018-07-31 11:31:18 +00:00
|
|
|
|
2018-07-30 13:57:50 +00:00
|
|
|
auto sample_block = subquery_for_set.source->getHeader();
|
2018-07-31 11:31:18 +00:00
|
|
|
for (const auto & name_with_alias : subquery_for_set.joined_block_aliases)
|
|
|
|
{
|
|
|
|
if (sample_block.has(name_with_alias.first))
|
|
|
|
{
|
|
|
|
auto pos = sample_block.getPositionByName(name_with_alias.first);
|
|
|
|
auto column = sample_block.getByPosition(pos);
|
|
|
|
sample_block.erase(pos);
|
|
|
|
column.name = name_with_alias.second;
|
|
|
|
sample_block.insert(std::move(column));
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-10-02 18:14:37 +00:00
|
|
|
analyzed_join.joined_block_actions->execute(sample_block);
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// TODO You do not need to set this up when JOIN is only needed on remote servers.
|
2017-04-01 07:20:54 +00:00
|
|
|
subquery_for_set.join = join;
|
2018-07-30 13:57:50 +00:00
|
|
|
subquery_for_set.join->setSampleBlock(sample_block);
|
|
|
|
subquery_for_set.joined_block_actions = analyzed_join.joined_block_actions;
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
addJoinAction(step.actions, false);
|
|
|
|
|
|
|
|
return true;
|
2014-06-13 02:05:05 +00:00
|
|
|
}
|
|
|
|
|
2018-10-04 08:58:19 +00:00
|
|
|
bool ExpressionAnalyzer::appendPrewhere(ExpressionActionsChain & chain, bool only_types,
|
|
|
|
const ASTPtr & sampling_expression, const ASTPtr & primary_expression)
|
2018-04-06 13:58:06 +00:00
|
|
|
{
|
|
|
|
assertSelect();
|
|
|
|
|
|
|
|
if (!select_query->prewhere_expression)
|
|
|
|
return false;
|
|
|
|
|
2018-10-04 08:58:19 +00:00
|
|
|
Names additional_required_mergetree_columns;
|
2018-09-07 15:13:08 +00:00
|
|
|
if (sampling_expression)
|
2018-10-04 08:58:19 +00:00
|
|
|
additional_required_mergetree_columns = ExpressionAnalyzer(sampling_expression, context, storage).getRequiredSourceColumns();
|
|
|
|
if (primary_expression)
|
|
|
|
{
|
|
|
|
auto required_primary_columns = ExpressionAnalyzer(primary_expression, context, storage).getRequiredSourceColumns();
|
|
|
|
additional_required_mergetree_columns.insert(additional_required_mergetree_columns.end(),
|
|
|
|
required_primary_columns.begin(), required_primary_columns.end());
|
|
|
|
}
|
2018-09-07 15:13:08 +00:00
|
|
|
|
2018-04-06 13:58:06 +00:00
|
|
|
initChain(chain, source_columns);
|
2018-04-12 09:45:24 +00:00
|
|
|
auto & step = chain.getLastStep();
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->prewhere_expression, only_types, step.actions);
|
2018-04-12 09:45:24 +00:00
|
|
|
String prewhere_column_name = select_query->prewhere_expression->getColumnName();
|
|
|
|
step.required_output.push_back(prewhere_column_name);
|
2018-06-29 11:42:44 +00:00
|
|
|
step.can_remove_required_output.push_back(true);
|
2018-04-12 09:45:24 +00:00
|
|
|
|
|
|
|
{
|
|
|
|
/// Remove unused source_columns from prewhere actions.
|
2018-08-30 16:31:20 +00:00
|
|
|
auto tmp_actions = std::make_shared<ExpressionActions>(source_columns, context);
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->prewhere_expression, only_types, tmp_actions);
|
2018-04-12 09:45:24 +00:00
|
|
|
tmp_actions->finalize({prewhere_column_name});
|
|
|
|
auto required_columns = tmp_actions->getRequiredColumns();
|
|
|
|
NameSet required_source_columns(required_columns.begin(), required_columns.end());
|
|
|
|
|
2018-10-04 08:58:19 +00:00
|
|
|
/// Add required columns to required output in order not to remove them after prewhere execution.
|
|
|
|
/// TODO: add sampling and final execution to common chain.
|
|
|
|
for (const auto & column : additional_required_mergetree_columns)
|
2018-09-10 10:06:12 +00:00
|
|
|
{
|
|
|
|
if (required_source_columns.count(column))
|
|
|
|
{
|
|
|
|
step.required_output.push_back(column);
|
|
|
|
step.can_remove_required_output.push_back(true);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-04-12 09:45:24 +00:00
|
|
|
auto names = step.actions->getSampleBlock().getNames();
|
|
|
|
NameSet name_set(names.begin(), names.end());
|
|
|
|
|
|
|
|
for (const auto & column : source_columns)
|
|
|
|
if (required_source_columns.count(column.name) == 0)
|
|
|
|
name_set.erase(column.name);
|
|
|
|
|
|
|
|
Names required_output(name_set.begin(), name_set.end());
|
|
|
|
step.actions->finalize(required_output);
|
|
|
|
}
|
|
|
|
|
|
|
|
{
|
|
|
|
/// Add empty action with input = {prewhere actions output} + {unused source columns}
|
|
|
|
/// Reasons:
|
|
|
|
/// 1. Remove remove source columns which are used only in prewhere actions during prewhere actions execution.
|
|
|
|
/// Example: select A prewhere B > 0. B can be removed at prewhere step.
|
|
|
|
/// 2. Store side columns which were calculated during prewhere actions execution if they are used.
|
|
|
|
/// Example: select F(A) prewhere F(A) > 0. F(A) can be saved from prewhere step.
|
2018-06-25 13:08:35 +00:00
|
|
|
/// 3. Check if we can remove filter column at prewhere step. If we can, action will store single REMOVE_COLUMN.
|
2018-04-12 09:45:24 +00:00
|
|
|
ColumnsWithTypeAndName columns = step.actions->getSampleBlock().getColumnsWithTypeAndName();
|
|
|
|
auto required_columns = step.actions->getRequiredColumns();
|
|
|
|
NameSet prewhere_input_names(required_columns.begin(), required_columns.end());
|
|
|
|
NameSet unused_source_columns;
|
|
|
|
|
|
|
|
for (const auto & column : source_columns)
|
|
|
|
{
|
|
|
|
if (prewhere_input_names.count(column.name) == 0)
|
|
|
|
{
|
|
|
|
columns.emplace_back(column.type, column.name);
|
|
|
|
unused_source_columns.emplace(column.name);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-08-30 16:31:20 +00:00
|
|
|
chain.steps.emplace_back(std::make_shared<ExpressionActions>(std::move(columns), context));
|
2018-04-12 09:45:24 +00:00
|
|
|
chain.steps.back().additional_input = std::move(unused_source_columns);
|
|
|
|
}
|
2018-04-06 13:58:06 +00:00
|
|
|
|
|
|
|
return true;
|
|
|
|
}
|
2016-07-22 20:39:28 +00:00
|
|
|
|
2014-03-28 12:13:58 +00:00
|
|
|
bool ExpressionAnalyzer::appendWhere(ExpressionActionsChain & chain, bool only_types)
|
2013-05-28 11:54:37 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertSelect();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!select_query->where_expression)
|
|
|
|
return false;
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
initChain(chain, source_columns);
|
2017-04-01 07:20:54 +00:00
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
step.required_output.push_back(select_query->where_expression->getColumnName());
|
2018-06-29 11:42:44 +00:00
|
|
|
step.can_remove_required_output = {true};
|
2018-04-23 19:05:46 +00:00
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->where_expression, only_types, step.actions);
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return true;
|
2013-05-28 11:54:37 +00:00
|
|
|
}
|
|
|
|
|
2014-03-28 12:13:58 +00:00
|
|
|
bool ExpressionAnalyzer::appendGroupBy(ExpressionActionsChain & chain, bool only_types)
|
2013-05-28 11:54:37 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertAggregation();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!select_query->group_expression_list)
|
|
|
|
return false;
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
initChain(chain, source_columns);
|
2017-04-01 07:20:54 +00:00
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
ASTs asts = select_query->group_expression_list->children;
|
|
|
|
for (size_t i = 0; i < asts.size(); ++i)
|
|
|
|
{
|
|
|
|
step.required_output.push_back(asts[i]->getColumnName());
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(asts[i], only_types, step.actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return true;
|
2013-05-28 11:54:37 +00:00
|
|
|
}
|
|
|
|
|
2014-03-28 12:13:58 +00:00
|
|
|
void ExpressionAnalyzer::appendAggregateFunctionsArguments(ExpressionActionsChain & chain, bool only_types)
|
2013-05-28 11:54:37 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertAggregation();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
initChain(chain, source_columns);
|
2017-04-01 07:20:54 +00:00
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
for (size_t i = 0; i < aggregate_descriptions.size(); ++i)
|
|
|
|
{
|
|
|
|
for (size_t j = 0; j < aggregate_descriptions[i].argument_names.size(); ++j)
|
|
|
|
{
|
|
|
|
step.required_output.push_back(aggregate_descriptions[i].argument_names[j]);
|
|
|
|
}
|
|
|
|
}
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
getActionsBeforeAggregation(select_query->select_expression_list, step.actions, only_types);
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (select_query->having_expression)
|
|
|
|
getActionsBeforeAggregation(select_query->having_expression, step.actions, only_types);
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (select_query->order_expression_list)
|
|
|
|
getActionsBeforeAggregation(select_query->order_expression_list, step.actions, only_types);
|
2013-05-28 11:54:37 +00:00
|
|
|
}
|
|
|
|
|
2014-03-28 12:13:58 +00:00
|
|
|
bool ExpressionAnalyzer::appendHaving(ExpressionActionsChain & chain, bool only_types)
|
2013-05-28 11:54:37 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertAggregation();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!select_query->having_expression)
|
|
|
|
return false;
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
initChain(chain, aggregated_columns);
|
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
step.required_output.push_back(select_query->having_expression->getColumnName());
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->having_expression, only_types, step.actions);
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return true;
|
2013-05-24 10:49:19 +00:00
|
|
|
}
|
|
|
|
|
2014-03-28 12:13:58 +00:00
|
|
|
void ExpressionAnalyzer::appendSelect(ExpressionActionsChain & chain, bool only_types)
|
2013-05-28 11:54:37 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertSelect();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
initChain(chain, aggregated_columns);
|
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->select_expression_list, only_types, step.actions);
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-02-26 09:55:14 +00:00
|
|
|
for (const auto & child : select_query->select_expression_list->children)
|
2018-03-01 01:25:06 +00:00
|
|
|
step.required_output.push_back(child->getColumnName());
|
2013-05-28 11:54:37 +00:00
|
|
|
}
|
2013-05-24 10:49:19 +00:00
|
|
|
|
2014-03-28 12:13:58 +00:00
|
|
|
bool ExpressionAnalyzer::appendOrderBy(ExpressionActionsChain & chain, bool only_types)
|
2013-05-24 10:49:19 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertSelect();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!select_query->order_expression_list)
|
|
|
|
return false;
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
initChain(chain, aggregated_columns);
|
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->order_expression_list, only_types, step.actions);
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
ASTs asts = select_query->order_expression_list->children;
|
|
|
|
for (size_t i = 0; i < asts.size(); ++i)
|
|
|
|
{
|
|
|
|
ASTOrderByElement * ast = typeid_cast<ASTOrderByElement *>(asts[i].get());
|
|
|
|
if (!ast || ast->children.size() < 1)
|
|
|
|
throw Exception("Bad order expression AST", ErrorCodes::UNKNOWN_TYPE_OF_AST_NODE);
|
|
|
|
ASTPtr order_expression = ast->children.at(0);
|
|
|
|
step.required_output.push_back(order_expression->getColumnName());
|
|
|
|
}
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return true;
|
2013-05-28 11:54:37 +00:00
|
|
|
}
|
|
|
|
|
2018-03-01 05:24:56 +00:00
|
|
|
bool ExpressionAnalyzer::appendLimitBy(ExpressionActionsChain & chain, bool only_types)
|
|
|
|
{
|
|
|
|
assertSelect();
|
|
|
|
|
|
|
|
if (!select_query->limit_by_expression_list)
|
|
|
|
return false;
|
|
|
|
|
|
|
|
initChain(chain, aggregated_columns);
|
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(select_query->limit_by_expression_list, only_types, step.actions);
|
2018-03-01 05:24:56 +00:00
|
|
|
|
|
|
|
for (const auto & child : select_query->limit_by_expression_list->children)
|
|
|
|
step.required_output.push_back(child->getColumnName());
|
|
|
|
|
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
2017-12-01 21:13:25 +00:00
|
|
|
void ExpressionAnalyzer::appendProjectResult(ExpressionActionsChain & chain) const
|
2013-05-28 11:54:37 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
assertSelect();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
initChain(chain, aggregated_columns);
|
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
NamesWithAliases result_columns;
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
ASTs asts = select_query->select_expression_list->children;
|
|
|
|
for (size_t i = 0; i < asts.size(); ++i)
|
|
|
|
{
|
2018-02-26 09:05:06 +00:00
|
|
|
String result_name = asts[i]->getAliasOrColumnName();
|
2018-08-05 03:28:31 +00:00
|
|
|
if (required_result_columns.empty()
|
|
|
|
|| std::find(required_result_columns.begin(), required_result_columns.end(), result_name) != required_result_columns.end())
|
2018-02-26 09:05:06 +00:00
|
|
|
{
|
|
|
|
result_columns.emplace_back(asts[i]->getColumnName(), result_name);
|
|
|
|
step.required_output.push_back(result_columns.back().second);
|
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
step.actions->add(ExpressionAction::project(result_columns));
|
2013-05-24 10:49:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2018-09-04 13:45:39 +00:00
|
|
|
void ExpressionAnalyzer::appendExpression(ExpressionActionsChain & chain, const ASTPtr & expr, bool only_types)
|
2018-09-03 13:36:58 +00:00
|
|
|
{
|
|
|
|
initChain(chain, source_columns);
|
|
|
|
ExpressionActionsChain::Step & step = chain.steps.back();
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(expr, only_types, step.actions);
|
2018-09-03 13:36:58 +00:00
|
|
|
step.required_output.push_back(expr->getColumnName());
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-09-08 03:47:27 +00:00
|
|
|
void ExpressionAnalyzer::getActionsBeforeAggregation(const ASTPtr & ast, ExpressionActionsPtr & actions, bool no_subqueries)
|
2013-05-24 10:49:19 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
ASTFunction * node = typeid_cast<ASTFunction *>(ast.get());
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-02-24 01:31:42 +00:00
|
|
|
if (node && AggregateFunctionFactory::instance().isAggregateFunctionName(node->name))
|
2017-04-01 07:20:54 +00:00
|
|
|
for (auto & argument : node->arguments->children)
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(argument, no_subqueries, actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
else
|
|
|
|
for (auto & child : ast->children)
|
|
|
|
getActionsBeforeAggregation(child, actions, no_subqueries);
|
2013-05-29 11:46:51 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2018-09-03 17:24:46 +00:00
|
|
|
ExpressionActionsPtr ExpressionAnalyzer::getActions(bool add_aliases, bool project_result)
|
2013-05-24 10:49:19 +00:00
|
|
|
{
|
2018-08-30 16:31:20 +00:00
|
|
|
ExpressionActionsPtr actions = std::make_shared<ExpressionActions>(source_columns, context);
|
2017-04-01 07:20:54 +00:00
|
|
|
NamesWithAliases result_columns;
|
|
|
|
Names result_names;
|
|
|
|
|
|
|
|
ASTs asts;
|
|
|
|
|
2018-08-27 17:58:43 +00:00
|
|
|
if (auto node = typeid_cast<const ASTExpressionList *>(query.get()))
|
2017-04-01 07:20:54 +00:00
|
|
|
asts = node->children;
|
|
|
|
else
|
2018-08-27 17:58:43 +00:00
|
|
|
asts = ASTs(1, query);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
for (size_t i = 0; i < asts.size(); ++i)
|
|
|
|
{
|
|
|
|
std::string name = asts[i]->getColumnName();
|
|
|
|
std::string alias;
|
2018-09-03 17:24:46 +00:00
|
|
|
if (add_aliases)
|
2017-04-01 07:20:54 +00:00
|
|
|
alias = asts[i]->getAliasOrColumnName();
|
|
|
|
else
|
|
|
|
alias = name;
|
|
|
|
result_columns.emplace_back(name, alias);
|
|
|
|
result_names.push_back(alias);
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(asts[i], false, actions);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
2018-09-03 17:24:46 +00:00
|
|
|
if (add_aliases)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2018-09-03 17:24:46 +00:00
|
|
|
if (project_result)
|
|
|
|
actions->add(ExpressionAction::project(result_columns));
|
|
|
|
else
|
|
|
|
actions->add(ExpressionAction::addAliases(result_columns));
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
2018-09-05 13:04:28 +00:00
|
|
|
|
|
|
|
if (!(add_aliases && project_result))
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// We will not delete the original columns.
|
2018-02-26 09:05:06 +00:00
|
|
|
for (const auto & column_name_type : source_columns)
|
2017-04-01 07:20:54 +00:00
|
|
|
result_names.push_back(column_name_type.name);
|
|
|
|
}
|
|
|
|
|
|
|
|
actions->finalize(result_names);
|
|
|
|
|
|
|
|
return actions;
|
2013-05-24 10:49:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
ExpressionActionsPtr ExpressionAnalyzer::getConstActions()
|
|
|
|
{
|
2018-08-30 16:31:20 +00:00
|
|
|
ExpressionActionsPtr actions = std::make_shared<ExpressionActions>(NamesAndTypesList(), context);
|
2013-10-17 13:32:32 +00:00
|
|
|
|
2018-10-16 12:34:20 +00:00
|
|
|
getRootActions(query, true, actions, true);
|
2017-04-01 07:20:54 +00:00
|
|
|
return actions;
|
2013-05-24 10:49:19 +00:00
|
|
|
}
|
|
|
|
|
2015-04-18 22:30:43 +00:00
|
|
|
void ExpressionAnalyzer::getAggregateInfo(Names & key_names, AggregateDescriptions & aggregates) const
|
2013-05-24 10:49:19 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
for (const auto & name_and_type : aggregation_keys)
|
|
|
|
key_names.emplace_back(name_and_type.name);
|
2015-03-12 02:22:55 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
aggregates = aggregate_descriptions;
|
2013-05-24 10:49:19 +00:00
|
|
|
}
|
|
|
|
|
2014-07-04 19:13:38 +00:00
|
|
|
void ExpressionAnalyzer::collectUsedColumns()
|
2013-05-30 16:52:21 +00:00
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/** Calculate which columns are required to execute the expression.
|
|
|
|
* Then, delete all other columns from the list of available columns.
|
|
|
|
* After execution, columns will only contain the list of columns needed to read from the table.
|
2017-04-01 07:20:54 +00:00
|
|
|
*/
|
|
|
|
|
|
|
|
NameSet required;
|
|
|
|
NameSet ignored;
|
|
|
|
|
2017-12-14 12:21:01 +00:00
|
|
|
NameSet available_columns;
|
2018-02-26 09:05:06 +00:00
|
|
|
for (const auto & column : source_columns)
|
2017-12-14 12:21:01 +00:00
|
|
|
available_columns.insert(column.name);
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (select_query && select_query->array_join_expression_list())
|
|
|
|
{
|
|
|
|
ASTs & expressions = select_query->array_join_expression_list()->children;
|
|
|
|
for (size_t i = 0; i < expressions.size(); ++i)
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Ignore the top-level identifiers from the ARRAY JOIN section.
|
|
|
|
/// Then add them separately.
|
2017-04-01 07:20:54 +00:00
|
|
|
if (typeid_cast<ASTIdentifier *>(expressions[i].get()))
|
|
|
|
{
|
|
|
|
ignored.insert(expressions[i]->getColumnName());
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Nothing needs to be ignored for expressions in ARRAY JOIN.
|
2017-04-01 07:20:54 +00:00
|
|
|
NameSet empty;
|
2018-10-16 19:00:05 +00:00
|
|
|
RequiredSourceColumnsVisitor visitor(available_columns, required, empty, empty, empty);
|
|
|
|
visitor.visit(expressions[i]);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
ignored.insert(expressions[i]->getAliasOrColumnName());
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/** You also need to ignore the identifiers of the columns that are obtained by JOIN.
|
|
|
|
* (Do not assume that they are required for reading from the "left" table).
|
2017-04-01 07:20:54 +00:00
|
|
|
*/
|
|
|
|
NameSet available_joined_columns;
|
2018-11-02 18:53:23 +00:00
|
|
|
for (const auto & joined_column : analyzed_join.columns_added_by_join)
|
|
|
|
available_joined_columns.insert(joined_column.name_and_type.name);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
NameSet required_joined_columns;
|
2018-07-25 16:11:20 +00:00
|
|
|
|
2018-07-30 13:57:50 +00:00
|
|
|
for (const auto & left_key_ast : analyzed_join.key_asts_left)
|
2018-10-16 19:00:05 +00:00
|
|
|
{
|
2018-10-18 19:27:28 +00:00
|
|
|
NameSet empty;
|
|
|
|
RequiredSourceColumnsVisitor columns_visitor(available_columns, required, ignored, empty, required_joined_columns);
|
2018-10-16 19:00:05 +00:00
|
|
|
columns_visitor.visit(left_key_ast);
|
|
|
|
}
|
2018-07-25 16:11:20 +00:00
|
|
|
|
2018-10-16 19:00:05 +00:00
|
|
|
RequiredSourceColumnsVisitor columns_visitor(available_columns, required, ignored, available_joined_columns, required_joined_columns);
|
|
|
|
columns_visitor.visit(query);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-07-30 13:57:50 +00:00
|
|
|
for (auto it = analyzed_join.columns_added_by_join.begin(); it != analyzed_join.columns_added_by_join.end();)
|
2017-12-25 21:57:19 +00:00
|
|
|
{
|
2018-07-31 11:31:18 +00:00
|
|
|
if (required_joined_columns.count(it->name_and_type.name))
|
2017-12-25 21:57:19 +00:00
|
|
|
++it;
|
|
|
|
else
|
2018-07-30 13:57:50 +00:00
|
|
|
analyzed_join.columns_added_by_join.erase(it++);
|
2017-12-25 21:57:19 +00:00
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-11-02 18:53:23 +00:00
|
|
|
NameSet source_columns_set;
|
|
|
|
for (const auto & type_name : source_columns)
|
|
|
|
source_columns_set.insert(type_name.name);
|
|
|
|
analyzed_join.createJoinedBlockActions(source_columns_set, select_query, context);
|
2018-07-31 11:31:18 +00:00
|
|
|
|
2018-07-30 13:57:50 +00:00
|
|
|
/// Some columns from right join key may be used in query. This columns will be appended to block during join.
|
|
|
|
for (const auto & right_key_name : analyzed_join.key_names_right)
|
2018-07-27 20:01:54 +00:00
|
|
|
if (required_joined_columns.count(right_key_name))
|
2018-07-30 13:57:50 +00:00
|
|
|
analyzed_join.columns_added_by_join_from_right_keys.insert(right_key_name);
|
2018-07-27 20:01:54 +00:00
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Insert the columns required for the ARRAY JOIN calculation into the required columns list.
|
2017-04-01 07:20:54 +00:00
|
|
|
NameSet array_join_sources;
|
|
|
|
for (const auto & result_source : array_join_result_to_source)
|
|
|
|
array_join_sources.insert(result_source.second);
|
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
for (const auto & column_name_type : source_columns)
|
2017-04-01 07:20:54 +00:00
|
|
|
if (array_join_sources.count(column_name_type.name))
|
|
|
|
required.insert(column_name_type.name);
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// You need to read at least one column to find the number of rows.
|
2018-02-22 10:54:28 +00:00
|
|
|
if (select_query && required.empty())
|
2018-02-26 09:05:06 +00:00
|
|
|
required.insert(ExpressionActions::getSmallestColumn(source_columns));
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-02-28 01:29:55 +00:00
|
|
|
NameSet unknown_required_source_columns = required;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
for (NamesAndTypesList::iterator it = source_columns.begin(); it != source_columns.end();)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2018-02-26 09:05:06 +00:00
|
|
|
unknown_required_source_columns.erase(it->name);
|
2017-12-25 21:57:19 +00:00
|
|
|
|
|
|
|
if (!required.count(it->name))
|
2018-02-26 09:05:06 +00:00
|
|
|
source_columns.erase(it++);
|
2017-12-25 21:57:19 +00:00
|
|
|
else
|
|
|
|
++it;
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
2018-02-26 21:00:42 +00:00
|
|
|
/// If there are virtual columns among the unknown columns. Remove them from the list of unknown and add
|
|
|
|
/// in columns list, so that when further processing they are also considered.
|
2017-04-01 07:20:54 +00:00
|
|
|
if (storage)
|
|
|
|
{
|
2018-02-26 09:05:06 +00:00
|
|
|
for (auto it = unknown_required_source_columns.begin(); it != unknown_required_source_columns.end();)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
if (storage->hasColumn(*it))
|
|
|
|
{
|
2018-02-26 09:05:06 +00:00
|
|
|
source_columns.push_back(storage->getColumn(*it));
|
|
|
|
unknown_required_source_columns.erase(it++);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
++it;
|
|
|
|
}
|
|
|
|
}
|
2018-02-28 01:29:55 +00:00
|
|
|
|
|
|
|
if (!unknown_required_source_columns.empty())
|
|
|
|
throw Exception("Unknown identifier: " + *unknown_required_source_columns.begin(), ErrorCodes::UNKNOWN_IDENTIFIER);
|
2013-06-20 13:50:55 +00:00
|
|
|
}
|
|
|
|
|
2018-07-25 16:11:20 +00:00
|
|
|
|
2018-02-26 09:05:06 +00:00
|
|
|
Names ExpressionAnalyzer::getRequiredSourceColumns() const
|
2013-06-20 13:50:55 +00:00
|
|
|
{
|
2018-02-28 01:29:55 +00:00
|
|
|
return source_columns.getNames();
|
2013-05-30 16:52:21 +00:00
|
|
|
}
|
|
|
|
|
2013-05-24 10:49:19 +00:00
|
|
|
}
|