2020-01-21 13:53:30 +00:00
|
|
|
#include "Common/quoteString.h"
|
2019-08-15 13:54:59 +00:00
|
|
|
#include <Common/typeid_cast.h>
|
2019-10-07 18:56:03 +00:00
|
|
|
#include <Common/PODArray.h>
|
2020-01-24 00:33:38 +00:00
|
|
|
#include <Core/Row.h>
|
2019-08-15 13:54:59 +00:00
|
|
|
|
2018-10-12 15:41:28 +00:00
|
|
|
#include <Functions/FunctionFactory.h>
|
|
|
|
#include <Functions/FunctionsMiscellaneous.h>
|
|
|
|
|
|
|
|
#include <AggregateFunctions/AggregateFunctionFactory.h>
|
|
|
|
|
|
|
|
#include <DataTypes/DataTypeSet.h>
|
|
|
|
#include <DataTypes/DataTypesNumber.h>
|
|
|
|
#include <DataTypes/DataTypeFunction.h>
|
2019-12-18 20:36:51 +00:00
|
|
|
#include <DataTypes/DataTypeString.h>
|
2018-10-12 15:41:28 +00:00
|
|
|
#include <DataTypes/DataTypeTuple.h>
|
2020-01-24 00:33:38 +00:00
|
|
|
#include <DataTypes/DataTypeArray.h>
|
2018-10-12 15:41:28 +00:00
|
|
|
#include <DataTypes/DataTypeLowCardinality.h>
|
|
|
|
#include <DataTypes/FieldToDataType.h>
|
|
|
|
|
|
|
|
#include <DataStreams/LazyBlockInputStream.h>
|
|
|
|
|
|
|
|
#include <Columns/ColumnSet.h>
|
|
|
|
#include <Columns/ColumnConst.h>
|
|
|
|
#include <Columns/ColumnsNumber.h>
|
|
|
|
|
|
|
|
#include <Storages/StorageSet.h>
|
|
|
|
|
|
|
|
#include <Parsers/ASTFunction.h>
|
|
|
|
#include <Parsers/ASTIdentifier.h>
|
|
|
|
#include <Parsers/ASTLiteral.h>
|
|
|
|
#include <Parsers/ASTSelectQuery.h>
|
|
|
|
#include <Parsers/ASTSubquery.h>
|
|
|
|
#include <Parsers/ASTTablesInSelectQuery.h>
|
|
|
|
|
|
|
|
#include <Interpreters/ExpressionActions.h>
|
2019-10-23 13:59:03 +00:00
|
|
|
#include <Interpreters/misc.h>
|
2018-10-12 15:41:28 +00:00
|
|
|
#include <Interpreters/ActionsVisitor.h>
|
|
|
|
#include <Interpreters/InterpreterSelectWithUnionQuery.h>
|
|
|
|
#include <Interpreters/Set.h>
|
|
|
|
#include <Interpreters/evaluateConstantExpression.h>
|
|
|
|
#include <Interpreters/convertFieldToType.h>
|
|
|
|
#include <Interpreters/interpretSubquery.h>
|
2019-02-11 14:19:09 +00:00
|
|
|
#include <Interpreters/DatabaseAndTableWithAlias.h>
|
2019-12-18 20:36:51 +00:00
|
|
|
#include <Interpreters/IdentifierSemantic.h>
|
2018-10-12 15:41:28 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2020-02-25 18:02:41 +00:00
|
|
|
extern const int ILLEGAL_TYPE_OF_ARGUMENT;
|
2018-10-12 15:41:28 +00:00
|
|
|
extern const int UNKNOWN_IDENTIFIER;
|
|
|
|
extern const int NOT_AN_AGGREGATE;
|
|
|
|
extern const int UNEXPECTED_EXPRESSION;
|
|
|
|
extern const int TYPE_MISMATCH;
|
|
|
|
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
|
2020-01-24 00:33:38 +00:00
|
|
|
extern const int INCORRECT_ELEMENT_OF_SET;
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
|
2019-12-15 06:34:43 +00:00
|
|
|
static NamesAndTypesList::iterator findColumn(const String & name, NamesAndTypesList & cols)
|
2018-11-02 18:53:23 +00:00
|
|
|
{
|
|
|
|
return std::find_if(cols.begin(), cols.end(),
|
|
|
|
[&](const NamesAndTypesList::value_type & val) { return val.name == name; });
|
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2020-01-24 21:36:28 +00:00
|
|
|
template<typename Collection>
|
2020-04-06 13:30:16 +00:00
|
|
|
static Block createBlockFromCollection(const Collection & collection, const DataTypes & types, const Context & context)
|
2020-01-24 00:33:38 +00:00
|
|
|
{
|
|
|
|
size_t columns_num = types.size();
|
|
|
|
MutableColumns columns(columns_num);
|
|
|
|
for (size_t i = 0; i < columns_num; ++i)
|
|
|
|
columns[i] = types[i]->createColumn();
|
|
|
|
|
|
|
|
Row tuple_values;
|
2020-01-24 21:36:28 +00:00
|
|
|
for (const auto & value : collection)
|
2020-01-24 00:33:38 +00:00
|
|
|
{
|
|
|
|
if (columns_num == 1)
|
|
|
|
{
|
|
|
|
auto field = convertFieldToType(value, *types[0]);
|
2020-04-06 13:30:16 +00:00
|
|
|
if (!field.isNull() || context.getSettingsRef().transform_null_in)
|
2020-01-24 00:33:38 +00:00
|
|
|
columns[0]->insert(std::move(field));
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
if (value.getType() != Field::Types::Tuple)
|
|
|
|
throw Exception("Invalid type in set. Expected tuple, got "
|
|
|
|
+ String(value.getTypeName()), ErrorCodes::INCORRECT_ELEMENT_OF_SET);
|
|
|
|
|
|
|
|
const auto & tuple = DB::get<const Tuple &>(value);
|
|
|
|
size_t tuple_size = tuple.size();
|
|
|
|
|
|
|
|
if (tuple_size != columns_num)
|
|
|
|
throw Exception("Incorrect size of tuple in set: " + toString(tuple_size)
|
|
|
|
+ " instead of " + toString(columns_num), ErrorCodes::INCORRECT_ELEMENT_OF_SET);
|
|
|
|
|
|
|
|
if (tuple_values.empty())
|
|
|
|
tuple_values.resize(tuple_size);
|
|
|
|
|
|
|
|
size_t i = 0;
|
|
|
|
for (; i < tuple_size; ++i)
|
|
|
|
{
|
|
|
|
tuple_values[i] = convertFieldToType(tuple[i], *types[i]);
|
2020-04-06 13:30:16 +00:00
|
|
|
if (tuple_values[i].isNull() && !context.getSettingsRef().transform_null_in)
|
2020-01-24 00:33:38 +00:00
|
|
|
break;
|
|
|
|
}
|
|
|
|
|
|
|
|
if (i == tuple_size)
|
|
|
|
for (i = 0; i < tuple_size; ++i)
|
|
|
|
columns[i]->insert(std::move(tuple_values[i]));
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
Block res;
|
|
|
|
for (size_t i = 0; i < columns_num; ++i)
|
|
|
|
res.insert(ColumnWithTypeAndName{std::move(columns[i]), types[i], "_" + toString(i)});
|
|
|
|
return res;
|
|
|
|
}
|
|
|
|
|
2019-01-22 12:33:56 +00:00
|
|
|
SetPtr makeExplicitSet(
|
|
|
|
const ASTFunction * node, const Block & sample_block, bool create_ordered_set,
|
|
|
|
const Context & context, const SizeLimits & size_limits, PreparedSets & prepared_sets)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
|
|
|
const IAST & args = *node->arguments;
|
|
|
|
|
|
|
|
if (args.children.size() != 2)
|
|
|
|
throw Exception("Wrong number of arguments passed to function in", ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
|
|
|
|
|
|
|
|
const ASTPtr & left_arg = args.children.at(0);
|
|
|
|
const ASTPtr & right_arg = args.children.at(1);
|
|
|
|
|
2019-01-22 12:33:56 +00:00
|
|
|
const DataTypePtr & left_arg_type = sample_block.getByName(left_arg->getColumnName()).type;
|
|
|
|
|
|
|
|
DataTypes set_element_types = {left_arg_type};
|
2020-04-22 05:39:31 +00:00
|
|
|
const auto * left_tuple_type = typeid_cast<const DataTypeTuple *>(left_arg_type.get());
|
2019-01-22 12:33:56 +00:00
|
|
|
if (left_tuple_type && left_tuple_type->getElements().size() != 1)
|
|
|
|
set_element_types = left_tuple_type->getElements();
|
|
|
|
|
|
|
|
for (auto & element_type : set_element_types)
|
|
|
|
if (const auto * low_cardinality_type = typeid_cast<const DataTypeLowCardinality *>(element_type.get()))
|
|
|
|
element_type = low_cardinality_type->getDictionaryType();
|
|
|
|
|
|
|
|
auto set_key = PreparedSetKey::forLiteral(*right_arg, set_element_types);
|
|
|
|
if (prepared_sets.count(set_key))
|
|
|
|
return prepared_sets.at(set_key); /// Already prepared.
|
|
|
|
|
2020-01-24 21:36:28 +00:00
|
|
|
auto [right_arg_value, right_arg_type] = evaluateConstantExpression(right_arg, context);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2020-03-23 02:12:31 +00:00
|
|
|
std::function<size_t(const DataTypePtr &)> get_type_depth;
|
|
|
|
get_type_depth = [&get_type_depth](const DataTypePtr & type) -> size_t
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2020-04-22 05:39:31 +00:00
|
|
|
if (const auto * array_type = typeid_cast<const DataTypeArray *>(type.get()))
|
2020-03-23 02:12:31 +00:00
|
|
|
return 1 + get_type_depth(array_type->getNestedType());
|
2020-04-22 05:39:31 +00:00
|
|
|
else if (const auto * tuple_type = typeid_cast<const DataTypeTuple *>(type.get()))
|
2020-03-23 02:12:31 +00:00
|
|
|
return 1 + (tuple_type->getElements().empty() ? 0 : get_type_depth(tuple_type->getElements().at(0)));
|
2018-10-12 15:41:28 +00:00
|
|
|
|
|
|
|
return 0;
|
|
|
|
};
|
|
|
|
|
2020-03-23 02:12:31 +00:00
|
|
|
const size_t left_type_depth = get_type_depth(left_arg_type);
|
|
|
|
const size_t right_type_depth = get_type_depth(right_arg_type);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2020-01-24 21:36:28 +00:00
|
|
|
auto throw_unsupported_type = [](const auto & type)
|
|
|
|
{
|
2020-02-02 01:33:13 +00:00
|
|
|
throw Exception("Unsupported value type at the right-side of IN: "
|
2020-01-24 21:36:28 +00:00
|
|
|
+ type->getName() + ".", ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT);
|
|
|
|
};
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-05-19 19:22:51 +00:00
|
|
|
Block block;
|
2018-10-12 15:41:28 +00:00
|
|
|
/// 1 in 1; (1, 2) in (1, 2); identity(tuple(tuple(tuple(1)))) in tuple(tuple(tuple(1))); etc.
|
2020-01-24 00:33:38 +00:00
|
|
|
if (left_type_depth == right_type_depth)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2020-01-24 00:33:38 +00:00
|
|
|
Array array{right_arg_value};
|
2020-04-06 13:30:16 +00:00
|
|
|
block = createBlockFromCollection(array, set_element_types, context);
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
/// 1 in (1, 2); (1, 2) in ((1, 2), (3, 4)); etc.
|
2020-01-24 00:33:38 +00:00
|
|
|
else if (left_type_depth + 1 == right_type_depth)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2020-01-24 21:36:28 +00:00
|
|
|
auto type_index = right_arg_type->getTypeId();
|
|
|
|
if (type_index == TypeIndex::Tuple)
|
2020-04-06 13:30:16 +00:00
|
|
|
block = createBlockFromCollection(DB::get<const Tuple &>(right_arg_value), set_element_types, context);
|
2020-01-24 21:36:28 +00:00
|
|
|
else if (type_index == TypeIndex::Array)
|
2020-04-06 13:30:16 +00:00
|
|
|
block = createBlockFromCollection(DB::get<const Array &>(right_arg_value), set_element_types, context);
|
2020-01-24 21:36:28 +00:00
|
|
|
else
|
|
|
|
throw_unsupported_type(right_arg_type);
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
else
|
2020-01-24 21:36:28 +00:00
|
|
|
throw_unsupported_type(right_arg_type);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2020-04-09 08:27:55 +00:00
|
|
|
SetPtr set = std::make_shared<Set>(size_limits, create_ordered_set, context.getSettingsRef().transform_null_in);
|
2019-05-19 19:22:51 +00:00
|
|
|
|
|
|
|
set->setHeader(block);
|
|
|
|
set->insertFromBlock(block);
|
|
|
|
|
2019-01-22 12:33:56 +00:00
|
|
|
prepared_sets[set_key] = set;
|
|
|
|
return set;
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
ScopeStack::ScopeStack(const ExpressionActionsPtr & actions, const Context & context_)
|
|
|
|
: context(context_)
|
|
|
|
{
|
|
|
|
stack.emplace_back();
|
|
|
|
stack.back().actions = actions;
|
|
|
|
|
|
|
|
const Block & sample_block = actions->getSampleBlock();
|
|
|
|
for (size_t i = 0, size = sample_block.columns(); i < size; ++i)
|
|
|
|
stack.back().new_columns.insert(sample_block.getByPosition(i).name);
|
|
|
|
}
|
|
|
|
|
|
|
|
void ScopeStack::pushLevel(const NamesAndTypesList & input_columns)
|
|
|
|
{
|
|
|
|
stack.emplace_back();
|
|
|
|
Level & prev = stack[stack.size() - 2];
|
|
|
|
|
|
|
|
ColumnsWithTypeAndName all_columns;
|
|
|
|
NameSet new_names;
|
|
|
|
|
2020-03-08 23:48:08 +00:00
|
|
|
for (const auto & input_column : input_columns)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2020-03-08 23:48:08 +00:00
|
|
|
all_columns.emplace_back(nullptr, input_column.type, input_column.name);
|
|
|
|
new_names.insert(input_column.name);
|
|
|
|
stack.back().new_columns.insert(input_column.name);
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
const Block & prev_sample_block = prev.actions->getSampleBlock();
|
|
|
|
for (size_t i = 0, size = prev_sample_block.columns(); i < size; ++i)
|
|
|
|
{
|
|
|
|
const ColumnWithTypeAndName & col = prev_sample_block.getByPosition(i);
|
|
|
|
if (!new_names.count(col.name))
|
|
|
|
all_columns.push_back(col);
|
|
|
|
}
|
|
|
|
|
|
|
|
stack.back().actions = std::make_shared<ExpressionActions>(all_columns, context);
|
|
|
|
}
|
|
|
|
|
|
|
|
size_t ScopeStack::getColumnLevel(const std::string & name)
|
|
|
|
{
|
|
|
|
for (int i = static_cast<int>(stack.size()) - 1; i >= 0; --i)
|
|
|
|
if (stack[i].new_columns.count(name))
|
|
|
|
return i;
|
|
|
|
|
|
|
|
throw Exception("Unknown identifier: " + name, ErrorCodes::UNKNOWN_IDENTIFIER);
|
|
|
|
}
|
|
|
|
|
|
|
|
void ScopeStack::addAction(const ExpressionAction & action)
|
|
|
|
{
|
|
|
|
size_t level = 0;
|
|
|
|
Names required = action.getNeededColumns();
|
2020-03-08 23:48:08 +00:00
|
|
|
for (const auto & elem : required)
|
|
|
|
level = std::max(level, getColumnLevel(elem));
|
2018-10-12 15:41:28 +00:00
|
|
|
|
|
|
|
Names added;
|
|
|
|
stack[level].actions->add(action, added);
|
|
|
|
|
|
|
|
stack[level].new_columns.insert(added.begin(), added.end());
|
|
|
|
|
2020-03-08 23:48:08 +00:00
|
|
|
for (const auto & elem : added)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2020-03-08 23:48:08 +00:00
|
|
|
const ColumnWithTypeAndName & col = stack[level].actions->getSampleBlock().getByName(elem);
|
2018-10-12 15:41:28 +00:00
|
|
|
for (size_t j = level + 1; j < stack.size(); ++j)
|
|
|
|
stack[j].actions->addInput(col);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
ExpressionActionsPtr ScopeStack::popLevel()
|
|
|
|
{
|
|
|
|
ExpressionActionsPtr res = stack.back().actions;
|
|
|
|
stack.pop_back();
|
|
|
|
return res;
|
|
|
|
}
|
|
|
|
|
|
|
|
const Block & ScopeStack::getSampleBlock() const
|
|
|
|
{
|
|
|
|
return stack.back().actions->getSampleBlock();
|
|
|
|
}
|
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
struct CachedColumnName
|
|
|
|
{
|
|
|
|
String cached;
|
|
|
|
|
|
|
|
const String & get(const ASTPtr & ast)
|
|
|
|
{
|
|
|
|
if (cached.empty())
|
|
|
|
cached = ast->getColumnName();
|
|
|
|
return cached;
|
|
|
|
}
|
|
|
|
};
|
|
|
|
|
|
|
|
bool ActionsMatcher::needChildVisit(const ASTPtr & node, const ASTPtr & child)
|
|
|
|
{
|
|
|
|
/// Visit children themself
|
|
|
|
if (node->as<ASTIdentifier>() ||
|
|
|
|
node->as<ASTFunction>() ||
|
|
|
|
node->as<ASTLiteral>())
|
|
|
|
return false;
|
|
|
|
|
|
|
|
/// Do not go to FROM, JOIN, UNION.
|
|
|
|
if (child->as<ASTTableExpression>() ||
|
|
|
|
child->as<ASTSelectQuery>())
|
|
|
|
return false;
|
|
|
|
|
|
|
|
return true;
|
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
void ActionsMatcher::visit(const ASTPtr & ast, Data & data)
|
2018-10-16 12:34:20 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
if (const auto * identifier = ast->as<ASTIdentifier>())
|
|
|
|
visit(*identifier, ast, data);
|
|
|
|
else if (const auto * node = ast->as<ASTFunction>())
|
|
|
|
visit(*node, ast, data);
|
|
|
|
else if (const auto * literal = ast->as<ASTLiteral>())
|
|
|
|
visit(*literal, ast, data);
|
2018-10-16 12:34:20 +00:00
|
|
|
}
|
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
void ActionsMatcher::visit(const ASTIdentifier & identifier, const ASTPtr & ast, Data & data)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
CachedColumnName column_name;
|
2019-08-15 14:22:33 +00:00
|
|
|
if (data.hasColumn(column_name.get(ast)))
|
|
|
|
return;
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 14:22:33 +00:00
|
|
|
if (!data.only_consts)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
/// The requested column is not in the block.
|
|
|
|
/// If such a column exists in the table, then the user probably forgot to surround it with an aggregate function or add it to GROUP BY.
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
bool found = false;
|
|
|
|
for (const auto & column_name_type : data.source_columns)
|
|
|
|
if (column_name_type.name == column_name.get(ast))
|
|
|
|
found = true;
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
if (found)
|
2020-01-21 13:53:30 +00:00
|
|
|
throw Exception("Column " + backQuote(column_name.get(ast)) + " is not under aggregate function and not in GROUP BY",
|
2019-08-15 13:54:59 +00:00
|
|
|
ErrorCodes::NOT_AN_AGGREGATE);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
/// Special check for WITH statement alias. Add alias action to be able to use this alias.
|
|
|
|
if (identifier.prefer_alias_to_column_name && !identifier.alias.empty())
|
2019-08-15 14:22:33 +00:00
|
|
|
data.addAction(ExpressionAction::addAliases({{identifier.name, identifier.alias}}));
|
2019-08-15 13:54:59 +00:00
|
|
|
}
|
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
void ActionsMatcher::visit(const ASTFunction & node, const ASTPtr & ast, Data & data)
|
|
|
|
{
|
|
|
|
CachedColumnName column_name;
|
|
|
|
if (data.hasColumn(column_name.get(ast)))
|
|
|
|
return;
|
2018-11-12 18:19:16 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
if (node.name == "lambda")
|
|
|
|
throw Exception("Unexpected lambda expression", ErrorCodes::UNEXPECTED_EXPRESSION);
|
|
|
|
|
|
|
|
/// Function arrayJoin.
|
|
|
|
if (node.name == "arrayJoin")
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
if (node.arguments->children.size() != 1)
|
|
|
|
throw Exception("arrayJoin requires exactly 1 argument", ErrorCodes::TYPE_MISMATCH);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
ASTPtr arg = node.arguments->children.at(0);
|
|
|
|
visit(arg, data);
|
|
|
|
if (!data.only_consts)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
String result_name = column_name.get(ast);
|
2019-08-15 14:22:33 +00:00
|
|
|
data.addAction(ExpressionAction::copyColumn(arg->getColumnName(), result_name));
|
2019-08-15 13:54:59 +00:00
|
|
|
NameSet joined_columns;
|
|
|
|
joined_columns.insert(result_name);
|
2019-08-15 14:22:33 +00:00
|
|
|
data.addAction(ExpressionAction::arrayJoin(joined_columns, false, data.context));
|
2019-08-15 13:54:59 +00:00
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
return;
|
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
SetPtr prepared_set;
|
|
|
|
if (functionIsInOrGlobalInOperator(node.name))
|
|
|
|
{
|
|
|
|
/// Let's find the type of the first argument (then getActionsImpl will be called again and will not affect anything).
|
|
|
|
visit(node.arguments->children.at(0), data);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2020-01-27 21:57:44 +00:00
|
|
|
if (!data.no_makeset && (prepared_set = makeSet(node, data, data.no_subqueries)))
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
/// Transform tuple or subquery into a set.
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
if (!data.only_consts)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
/// We are in the part of the tree that we are not going to compute. You just need to define types.
|
2020-05-14 14:21:38 +00:00
|
|
|
/// Do not subquery and create sets. We replace "in*" function to "in*IgnoreSet".
|
|
|
|
|
|
|
|
auto argument_name = node.arguments->children.at(0)->getColumnName();
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 14:22:33 +00:00
|
|
|
data.addAction(ExpressionAction::applyFunction(
|
2020-05-14 14:21:38 +00:00
|
|
|
FunctionFactory::instance().get(node.name + "IgnoreSet", data.context),
|
|
|
|
{ argument_name, argument_name },
|
2019-08-15 13:54:59 +00:00
|
|
|
column_name.get(ast)));
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
return;
|
|
|
|
}
|
2019-08-15 13:54:59 +00:00
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
if (AggregateFunctionFactory::instance().isAggregateFunctionName(node.name))
|
|
|
|
return;
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
/// Context object that we pass to function should live during query.
|
|
|
|
const Context & function_context = data.context.hasQueryContext()
|
|
|
|
? data.context.getQueryContext()
|
|
|
|
: data.context;
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-12-08 21:06:37 +00:00
|
|
|
FunctionOverloadResolverPtr function_builder;
|
2019-08-15 13:54:59 +00:00
|
|
|
try
|
|
|
|
{
|
|
|
|
function_builder = FunctionFactory::instance().get(node.name, function_context);
|
|
|
|
}
|
|
|
|
catch (DB::Exception & e)
|
|
|
|
{
|
|
|
|
auto hints = AggregateFunctionFactory::instance().getHints(node.name);
|
|
|
|
if (!hints.empty())
|
|
|
|
e.addMessage("Or unknown aggregate function " + node.name + ". Maybe you meant: " + toString(hints));
|
|
|
|
e.rethrow();
|
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
Names argument_names;
|
|
|
|
DataTypes argument_types;
|
|
|
|
bool arguments_present = true;
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
/// If the function has an argument-lambda expression, you need to determine its type before the recursive call.
|
|
|
|
bool has_lambda_arguments = false;
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
for (size_t arg = 0; arg < node.arguments->children.size(); ++arg)
|
|
|
|
{
|
|
|
|
auto & child = node.arguments->children[arg];
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
const auto * lambda = child->as<ASTFunction>();
|
2019-12-18 20:36:51 +00:00
|
|
|
const auto * identifier = child->as<ASTIdentifier>();
|
2019-08-15 13:54:59 +00:00
|
|
|
if (lambda && lambda->name == "lambda")
|
|
|
|
{
|
|
|
|
/// If the argument is a lambda expression, just remember its approximate type.
|
|
|
|
if (lambda->arguments->children.size() != 2)
|
|
|
|
throw Exception("lambda requires two arguments", ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
const auto * lambda_args_tuple = lambda->arguments->children.at(0)->as<ASTFunction>();
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
if (!lambda_args_tuple || lambda_args_tuple->name != "tuple")
|
|
|
|
throw Exception("First argument of lambda must be a tuple", ErrorCodes::TYPE_MISMATCH);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
has_lambda_arguments = true;
|
|
|
|
argument_types.emplace_back(std::make_shared<DataTypeFunction>(DataTypes(lambda_args_tuple->arguments->children.size())));
|
|
|
|
/// Select the name in the next cycle.
|
|
|
|
argument_names.emplace_back();
|
|
|
|
}
|
|
|
|
else if (functionIsInOrGlobalInOperator(node.name) && arg == 1 && prepared_set)
|
|
|
|
{
|
|
|
|
ColumnWithTypeAndName column;
|
|
|
|
column.type = std::make_shared<DataTypeSet>();
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
/// If the argument is a set given by an enumeration of values (so, the set was already built), give it a unique name,
|
|
|
|
/// so that sets with the same literal representation do not fuse together (they can have different types).
|
|
|
|
if (!prepared_set->empty())
|
2020-04-14 17:41:06 +00:00
|
|
|
column.name = data.getUniqueName("__set");
|
2019-08-15 13:54:59 +00:00
|
|
|
else
|
2020-03-30 18:00:38 +00:00
|
|
|
column.name = child->getColumnName();
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 14:22:33 +00:00
|
|
|
if (!data.hasColumn(column.name))
|
2019-08-15 13:54:59 +00:00
|
|
|
{
|
2019-10-27 18:12:40 +00:00
|
|
|
auto column_set = ColumnSet::create(1, prepared_set);
|
|
|
|
/// If prepared_set is not empty, we have a set made with literals.
|
|
|
|
/// Create a const ColumnSet to make constant folding work
|
|
|
|
if (!prepared_set->empty())
|
|
|
|
column.column = ColumnConst::create(std::move(column_set), 1);
|
|
|
|
else
|
|
|
|
column.column = std::move(column_set);
|
2019-08-15 14:22:33 +00:00
|
|
|
data.addAction(ExpressionAction::addColumn(column));
|
2019-08-15 13:54:59 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
argument_types.push_back(column.type);
|
|
|
|
argument_names.push_back(column.name);
|
|
|
|
}
|
2020-05-19 18:12:30 +00:00
|
|
|
else if (identifier && (functionIsJoinGet(node.name) || functionIsDictGet(node.name)) && arg == 0)
|
2019-12-18 20:36:51 +00:00
|
|
|
{
|
2020-03-12 18:04:29 +00:00
|
|
|
auto table_id = IdentifierSemantic::extractDatabaseAndTable(*identifier);
|
|
|
|
table_id = data.context.resolveStorageID(table_id, Context::ResolveOrdinary);
|
2019-12-18 20:36:51 +00:00
|
|
|
auto column_string = ColumnString::create();
|
2020-03-12 18:04:29 +00:00
|
|
|
column_string->insert(table_id.getDatabaseName() + "." + table_id.getTableName());
|
2019-12-18 20:36:51 +00:00
|
|
|
ColumnWithTypeAndName column(
|
|
|
|
ColumnConst::create(std::move(column_string), 1),
|
|
|
|
std::make_shared<DataTypeString>(),
|
2020-05-19 18:12:30 +00:00
|
|
|
data.getUniqueName("__" + node.name));
|
2019-12-18 20:36:51 +00:00
|
|
|
data.addAction(ExpressionAction::addColumn(column));
|
|
|
|
argument_types.push_back(column.type);
|
|
|
|
argument_names.push_back(column.name);
|
|
|
|
}
|
2019-08-15 13:54:59 +00:00
|
|
|
else
|
|
|
|
{
|
|
|
|
/// If the argument is not a lambda expression, call it recursively and find out its type.
|
|
|
|
visit(child, data);
|
2020-03-30 18:00:38 +00:00
|
|
|
|
|
|
|
// In the above visit() call, if the argument is a literal, we
|
|
|
|
// generated a unique column name for it. Use it instead of a generic
|
|
|
|
// display name.
|
|
|
|
auto child_column_name = child->getColumnName();
|
2020-04-22 05:39:31 +00:00
|
|
|
const auto * as_literal = child->as<ASTLiteral>();
|
2020-03-30 20:47:28 +00:00
|
|
|
if (as_literal)
|
2020-03-30 18:00:38 +00:00
|
|
|
{
|
2020-03-30 20:47:28 +00:00
|
|
|
assert(!as_literal->unique_column_name.empty());
|
|
|
|
child_column_name = as_literal->unique_column_name;
|
2020-03-30 18:00:38 +00:00
|
|
|
}
|
|
|
|
|
2020-03-18 03:27:32 +00:00
|
|
|
if (data.hasColumn(child_column_name))
|
2019-08-15 13:54:59 +00:00
|
|
|
{
|
2020-03-18 03:27:32 +00:00
|
|
|
argument_types.push_back(data.getSampleBlock().getByName(child_column_name).type);
|
|
|
|
argument_names.push_back(child_column_name);
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
if (data.only_consts)
|
|
|
|
arguments_present = false;
|
2018-10-12 15:41:28 +00:00
|
|
|
else
|
2020-03-18 03:27:32 +00:00
|
|
|
throw Exception("Unknown identifier: " + child_column_name, ErrorCodes::UNKNOWN_IDENTIFIER);
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
}
|
2019-08-15 13:54:59 +00:00
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
if (data.only_consts && !arguments_present)
|
|
|
|
return;
|
|
|
|
|
|
|
|
if (has_lambda_arguments && !data.only_consts)
|
|
|
|
{
|
|
|
|
function_builder->getLambdaArgumentTypes(argument_types);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
/// Call recursively for lambda expressions.
|
|
|
|
for (size_t i = 0; i < node.arguments->children.size(); ++i)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
ASTPtr child = node.arguments->children[i];
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
const auto * lambda = child->as<ASTFunction>();
|
|
|
|
if (lambda && lambda->name == "lambda")
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
const DataTypeFunction * lambda_type = typeid_cast<const DataTypeFunction *>(argument_types[i].get());
|
|
|
|
const auto * lambda_args_tuple = lambda->arguments->children.at(0)->as<ASTFunction>();
|
|
|
|
const ASTs & lambda_arg_asts = lambda_args_tuple->arguments->children;
|
|
|
|
NamesAndTypesList lambda_arguments;
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
for (size_t j = 0; j < lambda_arg_asts.size(); ++j)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
auto opt_arg_name = tryGetIdentifierName(lambda_arg_asts[j]);
|
|
|
|
if (!opt_arg_name)
|
|
|
|
throw Exception("lambda argument declarations must be identifiers", ErrorCodes::TYPE_MISMATCH);
|
|
|
|
|
|
|
|
lambda_arguments.emplace_back(*opt_arg_name, lambda_type->getArgumentTypes()[j]);
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
2019-08-15 13:54:59 +00:00
|
|
|
|
|
|
|
data.actions_stack.pushLevel(lambda_arguments);
|
|
|
|
visit(lambda->arguments->children.at(1), data);
|
|
|
|
ExpressionActionsPtr lambda_actions = data.actions_stack.popLevel();
|
|
|
|
|
|
|
|
String result_name = lambda->arguments->children.at(1)->getColumnName();
|
|
|
|
lambda_actions->finalize(Names(1, result_name));
|
|
|
|
DataTypePtr result_type = lambda_actions->getSampleBlock().getByName(result_name).type;
|
|
|
|
|
|
|
|
Names captured;
|
|
|
|
Names required = lambda_actions->getRequiredColumns();
|
|
|
|
for (const auto & required_arg : required)
|
|
|
|
if (findColumn(required_arg, lambda_arguments) == lambda_arguments.end())
|
|
|
|
captured.push_back(required_arg);
|
|
|
|
|
|
|
|
/// We can not name `getColumnName()`,
|
|
|
|
/// because it does not uniquely define the expression (the types of arguments can be different).
|
2020-04-14 17:41:06 +00:00
|
|
|
String lambda_name = data.getUniqueName("__lambda");
|
2019-08-15 13:54:59 +00:00
|
|
|
|
2019-12-09 14:41:55 +00:00
|
|
|
auto function_capture = std::make_unique<FunctionCaptureOverloadResolver>(
|
2019-08-15 13:54:59 +00:00
|
|
|
lambda_actions, captured, lambda_arguments, result_type, result_name);
|
2019-12-09 14:41:55 +00:00
|
|
|
auto function_capture_adapter = std::make_shared<FunctionOverloadResolverAdaptor>(std::move(function_capture));
|
|
|
|
data.addAction(ExpressionAction::applyFunction(function_capture_adapter, captured, lambda_name));
|
2019-08-15 13:54:59 +00:00
|
|
|
|
|
|
|
argument_types[i] = std::make_shared<DataTypeFunction>(lambda_type->getArgumentTypes(), result_type);
|
|
|
|
argument_names[i] = lambda_name;
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
}
|
2019-08-15 13:54:59 +00:00
|
|
|
}
|
2018-10-12 15:41:28 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
if (data.only_consts)
|
|
|
|
{
|
|
|
|
for (const auto & argument_name : argument_names)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 14:22:33 +00:00
|
|
|
if (!data.hasColumn(argument_name))
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
arguments_present = false;
|
|
|
|
break;
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
if (arguments_present)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-08-15 14:22:33 +00:00
|
|
|
data.addAction(ExpressionAction::applyFunction(function_builder, argument_names, column_name.get(ast)));
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-03-30 18:00:38 +00:00
|
|
|
void ActionsMatcher::visit(const ASTLiteral & literal, const ASTPtr & /* ast */,
|
|
|
|
Data & data)
|
2019-08-15 13:54:59 +00:00
|
|
|
{
|
|
|
|
DataTypePtr type = applyVisitor(FieldToDataType(), literal.value);
|
2020-04-01 12:16:39 +00:00
|
|
|
const auto value = convertFieldToType(literal.value, *type);
|
|
|
|
|
|
|
|
// FIXME why do we have a second pass with a clean sample block over the same
|
|
|
|
// AST here? Anyway, do not modify the column name if it is set already.
|
|
|
|
if (literal.unique_column_name.empty())
|
|
|
|
{
|
|
|
|
const auto default_name = literal.getColumnName();
|
2020-04-22 05:39:31 +00:00
|
|
|
const auto & block = data.getSampleBlock();
|
|
|
|
const auto * existing_column = block.findByName(default_name);
|
2020-04-01 12:16:39 +00:00
|
|
|
|
|
|
|
/*
|
2020-04-01 12:21:13 +00:00
|
|
|
* To approximate CSE, bind all identical literals to a single temporary
|
2020-04-01 12:16:39 +00:00
|
|
|
* columns. We try to find the column by its default name, but after that
|
|
|
|
* we have to check that it contains the correct data. This might not be
|
|
|
|
* the case if it is a user-supplied column, or it is from under a join,
|
|
|
|
* etc.
|
|
|
|
* Overall, this is a hack around a generally poor name-based notion of
|
|
|
|
* column identity we currently use.
|
|
|
|
*/
|
|
|
|
if (existing_column
|
|
|
|
&& existing_column->column
|
|
|
|
&& isColumnConst(*existing_column->column)
|
|
|
|
&& existing_column->column->size() == 1
|
2020-04-15 14:10:33 +00:00
|
|
|
&& existing_column->column->operator[](0) == value)
|
2020-04-01 12:16:39 +00:00
|
|
|
{
|
|
|
|
const_cast<ASTLiteral &>(literal).unique_column_name = default_name;
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
const_cast<ASTLiteral &>(literal).unique_column_name
|
2020-04-14 17:41:06 +00:00
|
|
|
= data.getUniqueName(default_name);
|
2020-04-01 12:16:39 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (data.hasColumn(literal.unique_column_name))
|
|
|
|
{
|
|
|
|
return;
|
|
|
|
}
|
2019-08-15 13:54:59 +00:00
|
|
|
|
|
|
|
ColumnWithTypeAndName column;
|
2020-04-01 12:16:39 +00:00
|
|
|
column.name = literal.unique_column_name;
|
|
|
|
column.column = type->createColumnConst(1, value);
|
2019-08-15 13:54:59 +00:00
|
|
|
column.type = type;
|
2020-03-30 18:00:38 +00:00
|
|
|
|
2019-08-15 14:22:33 +00:00
|
|
|
data.addAction(ExpressionAction::addColumn(column));
|
2019-08-15 13:54:59 +00:00
|
|
|
}
|
|
|
|
|
2019-10-27 18:12:40 +00:00
|
|
|
SetPtr ActionsMatcher::makeSet(const ASTFunction & node, Data & data, bool no_subqueries)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
|
|
|
/** You need to convert the right argument to a set.
|
|
|
|
* This can be a table name, a value, a value enumeration, or a subquery.
|
|
|
|
* The enumeration of values is parsed as a function `tuple`.
|
|
|
|
*/
|
2019-08-15 13:54:59 +00:00
|
|
|
const IAST & args = *node.arguments;
|
2019-10-27 18:12:40 +00:00
|
|
|
const ASTPtr & left_in_operand = args.children.at(0);
|
|
|
|
const ASTPtr & right_in_operand = args.children.at(1);
|
2019-08-15 14:22:33 +00:00
|
|
|
const Block & sample_block = data.getSampleBlock();
|
2018-10-12 15:41:28 +00:00
|
|
|
|
|
|
|
/// If the subquery or table name for SELECT.
|
2019-10-27 18:12:40 +00:00
|
|
|
const auto * identifier = right_in_operand->as<ASTIdentifier>();
|
|
|
|
if (right_in_operand->as<ASTSubquery>() || identifier)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-10-27 18:12:40 +00:00
|
|
|
if (no_subqueries)
|
|
|
|
return {};
|
|
|
|
auto set_key = PreparedSetKey::forSubquery(*right_in_operand);
|
2019-08-15 13:54:59 +00:00
|
|
|
if (data.prepared_sets.count(set_key))
|
|
|
|
return data.prepared_sets.at(set_key);
|
2019-01-22 12:33:56 +00:00
|
|
|
|
2018-10-12 15:41:28 +00:00
|
|
|
/// A special case is if the name of the table is specified on the right side of the IN statement,
|
|
|
|
/// and the table has the type Set (a previously prepared set).
|
|
|
|
if (identifier)
|
|
|
|
{
|
2020-03-13 10:30:55 +00:00
|
|
|
auto table_id = data.context.resolveStorageID(right_in_operand);
|
2020-03-04 20:29:52 +00:00
|
|
|
StoragePtr table = DatabaseCatalog::instance().tryGetTable(table_id);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
|
|
|
if (table)
|
|
|
|
{
|
|
|
|
StorageSet * storage_set = dynamic_cast<StorageSet *>(table.get());
|
|
|
|
if (storage_set)
|
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
data.prepared_sets[set_key] = storage_set->getSet();
|
2019-01-22 12:33:56 +00:00
|
|
|
return storage_set->getSet();
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2019-01-16 12:58:27 +00:00
|
|
|
/// We get the stream of blocks for the subquery. Create Set and put it in place of the subquery.
|
2019-10-27 18:12:40 +00:00
|
|
|
String set_id = right_in_operand->getColumnName();
|
2019-01-16 12:58:27 +00:00
|
|
|
|
2019-08-15 13:54:59 +00:00
|
|
|
SubqueryForSet & subquery_for_set = data.subqueries_for_sets[set_id];
|
2018-10-12 15:41:28 +00:00
|
|
|
|
|
|
|
/// If you already created a Set with the same subquery / table.
|
|
|
|
if (subquery_for_set.set)
|
|
|
|
{
|
2019-08-15 13:54:59 +00:00
|
|
|
data.prepared_sets[set_key] = subquery_for_set.set;
|
2019-01-22 12:33:56 +00:00
|
|
|
return subquery_for_set.set;
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
|
2020-04-09 08:27:55 +00:00
|
|
|
SetPtr set = std::make_shared<Set>(data.set_size_limit, false, data.context.getSettingsRef().transform_null_in);
|
2018-10-12 15:41:28 +00:00
|
|
|
|
|
|
|
/** The following happens for GLOBAL INs:
|
|
|
|
* - in the addExternalStorage function, the IN (SELECT ...) subquery is replaced with IN _data1,
|
|
|
|
* in the subquery_for_set object, this subquery is set as source and the temporary table _data1 as the table.
|
|
|
|
* - this function shows the expression IN_data1.
|
|
|
|
*/
|
2019-08-15 13:54:59 +00:00
|
|
|
if (!subquery_for_set.source && data.no_storage_or_local)
|
2018-10-12 15:41:28 +00:00
|
|
|
{
|
2019-10-27 18:12:40 +00:00
|
|
|
auto interpreter = interpretSubquery(right_in_operand, data.context, data.subquery_depth, {});
|
2018-10-12 15:41:28 +00:00
|
|
|
subquery_for_set.source = std::make_shared<LazyBlockInputStream>(
|
|
|
|
interpreter->getSampleBlock(), [interpreter]() mutable { return interpreter->execute().in; });
|
|
|
|
|
|
|
|
/** Why is LazyBlockInputStream used?
|
|
|
|
*
|
|
|
|
* The fact is that when processing a query of the form
|
|
|
|
* SELECT ... FROM remote_test WHERE column GLOBAL IN (subquery),
|
|
|
|
* if the distributed remote_test table contains localhost as one of the servers,
|
|
|
|
* the query will be interpreted locally again (and not sent over TCP, as in the case of a remote server).
|
|
|
|
*
|
|
|
|
* The query execution pipeline will be:
|
|
|
|
* CreatingSets
|
|
|
|
* subquery execution, filling the temporary table with _data1 (1)
|
|
|
|
* CreatingSets
|
|
|
|
* reading from the table _data1, creating the set (2)
|
|
|
|
* read from the table subordinate to remote_test.
|
|
|
|
*
|
|
|
|
* (The second part of the pipeline under CreateSets is a reinterpretation of the query inside StorageDistributed,
|
|
|
|
* the query differs in that the database name and tables are replaced with subordinates, and the subquery is replaced with _data1.)
|
|
|
|
*
|
|
|
|
* But when creating the pipeline, when creating the source (2), it will be found that the _data1 table is empty
|
|
|
|
* (because the query has not started yet), and empty source will be returned as the source.
|
|
|
|
* And then, when the query is executed, an empty set will be created in step (2).
|
|
|
|
*
|
|
|
|
* Therefore, we make the initialization of step (2) lazy
|
|
|
|
* - so that it does not occur until step (1) is completed, on which the table will be populated.
|
|
|
|
*
|
|
|
|
* Note: this solution is not very good, you need to think better.
|
|
|
|
*/
|
|
|
|
}
|
|
|
|
|
|
|
|
subquery_for_set.set = set;
|
2019-08-15 13:54:59 +00:00
|
|
|
data.prepared_sets[set_key] = set;
|
2019-01-22 12:33:56 +00:00
|
|
|
return set;
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2019-10-27 18:12:40 +00:00
|
|
|
if (sample_block.has(left_in_operand->getColumnName()))
|
|
|
|
/// An explicit enumeration of values in parentheses.
|
|
|
|
return makeExplicitSet(&node, sample_block, false, data.context, data.set_size_limit, data.prepared_sets);
|
|
|
|
else
|
|
|
|
return {};
|
2018-10-12 15:41:28 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
}
|