ClickHouse/src/Interpreters/ExpressionActions.cpp

1097 lines
39 KiB
C++
Raw Normal View History

2019-10-11 17:27:54 +00:00
#include <Interpreters/Set.h>
#include <Common/ProfileEvents.h>
2020-08-13 20:17:18 +00:00
#include <Interpreters/ArrayJoinAction.h>
#include <Interpreters/ExpressionActions.h>
#include <Interpreters/TableJoin.h>
#include <Interpreters/Context.h>
2020-08-13 20:17:18 +00:00
#include <Columns/ColumnArray.h>
#include <Columns/ColumnFunction.h>
2017-07-13 20:58:19 +00:00
#include <Common/typeid_cast.h>
#include <DataTypes/DataTypeArray.h>
#include <DataTypes/DataTypesNumber.h>
#include <Functions/IFunction.h>
2020-11-10 18:22:26 +00:00
#include <IO/WriteBufferFromString.h>
2020-09-10 16:01:41 +00:00
#include <IO/Operators.h>
#include <optional>
2019-10-11 17:27:54 +00:00
#include <Columns/ColumnSet.h>
2020-09-13 13:51:31 +00:00
#include <queue>
2020-11-03 11:28:28 +00:00
#include <stack>
#include <Common/JSONBuilder.h>
2021-08-16 11:30:56 +00:00
#include <Core/SettingsEnums.h>
2021-06-22 16:21:23 +00:00
#if defined(MEMORY_SANITIZER)
#include <sanitizer/msan_interface.h>
#endif
#if defined(ADDRESS_SANITIZER)
#include <sanitizer/asan_interface.h>
#endif
namespace ProfileEvents
{
extern const Event FunctionExecute;
extern const Event CompiledFunctionExecute;
}
namespace DB
{
namespace ErrorCodes
{
2021-08-13 08:18:34 +00:00
extern const int LOGICAL_ERROR;
extern const int NOT_FOUND_COLUMN_IN_BLOCK;
extern const int TOO_MANY_TEMPORARY_COLUMNS;
extern const int TOO_MANY_TEMPORARY_NON_CONST_COLUMNS;
extern const int TYPE_MISMATCH;
}
2020-11-10 16:27:55 +00:00
ExpressionActions::~ExpressionActions() = default;
2021-08-16 11:30:56 +00:00
static std::unordered_set<const ActionsDAG::Node *> processShortCircuitFunctions(const ActionsDAG & actions_dag, ShortCircuitFunctionEvaluation short_circuit_function_evaluation);
2021-08-10 11:31:15 +00:00
2021-05-19 14:32:07 +00:00
ExpressionActions::ExpressionActions(ActionsDAGPtr actions_dag_, const ExpressionActionsSettings & settings_)
2021-03-10 10:48:08 +00:00
: settings(settings_)
2020-11-10 16:27:55 +00:00
{
actions_dag = actions_dag_->clone();
2021-08-11 15:56:03 +00:00
/// It's important to determine lazy executed nodes before compiling expressions.
2021-08-16 11:30:56 +00:00
std::unordered_set<const ActionsDAG::Node *> lazy_executed_nodes = processShortCircuitFunctions(*actions_dag, settings.short_circuit_function_evaluation);
2021-06-22 16:21:23 +00:00
2021-03-05 09:54:17 +00:00
#if USE_EMBEDDED_COMPILER
2021-05-19 14:32:07 +00:00
if (settings.can_compile_expressions && settings.compile_expressions == CompileExpressions::yes)
2021-08-10 11:31:15 +00:00
actions_dag->compileExpressions(settings.min_count_to_compile_expression, lazy_executed_nodes);
2021-03-05 09:54:17 +00:00
#endif
2020-11-10 16:27:55 +00:00
2021-08-10 11:31:15 +00:00
linearizeActions(lazy_executed_nodes);
2020-11-10 16:27:55 +00:00
if (settings.max_temporary_columns && num_columns > settings.max_temporary_columns)
throw Exception(ErrorCodes::TOO_MANY_TEMPORARY_COLUMNS,
"Too many temporary columns: {}. Maximum: {}",
actions_dag->dumpNames(), std::to_string(settings.max_temporary_columns));
}
ExpressionActionsPtr ExpressionActions::clone() const
{
2020-11-10 17:05:56 +00:00
return std::make_shared<ExpressionActions>(*this);
2020-11-10 16:27:55 +00:00
}
2021-08-10 11:31:15 +00:00
namespace
{
struct ActionsDAGReverseInfo
{
struct NodeInfo
{
std::vector<const ActionsDAG::Node *> parents;
bool used_in_result = false;
};
using ReverseIndex = std::unordered_map<const ActionsDAG::Node *, size_t>;
std::vector<NodeInfo> nodes_info;
ReverseIndex reverse_index;
};
}
static ActionsDAGReverseInfo getActionsDAGReverseInfo(const std::list<ActionsDAG::Node> & nodes, const ActionsDAG::NodeRawConstPtrs & index)
{
ActionsDAGReverseInfo result_info;
result_info.nodes_info.resize(nodes.size());
for (const auto & node : nodes)
{
size_t id = result_info.reverse_index.size();
result_info.reverse_index[&node] = id;
}
for (const auto * node : index)
result_info.nodes_info[result_info.reverse_index[node]].used_in_result = true;
for (const auto & node : nodes)
{
for (const auto & child : node.children)
result_info.nodes_info[result_info.reverse_index[child]].parents.emplace_back(&node);
}
return result_info;
}
2021-06-22 16:21:23 +00:00
static DataTypesWithConstInfo getDataTypesWithConstInfoFromNodes(const ActionsDAG::NodeRawConstPtrs & nodes)
{
DataTypesWithConstInfo types;
types.reserve(nodes.size());
for (const auto & child : nodes)
{
bool is_const = child->column && isColumnConst(*child->column);
types.push_back({child->result_type, is_const});
}
return types;
}
2021-08-11 15:56:03 +00:00
namespace
{
/// Information about the node that helps to determine if it can be executed lazily.
struct LazyExecutionInfo
{
bool can_be_lazy_executed;
/// For each node we need to know all it's ancestors that are short-circuit functions.
/// Also we need to know which arguments of this short-circuit functions are ancestors for the node
/// (we will store the set of indexes of arguments), because for some short-circuit function we shouldn't
/// enable lazy execution for nodes that are common descendants of different function arguments.
/// Example: if(cond, expr1(..., expr, ...), expr2(..., expr, ...))).
std::unordered_map<const ActionsDAG::Node *, std::unordered_set<size_t>> short_circuit_ancestors_info;
};
}
/// Create lazy execution info for node.
static void setLazyExecutionInfo(
const ActionsDAG::Node * node,
const ActionsDAGReverseInfo & reverse_info,
const std::unordered_map<const ActionsDAG::Node *, IFunctionBase::ShortCircuitSettings> & short_circuit_nodes,
std::unordered_map<const ActionsDAG::Node *, LazyExecutionInfo> & lazy_execution_infos)
{
/// If we already created info about this node, just do nothing.
if (lazy_execution_infos.contains(node))
return;
LazyExecutionInfo & lazy_execution_info = lazy_execution_infos[node];
lazy_execution_info.can_be_lazy_executed = true;
2021-08-13 08:18:34 +00:00
const ActionsDAGReverseInfo::NodeInfo & node_info = reverse_info.nodes_info[reverse_info.reverse_index.at(node)];
2021-08-11 15:56:03 +00:00
/// If node is used in result, we can't enable lazy execution.
if (node_info.used_in_result)
lazy_execution_info.can_be_lazy_executed = false;
/// To fill lazy execution info for current node we need to create it for all it's parents.
for (const auto & parent : node_info.parents)
{
setLazyExecutionInfo(parent, reverse_info, short_circuit_nodes, lazy_execution_infos);
/// Update current node info according to parent info.
if (short_circuit_nodes.contains(parent))
{
/// Use set, because one node can be more than one argument.
/// Example: expr1 AND expr2 AND expr1.
std::set<size_t> indexes;
for (size_t i = 0; i != parent->children.size(); ++i)
{
if (node == parent->children[i])
indexes.insert(i);
}
if (!short_circuit_nodes.at(parent).enable_lazy_execution_for_first_argument && node == parent->children[0])
{
/// We shouldn't add 0 index in node info in this case.
indexes.erase(0);
/// Disable lazy execution for current node only if it's disabled for short-circuit node,
/// because we can have nested short-circuit nodes.
if (!lazy_execution_infos[parent].can_be_lazy_executed)
lazy_execution_info.can_be_lazy_executed = false;
}
lazy_execution_info.short_circuit_ancestors_info[parent].insert(indexes.begin(), indexes.end());
}
else
/// If lazy execution is disabled for one of parents, we should disable it for current node.
lazy_execution_info.can_be_lazy_executed &= lazy_execution_infos[parent].can_be_lazy_executed;
/// Update info about short-circuit ancestors according to parent info.
for (const auto & [short_circuit_node, indexes] : lazy_execution_infos[parent].short_circuit_ancestors_info)
lazy_execution_info.short_circuit_ancestors_info[short_circuit_node].insert(indexes.begin(), indexes.end());
}
if (!lazy_execution_info.can_be_lazy_executed)
return;
/// Check if current node is common descendant of different function arguments of
/// short-circuit function that disables lazy execution on this case.
for (const auto & [short_circuit_node, indexes] : lazy_execution_info.short_circuit_ancestors_info)
{
/// If lazy execution is enabled for this short-circuit node,
/// we shouldn't disable it for current node.
if (lazy_execution_infos[short_circuit_node].can_be_lazy_executed)
continue;
if (!short_circuit_nodes.at(short_circuit_node).enable_lazy_execution_for_common_descendants_of_arguments && indexes.size() > 1)
{
lazy_execution_info.can_be_lazy_executed = false;
return;
}
}
}
2021-08-10 11:31:15 +00:00
/// Enable lazy execution for short-circuit function arguments.
static bool findLazyExecutedNodes(
const ActionsDAG::NodeRawConstPtrs & children,
2021-08-16 11:30:56 +00:00
std::unordered_map<const ActionsDAG::Node *, LazyExecutionInfo> & lazy_execution_infos,
2021-08-10 11:31:15 +00:00
bool force_enable_lazy_execution,
std::unordered_set<const ActionsDAG::Node *> & lazy_executed_nodes_out)
{
2021-06-22 16:21:23 +00:00
bool has_lazy_node = false;
for (const auto * child : children)
{
2021-08-11 15:56:03 +00:00
/// Skip node that have already been found as lazy executed.
2021-08-10 11:31:15 +00:00
if (lazy_executed_nodes_out.contains(child))
{
has_lazy_node = true;
continue;
}
2021-08-11 15:56:03 +00:00
/// Skip nodes that cannot be lazy executed.
if (!lazy_execution_infos[child].can_be_lazy_executed)
continue;
2021-04-27 22:31:49 +00:00
/// We cannot propagate lazy execution through arrayJoin, because when we execute
/// arrayJoin we need to know the exact offset of it's argument to replicate the other arguments.
/// We cannot determine the exact offset without it's argument execution, because the offset
/// can depend on on it.
/// Example: arrayJoin(range(number)), we use lazy execution for masked function execution,
/// but if we filter column number by mask and then execute function range() and arrayJoin, we will get
/// the offset that is differ from what we would get without filtering.
switch (child->type)
{
case ActionsDAG::ActionType::FUNCTION:
2021-06-22 16:21:23 +00:00
{
/// Propagate lazy execution through function arguments.
2021-08-11 15:56:03 +00:00
bool has_lazy_child = findLazyExecutedNodes(child->children, lazy_execution_infos, force_enable_lazy_execution, lazy_executed_nodes_out);
2021-06-22 16:21:23 +00:00
/// Use lazy execution when:
/// - It's force enabled.
/// - Function is suitable for lazy execution.
/// - Function has lazy executed arguments.
if (force_enable_lazy_execution || has_lazy_child || child->function_base->isSuitableForShortCircuitArgumentsExecution(getDataTypesWithConstInfoFromNodes(child->children)))
{
has_lazy_node = true;
2021-08-10 11:31:15 +00:00
lazy_executed_nodes_out.insert(child);
2021-06-22 16:21:23 +00:00
}
break;
2021-06-22 16:21:23 +00:00
}
case ActionsDAG::ActionType::ALIAS:
/// Propagate lazy execution through alias.
2021-08-11 15:56:03 +00:00
has_lazy_node |= findLazyExecutedNodes(child->children, lazy_execution_infos, force_enable_lazy_execution, lazy_executed_nodes_out);
break;
default:
break;
}
}
2021-06-22 16:21:23 +00:00
return has_lazy_node;
}
2021-08-16 11:30:56 +00:00
static std::unordered_set<const ActionsDAG::Node *> processShortCircuitFunctions(const ActionsDAG & actions_dag, ShortCircuitFunctionEvaluation short_circuit_function_evaluation)
{
2021-08-16 11:30:56 +00:00
if (short_circuit_function_evaluation == ShortCircuitFunctionEvaluation::DISABLE)
return {};
2021-08-10 11:31:15 +00:00
const auto & nodes = actions_dag.getNodes();
2021-06-22 16:21:23 +00:00
2021-08-11 15:56:03 +00:00
/// Firstly, find all short-circuit functions and get their settings.
std::unordered_map<const ActionsDAG::Node *, IFunctionBase::ShortCircuitSettings> short_circuit_nodes;
IFunctionBase::ShortCircuitSettings short_circuit_settings;
for (const auto & node : nodes)
{
2021-06-22 16:21:23 +00:00
if (node.type == ActionsDAG::ActionType::FUNCTION && node.function_base->isShortCircuit(short_circuit_settings, node.children.size()) && !node.children.empty())
2021-08-11 15:56:03 +00:00
short_circuit_nodes[&node] = short_circuit_settings;
}
2021-08-11 15:56:03 +00:00
auto reverse_info = getActionsDAGReverseInfo(nodes, actions_dag.getIndex());
2021-08-11 15:56:03 +00:00
/// For each node we fill LazyExecutionInfo.
std::unordered_map<const ActionsDAG::Node *, LazyExecutionInfo> lazy_execution_infos;
for (const auto & node : nodes)
setLazyExecutionInfo(&node, reverse_info, short_circuit_nodes, lazy_execution_infos);
2020-11-10 16:27:55 +00:00
2021-08-10 11:31:15 +00:00
std::unordered_set<const ActionsDAG::Node *> lazy_executed_nodes;
2021-08-12 08:31:28 +00:00
for (const auto & [node, settings] : short_circuit_nodes)
2020-11-10 16:27:55 +00:00
{
2021-08-11 15:56:03 +00:00
/// Recursively find nodes that should be lazy executed.
findLazyExecutedNodes(
node->children,
lazy_execution_infos,
2021-08-16 11:30:56 +00:00
settings.force_enable_lazy_execution || short_circuit_function_evaluation == ShortCircuitFunctionEvaluation::FORCE_ENABLE,
lazy_executed_nodes);
2020-11-10 16:27:55 +00:00
}
2021-08-10 11:31:15 +00:00
return lazy_executed_nodes;
2021-06-22 16:21:23 +00:00
}
2021-08-10 11:31:15 +00:00
void ExpressionActions::linearizeActions(const std::unordered_set<const ActionsDAG::Node *> & lazy_executed_nodes)
2021-06-22 16:21:23 +00:00
{
2021-08-13 08:18:34 +00:00
/// This function does the topological sort on DAG and fills all the fields of ExpressionActions.
/// Algorithm traverses DAG starting from nodes without children.
/// For every node we support the number of created children, and if all children are created, put node into queue.
2021-08-10 11:31:15 +00:00
struct Data
{
const Node * node = nullptr;
size_t num_created_children = 0;
ssize_t position = -1;
size_t num_created_parents = 0;
};
2021-06-22 16:21:23 +00:00
const auto & nodes = getNodes();
const auto & index = actions_dag->getIndex();
const auto & inputs = actions_dag->getInputs();
2021-08-10 11:31:15 +00:00
auto reverse_info = getActionsDAGReverseInfo(nodes, index);
std::vector<Data> data;
for (const auto & node : nodes)
data.push_back({.node = &node});
2021-06-22 16:21:23 +00:00
/// There are independent queues for arrayJoin and other actions.
/// We delay creation of arrayJoin as long as we can, so that they will be executed closer to end.
std::queue<const Node *> ready_nodes;
std::queue<const Node *> ready_array_joins;
2020-11-10 16:27:55 +00:00
for (const auto & node : nodes)
{
if (node.children.empty())
ready_nodes.emplace(&node);
}
2020-11-11 14:56:56 +00:00
/// Every argument will have fixed position in columns list.
/// If argument is removed, it's position may be reused by other action.
2020-11-10 16:27:55 +00:00
std::stack<size_t> free_positions;
while (!ready_nodes.empty() || !ready_array_joins.empty())
{
auto & stack = ready_nodes.empty() ? ready_array_joins : ready_nodes;
const Node * node = stack.front();
stack.pop();
2021-08-10 11:31:15 +00:00
auto cur_index = reverse_info.reverse_index[node];
auto & cur = data[cur_index];
auto & cur_info = reverse_info.nodes_info[cur_index];
2020-11-10 16:27:55 +00:00
2020-11-11 14:56:56 +00:00
/// Select position for action result.
2020-11-10 16:27:55 +00:00
size_t free_position = num_columns;
if (free_positions.empty())
++num_columns;
else
{
free_position = free_positions.top();
free_positions.pop();
}
cur.position = free_position;
ExpressionActions::Arguments arguments;
arguments.reserve(cur.node->children.size());
2021-03-04 17:38:12 +00:00
for (const auto * child : cur.node->children)
2020-11-10 16:27:55 +00:00
{
2021-08-10 11:31:15 +00:00
auto arg_index = reverse_info.reverse_index[child];
auto & arg = data[arg_index];
auto arg_info = reverse_info.nodes_info[arg_index];
2020-11-10 16:27:55 +00:00
if (arg.position < 0)
throw Exception(ErrorCodes::LOGICAL_ERROR, "Argument was not calculated for {}", child->result_name);
++arg.num_created_parents;
ExpressionActions::Argument argument;
argument.pos = arg.position;
2021-08-10 11:31:15 +00:00
argument.needed_later = arg_info.used_in_result || arg.num_created_parents != arg_info.parents.size();
2020-11-10 16:27:55 +00:00
if (!argument.needed_later)
free_positions.push(argument.pos);
arguments.emplace_back(argument);
}
if (node->type == ActionsDAG::ActionType::INPUT)
{
/// Argument for input is special. It contains the position from required columns.
ExpressionActions::Argument argument;
2021-08-10 11:31:15 +00:00
argument.needed_later = !cur_info.parents.empty();
2020-11-10 16:27:55 +00:00
arguments.emplace_back(argument);
2020-11-17 12:34:31 +00:00
//required_columns.push_back({node->result_name, node->result_type});
2020-11-10 16:27:55 +00:00
}
2021-08-12 08:31:28 +00:00
actions.push_back({node, arguments, free_position, lazy_executed_nodes.contains(node)});
2020-11-10 16:27:55 +00:00
2021-08-10 11:31:15 +00:00
for (const auto & parent : cur_info.parents)
2020-11-10 16:27:55 +00:00
{
2021-08-10 11:31:15 +00:00
auto & parent_data = data[reverse_info.reverse_index[parent]];
2020-11-10 16:27:55 +00:00
++parent_data.num_created_children;
if (parent_data.num_created_children == parent->children.size())
{
auto & push_stack = parent->type == ActionsDAG::ActionType::ARRAY_JOIN ? ready_array_joins : ready_nodes;
push_stack.push(parent);
}
}
}
result_positions.reserve(index.size());
for (const auto & node : index)
{
2021-08-10 11:31:15 +00:00
auto pos = data[reverse_info.reverse_index[node]].position;
2020-11-10 16:27:55 +00:00
if (pos < 0)
throw Exception(ErrorCodes::LOGICAL_ERROR, "Action for {} was not calculated", node->result_name);
result_positions.push_back(pos);
ColumnWithTypeAndName col{node->column, node->result_type, node->result_name};
sample_block.insert(std::move(col));
}
2020-11-17 12:34:31 +00:00
for (const auto * input : inputs)
{
2021-08-10 11:31:15 +00:00
const auto & cur = data[reverse_info.reverse_index[input]];
2020-11-17 12:34:31 +00:00
auto pos = required_columns.size();
actions[cur.position].arguments.front().pos = pos;
required_columns.push_back({input->result_name, input->result_type});
2021-03-04 17:38:12 +00:00
input_positions[input->result_name].emplace_back(pos);
2020-11-17 12:34:31 +00:00
}
2020-11-10 16:27:55 +00:00
}
static WriteBuffer & operator << (WriteBuffer & out, const ExpressionActions::Argument & argument)
{
2020-11-09 15:01:08 +00:00
return out << (argument.needed_later ? ": " : ":: ") << argument.pos;
}
2020-11-03 11:28:28 +00:00
std::string ExpressionActions::Action::toString() const
{
WriteBufferFromOwnString out;
2020-11-03 11:28:28 +00:00
switch (node->type)
{
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::COLUMN:
2020-11-03 11:28:28 +00:00
out << "COLUMN "
<< (node->column ? node->column->getName() : "(no column)");
break;
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::ALIAS:
2020-11-03 11:28:28 +00:00
out << "ALIAS " << node->children.front()->result_name << " " << arguments.front();
break;
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::FUNCTION:
2020-11-03 11:28:28 +00:00
out << "FUNCTION " << (node->is_function_compiled ? "[compiled] " : "")
<< (node->function_base ? node->function_base->getName() : "(no function)") << "(";
for (size_t i = 0; i < node->children.size(); ++i)
{
if (i)
2020-11-03 11:28:28 +00:00
out << ", ";
out << node->children[i]->result_name << " " << arguments[i];
}
2020-11-03 11:28:28 +00:00
out << ")";
break;
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::ARRAY_JOIN:
2020-11-03 11:28:28 +00:00
out << "ARRAY JOIN " << node->children.front()->result_name << " " << arguments.front();
break;
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::INPUT:
2020-11-03 11:28:28 +00:00
out << "INPUT " << arguments.front();
break;
}
2020-11-03 11:28:28 +00:00
out << " -> " << node->result_name
2020-11-09 15:01:08 +00:00
<< " " << (node->result_type ? node->result_type->getName() : "(no type)") << " : " << result_position;
2020-11-03 11:28:28 +00:00
return out.str();
}
JSONBuilder::ItemPtr ExpressionActions::Action::toTree() const
2021-04-09 16:18:45 +00:00
{
auto map = std::make_unique<JSONBuilder::JSONMap>();
2021-04-09 16:18:45 +00:00
if (node)
node->toTree(*map);
2021-04-09 16:18:45 +00:00
auto args = std::make_unique<JSONBuilder::JSONArray>();
auto dropped_args = std::make_unique<JSONBuilder::JSONArray>();
2021-04-09 16:18:45 +00:00
for (auto arg : arguments)
{
args->add(arg.pos);
2021-04-09 16:18:45 +00:00
if (!arg.needed_later)
dropped_args->add(arg.pos);
2021-04-09 16:18:45 +00:00
}
map->add("Arguments", std::move(args));
map->add("Removed Arguments", std::move(dropped_args));
map->add("Result", result_position);
2021-04-09 16:18:45 +00:00
return map;
2021-04-09 16:18:45 +00:00
}
2020-11-10 20:36:38 +00:00
void ExpressionActions::checkLimits(const ColumnsWithTypeAndName & columns) const
{
2021-03-10 10:48:08 +00:00
if (settings.max_temporary_non_const_columns)
{
size_t non_const_columns = 0;
2020-11-10 20:36:38 +00:00
for (const auto & column : columns)
2020-11-03 11:28:28 +00:00
if (column.column && !isColumnConst(*column.column))
++non_const_columns;
2021-03-10 10:48:08 +00:00
if (non_const_columns > settings.max_temporary_non_const_columns)
{
WriteBufferFromOwnString list_of_non_const_columns;
2020-11-10 20:36:38 +00:00
for (const auto & column : columns)
2020-11-03 11:28:28 +00:00
if (column.column && !isColumnConst(*column.column))
list_of_non_const_columns << "\n" << column.name;
throw Exception("Too many temporary non-const columns:" + list_of_non_const_columns.str()
2021-08-13 08:18:34 +00:00
+ ". Maximum: " + std::to_string(settings.max_temporary_non_const_columns),
ErrorCodes::TOO_MANY_TEMPORARY_NON_CONST_COLUMNS);
}
}
}
2020-11-10 20:36:38 +00:00
namespace
{
2020-11-11 14:26:18 +00:00
/// This struct stores context needed to execute actions.
///
/// Execution model is following:
/// * execution is performed over list of columns (with fixed size = ExpressionActions::num_columns)
/// * every argument has fixed position in columns list, every action has fixed position for result
/// * if argument is not needed anymore (Argument::needed_later == false), it is removed from list
/// * argument for INPUT is in inputs[inputs_pos[argument.pos]]
///
/// Columns on positions `ExpressionActions::result_positions` are inserted back into block.
2020-11-10 20:36:38 +00:00
struct ExecutionContext
2020-10-13 08:16:47 +00:00
{
2020-11-10 20:36:38 +00:00
ColumnsWithTypeAndName & inputs;
ColumnsWithTypeAndName columns = {};
std::vector<ssize_t> inputs_pos = {};
2021-04-27 13:12:57 +00:00
size_t num_rows = 0;
2020-10-13 08:16:47 +00:00
};
2013-05-28 12:05:47 +00:00
}
static void executeAction(const ExpressionActions::Action & action, ExecutionContext & execution_context, bool dry_run)
2020-10-07 18:37:27 +00:00
{
2020-11-03 11:28:28 +00:00
auto & inputs = execution_context.inputs;
2020-10-13 08:16:47 +00:00
auto & columns = execution_context.columns;
auto & num_rows = execution_context.num_rows;
2020-10-07 18:37:27 +00:00
switch (action.node->type)
{
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::FUNCTION:
2020-10-07 18:37:27 +00:00
{
auto & res_column = columns[action.result_position];
if (res_column.type || res_column.column)
throw Exception("Result column is not empty", ErrorCodes::LOGICAL_ERROR);
res_column.type = action.node->result_type;
res_column.name = action.node->result_name;
2021-08-09 12:19:28 +00:00
if (action.node->column)
{
2021-08-09 13:47:30 +00:00
/// Do not execute function if it's result is already known.
2021-08-09 12:19:28 +00:00
res_column.column = action.node->column->cloneResized(num_rows);
break;
}
ColumnsWithTypeAndName arguments(action.arguments.size());
for (size_t i = 0; i < arguments.size(); ++i)
{
if (!action.arguments[i].needed_later)
arguments[i] = std::move(columns[action.arguments[i].pos]);
else
arguments[i] = columns[action.arguments[i].pos];
}
2021-08-12 08:31:28 +00:00
if (action.is_lazy_executed)
2021-06-22 16:21:23 +00:00
res_column.column = ColumnFunction::create(num_rows, action.node->function_base, std::move(arguments), true, action.node->is_function_compiled);
2021-04-23 11:42:13 +00:00
else
{
ProfileEvents::increment(ProfileEvents::FunctionExecute);
if (action.node->is_function_compiled)
ProfileEvents::increment(ProfileEvents::CompiledFunctionExecute);
2021-04-23 11:42:13 +00:00
res_column.column = action.node->function->execute(arguments, res_column.type, num_rows, dry_run);
}
break;
}
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::ARRAY_JOIN:
2020-10-07 18:37:27 +00:00
{
2020-11-03 11:28:28 +00:00
size_t array_join_key_pos = action.arguments.front().pos;
2020-10-13 08:16:47 +00:00
auto array_join_key = columns[array_join_key_pos];
2020-10-07 18:37:27 +00:00
2020-10-13 08:16:47 +00:00
/// Remove array join argument in advance if it is not needed.
2020-11-09 15:01:08 +00:00
if (!action.arguments.front().needed_later)
2020-10-13 08:16:47 +00:00
columns[array_join_key_pos] = {};
2020-10-07 18:37:27 +00:00
2020-10-13 08:16:47 +00:00
array_join_key.column = array_join_key.column->convertToFullColumnIfConst();
2020-10-07 18:37:27 +00:00
2020-10-13 08:16:47 +00:00
const ColumnArray * array = typeid_cast<const ColumnArray *>(array_join_key.column.get());
if (!array)
throw Exception("ARRAY JOIN of not array: " + action.node->result_name, ErrorCodes::TYPE_MISMATCH);
2020-10-07 18:37:27 +00:00
2020-10-13 08:16:47 +00:00
for (auto & column : columns)
if (column.column)
column.column = column.column->replicate(array->getOffsets());
2020-10-07 18:37:27 +00:00
2020-11-03 11:28:28 +00:00
for (auto & column : inputs)
2020-10-13 08:16:47 +00:00
if (column.column)
column.column = column.column->replicate(array->getOffsets());
2020-10-07 18:37:27 +00:00
2020-10-13 08:16:47 +00:00
auto & res_column = columns[action.result_position];
2020-10-07 18:37:27 +00:00
2020-10-13 08:16:47 +00:00
res_column.column = array->getDataPtr();
res_column.type = assert_cast<const DataTypeArray &>(*array_join_key.type).getNestedType();
2020-11-03 11:28:28 +00:00
res_column.name = action.node->result_name;
2020-10-07 18:37:27 +00:00
2020-10-13 08:16:47 +00:00
num_rows = res_column.column->size();
2020-10-07 18:37:27 +00:00
break;
}
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::COLUMN:
2020-10-07 18:37:27 +00:00
{
2020-10-13 08:16:47 +00:00
auto & res_column = columns[action.result_position];
res_column.column = action.node->column->cloneResized(num_rows);
res_column.type = action.node->result_type;
2020-11-03 11:28:28 +00:00
res_column.name = action.node->result_name;
2020-10-07 18:37:27 +00:00
break;
}
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::ALIAS:
2020-10-13 08:16:47 +00:00
{
2020-11-03 11:28:28 +00:00
const auto & arg = action.arguments.front();
if (action.result_position != arg.pos)
{
columns[action.result_position].column = columns[arg.pos].column;
columns[action.result_position].type = columns[arg.pos].type;
2020-11-03 11:28:28 +00:00
2020-11-09 15:01:08 +00:00
if (!arg.needed_later)
2020-11-03 11:28:28 +00:00
columns[arg.pos] = {};
}
columns[action.result_position].name = action.node->result_name;
2020-10-07 18:37:27 +00:00
break;
2020-10-13 08:16:47 +00:00
}
2020-10-07 18:37:27 +00:00
2020-11-10 14:54:59 +00:00
case ActionsDAG::ActionType::INPUT:
2020-10-13 08:16:47 +00:00
{
2020-11-03 11:28:28 +00:00
auto pos = execution_context.inputs_pos[action.arguments.front().pos];
if (pos < 0)
{
2020-11-11 14:56:56 +00:00
/// Here we allow to skip input if it is not in block (in case it is not needed).
/// It may be unusual, but some code depend on such behaviour.
2020-11-09 15:01:08 +00:00
if (action.arguments.front().needed_later)
2020-11-03 11:28:28 +00:00
throw Exception(ErrorCodes::NOT_FOUND_COLUMN_IN_BLOCK,
"Not found column {} in block",
action.node->result_name);
}
else
columns[action.result_position] = std::move(inputs[pos]);
2020-11-03 11:28:28 +00:00
break;
2020-10-13 08:16:47 +00:00
}
2020-10-07 18:37:27 +00:00
}
2020-11-03 11:28:28 +00:00
}
2020-10-13 08:16:47 +00:00
2020-11-10 20:36:38 +00:00
void ExpressionActions::execute(Block & block, size_t & num_rows, bool dry_run) const
{
ExecutionContext execution_context
2021-08-13 08:18:34 +00:00
{
.inputs = block.data,
.num_rows = num_rows,
};
2020-11-10 20:36:38 +00:00
2020-11-17 12:34:31 +00:00
execution_context.inputs_pos.assign(required_columns.size(), -1);
for (size_t pos = 0; pos < block.columns(); ++pos)
{
const auto & col = block.getByPosition(pos);
auto it = input_positions.find(col.name);
if (it != input_positions.end())
{
for (auto input_pos : it->second)
{
if (execution_context.inputs_pos[input_pos] < 0)
{
execution_context.inputs_pos[input_pos] = pos;
break;
}
}
}
}
2020-11-10 20:36:38 +00:00
execution_context.columns.resize(num_columns);
for (const auto & action : actions)
{
try
{
executeAction(action, execution_context, dry_run);
2020-11-10 20:36:38 +00:00
checkLimits(execution_context.columns);
//std::cerr << "Action: " << action.toString() << std::endl;
//for (const auto & col : execution_context.columns)
// std::cerr << col.dumpStructure() << std::endl;
}
catch (Exception & e)
{
e.addMessage(fmt::format("while executing '{}'", action.toString()));
throw;
}
}
if (actions_dag->isInputProjected())
2020-11-10 20:36:38 +00:00
{
block.clear();
}
else
{
std::sort(execution_context.inputs_pos.rbegin(), execution_context.inputs_pos.rend());
for (auto input : execution_context.inputs_pos)
if (input >= 0)
block.erase(input);
}
2021-01-18 21:54:01 +00:00
Block res;
2020-11-10 20:36:38 +00:00
for (auto pos : result_positions)
if (execution_context.columns[pos].column)
2021-01-18 21:54:01 +00:00
res.insert(execution_context.columns[pos]);
for (const auto & item : block)
res.insert(std::move(item));
block.swap(res);
2020-11-10 20:36:38 +00:00
num_rows = execution_context.num_rows;
}
void ExpressionActions::execute(Block & block, bool dry_run) const
{
size_t num_rows = block.rows();
execute(block, num_rows, dry_run);
if (!block)
block.insert({DataTypeUInt8().createColumnConst(num_rows, 0), std::make_shared<DataTypeUInt8>(), "_dummy"});
}
2020-11-03 11:28:28 +00:00
Names ExpressionActions::getRequiredColumns() const
{
Names names;
for (const auto & input : required_columns)
names.push_back(input.name);
return names;
2020-10-07 18:37:27 +00:00
}
2020-09-08 10:40:53 +00:00
bool ExpressionActions::hasArrayJoin() const
2020-06-18 13:00:16 +00:00
{
return getActionsDAG().hasArrayJoin();
}
void ExpressionActions::assertDeterministic() const
{
getActionsDAG().assertDeterministic();
2020-06-18 13:00:16 +00:00
}
std::string ExpressionActions::getSmallestColumn(const NamesAndTypesList & columns)
{
std::optional<size_t> min_size;
String res;
for (const auto & column : columns)
{
/// Skip .sizeX and similar meta information
if (column.isSubcolumn())
continue;
/// @todo resolve evil constant
size_t size = column.type->haveMaximumSizeOfValue() ? column.type->getMaximumSizeOfValueInMemory() : 100;
if (!min_size || size < *min_size)
{
min_size = size;
res = column.name;
}
}
if (!min_size)
throw Exception("No available columns", ErrorCodes::LOGICAL_ERROR);
return res;
}
std::string ExpressionActions::dumpActions() const
{
2020-11-10 18:22:26 +00:00
WriteBufferFromOwnString ss;
ss << "input:\n";
2020-11-03 11:28:28 +00:00
for (const auto & input_column : required_columns)
2020-03-08 23:48:08 +00:00
ss << input_column.name << " " << input_column.type->getName() << "\n";
ss << "\nactions:\n";
2020-03-08 23:48:08 +00:00
for (const auto & action : actions)
ss << action.toString() << '\n';
ss << "\noutput:\n";
NamesAndTypesList output_columns = sample_block.getNamesAndTypesList();
2020-03-08 23:48:08 +00:00
for (const auto & output_column : output_columns)
ss << output_column.name << " " << output_column.type->getName() << "\n";
ss << "\nproject input: " << actions_dag->isInputProjected() << "\noutput positions:";
2020-11-03 11:28:28 +00:00
for (auto pos : result_positions)
ss << " " << pos;
ss << "\n";
return ss.str();
}
JSONBuilder::ItemPtr ExpressionActions::toTree() const
2021-04-09 16:18:45 +00:00
{
auto inputs_array = std::make_unique<JSONBuilder::JSONArray>();
2021-04-09 16:18:45 +00:00
for (const auto & input_column : required_columns)
{
auto map = std::make_unique<JSONBuilder::JSONMap>();
map->add("Name", input_column.name);
2021-04-09 16:18:45 +00:00
if (input_column.type)
map->add("Type", input_column.type->getName());
2021-04-09 16:18:45 +00:00
inputs_array->add(std::move(map));
2021-04-09 16:18:45 +00:00
}
auto outputs_array = std::make_unique<JSONBuilder::JSONArray>();
2021-04-09 16:18:45 +00:00
for (const auto & output_column : sample_block)
{
auto map = std::make_unique<JSONBuilder::JSONMap>();
map->add("Name", output_column.name);
2021-04-09 16:18:45 +00:00
if (output_column.type)
map->add("Type", output_column.type->getName());
2021-04-09 16:18:45 +00:00
outputs_array->add(std::move(map));
2021-04-09 16:18:45 +00:00
}
auto actions_array = std::make_unique<JSONBuilder::JSONArray>();
2021-04-09 16:18:45 +00:00
for (const auto & action : actions)
actions_array->add(action.toTree());
2021-04-09 16:18:45 +00:00
auto positions_array = std::make_unique<JSONBuilder::JSONArray>();
2021-04-09 16:18:45 +00:00
for (auto pos : result_positions)
positions_array->add(pos);
2021-04-09 16:18:45 +00:00
auto map = std::make_unique<JSONBuilder::JSONMap>();
map->add("Inputs", std::move(inputs_array));
map->add("Actions", std::move(actions_array));
map->add("Outputs", std::move(outputs_array));
map->add("Positions", std::move(positions_array));
map->add("Project Input", actions_dag->isInputProjected());
2021-04-09 16:18:45 +00:00
return map;
2021-04-09 16:18:45 +00:00
}
2019-10-11 17:27:54 +00:00
bool ExpressionActions::checkColumnIsAlwaysFalse(const String & column_name) const
{
/// Check has column in (empty set).
String set_to_check;
2019-11-01 10:58:29 +00:00
for (auto it = actions.rbegin(); it != actions.rend(); ++it)
2019-10-11 17:27:54 +00:00
{
2020-04-22 06:01:33 +00:00
const auto & action = *it;
2020-11-10 14:54:59 +00:00
if (action.node->type == ActionsDAG::ActionType::FUNCTION && action.node->function_base)
2019-10-11 17:27:54 +00:00
{
2020-11-03 11:28:28 +00:00
if (action.node->result_name == column_name && action.node->children.size() > 1)
2019-10-11 17:55:33 +00:00
{
2020-11-03 11:28:28 +00:00
auto name = action.node->function_base->getName();
if ((name == "in" || name == "globalIn"))
{
set_to_check = action.node->children[1]->result_name;
break;
}
2019-10-11 17:55:33 +00:00
}
2019-10-11 17:27:54 +00:00
}
}
if (!set_to_check.empty())
{
2020-04-22 06:01:33 +00:00
for (const auto & action : actions)
2019-10-11 17:27:54 +00:00
{
2020-11-10 14:54:59 +00:00
if (action.node->type == ActionsDAG::ActionType::COLUMN && action.node->result_name == set_to_check)
2019-10-11 17:27:54 +00:00
{
2019-10-27 18:12:40 +00:00
// Constant ColumnSet cannot be empty, so we only need to check non-constant ones.
2020-11-03 11:28:28 +00:00
if (const auto * column_set = checkAndGetColumn<const ColumnSet>(action.node->column.get()))
2019-10-11 17:27:54 +00:00
{
2019-11-01 13:56:33 +00:00
if (column_set->getData()->isCreated() && column_set->getData()->getTotalRowCount() == 0)
2019-10-11 17:27:54 +00:00
return true;
}
}
}
}
return false;
}
2020-11-03 11:28:28 +00:00
void ExpressionActionsChain::addStep(NameSet non_constant_inputs)
{
if (steps.empty())
throw Exception("Cannot add action to empty ExpressionActionsChain", ErrorCodes::LOGICAL_ERROR);
2020-08-19 19:33:49 +00:00
ColumnsWithTypeAndName columns = steps.back()->getResultColumns();
2020-11-03 11:28:28 +00:00
for (auto & column : columns)
if (column.column && isColumnConst(*column.column) && non_constant_inputs.count(column.name))
column.column = nullptr;
2020-09-11 12:24:41 +00:00
steps.push_back(std::make_unique<ExpressionActionsStep>(std::make_shared<ActionsDAG>(columns)));
}
void ExpressionActionsChain::finalize()
{
2017-04-02 17:37:49 +00:00
/// Finalize all steps. Right to left to define unnecessary input columns.
for (int i = static_cast<int>(steps.size()) - 1; i >= 0; --i)
{
2021-03-05 13:56:44 +00:00
auto & required_output = steps[i]->required_output;
2021-03-10 08:41:24 +00:00
NameSet required_names;
2021-03-05 13:56:44 +00:00
for (const auto & output : required_output)
2021-03-10 08:41:24 +00:00
required_names.insert(output.first);
2018-04-12 09:45:24 +00:00
if (i + 1 < static_cast<int>(steps.size()))
{
2020-08-19 19:33:49 +00:00
const NameSet & additional_input = steps[i + 1]->additional_input;
for (const auto & it : steps[i + 1]->getRequiredColumns())
2018-04-12 09:45:24 +00:00
{
if (additional_input.count(it.name) == 0)
{
2021-03-05 13:56:44 +00:00
auto iter = required_output.find(it.name);
if (iter == required_output.end())
2021-03-10 08:41:24 +00:00
required_names.insert(it.name);
2021-03-05 13:56:44 +00:00
else
iter->second = false;
2018-04-12 09:45:24 +00:00
}
}
}
2021-03-05 13:56:44 +00:00
steps[i]->finalize(required_names);
}
2017-04-02 17:37:49 +00:00
/// Adding the ejection of unnecessary columns to the beginning of each step.
for (size_t i = 1; i < steps.size(); ++i)
{
2020-08-19 19:33:49 +00:00
size_t columns_from_previous = steps[i - 1]->getResultColumns().size();
2017-04-02 17:37:49 +00:00
/// If unnecessary columns are formed at the output of the previous step, we'll add them to the beginning of this step.
/// Except when we drop all the columns and lose the number of rows in the block.
2020-08-19 19:33:49 +00:00
if (!steps[i]->getResultColumns().empty()
&& columns_from_previous > steps[i]->getRequiredColumns().size())
steps[i]->prependProjectInput();
}
}
2020-07-26 14:21:57 +00:00
std::string ExpressionActionsChain::dumpChain() const
{
2020-11-10 18:22:26 +00:00
WriteBufferFromOwnString ss;
for (size_t i = 0; i < steps.size(); ++i)
{
ss << "step " << i << "\n";
ss << "required output:\n";
2021-03-05 13:56:44 +00:00
for (const auto & it : steps[i]->required_output)
ss << it.first << "\n";
2020-08-19 19:33:49 +00:00
ss << "\n" << steps[i]->dump() << "\n";
}
return ss.str();
}
2020-09-08 10:40:53 +00:00
ExpressionActionsChain::ArrayJoinStep::ArrayJoinStep(ArrayJoinActionPtr array_join_, ColumnsWithTypeAndName required_columns_)
: Step({})
2020-08-19 19:33:49 +00:00
, array_join(std::move(array_join_))
2020-08-13 20:17:18 +00:00
, result_columns(std::move(required_columns_))
2020-08-12 08:55:16 +00:00
{
2020-08-13 20:17:18 +00:00
for (auto & column : result_columns)
2020-08-12 08:55:16 +00:00
{
required_columns.emplace_back(NameAndTypePair(column.name, column.type));
if (array_join->columns.count(column.name) > 0)
{
const auto * array = typeid_cast<const DataTypeArray *>(column.type.get());
column.type = array->getNestedType();
/// Arrays are materialized
column.column = nullptr;
}
}
}
2021-03-10 08:41:24 +00:00
void ExpressionActionsChain::ArrayJoinStep::finalize(const NameSet & required_output_)
2020-08-12 08:55:16 +00:00
{
2020-08-13 20:17:18 +00:00
NamesAndTypesList new_required_columns;
ColumnsWithTypeAndName new_result_columns;
for (const auto & column : result_columns)
2020-08-12 08:55:16 +00:00
{
2021-03-10 08:41:24 +00:00
if (array_join->columns.count(column.name) != 0 || required_output_.count(column.name) != 0)
2020-08-13 20:17:18 +00:00
new_result_columns.emplace_back(column);
}
for (const auto & column : required_columns)
{
2021-03-10 08:41:24 +00:00
if (array_join->columns.count(column.name) != 0 || required_output_.count(column.name) != 0)
2020-08-13 20:17:18 +00:00
new_required_columns.emplace_back(column);
}
2020-08-12 08:55:16 +00:00
2020-08-13 20:17:18 +00:00
std::swap(required_columns, new_required_columns);
std::swap(result_columns, new_result_columns);
}
2020-08-12 08:55:16 +00:00
2020-09-08 10:40:53 +00:00
ExpressionActionsChain::JoinStep::JoinStep(
std::shared_ptr<TableJoin> analyzed_join_,
JoinPtr join_,
ColumnsWithTypeAndName required_columns_)
: Step({})
, analyzed_join(std::move(analyzed_join_))
, join(std::move(join_))
{
for (const auto & column : required_columns_)
2020-09-08 12:31:36 +00:00
required_columns.emplace_back(column.name, column.type);
NamesAndTypesList result_names_and_types = required_columns;
analyzed_join->addJoinedColumnsAndCorrectTypes(result_names_and_types, true);
for (const auto & [name, type] : result_names_and_types)
/// `column` is `nullptr` because we don't care on constness here, it may be changed in join
result_columns.emplace_back(nullptr, type, name);
2020-09-08 10:40:53 +00:00
}
2021-03-10 08:41:24 +00:00
void ExpressionActionsChain::JoinStep::finalize(const NameSet & required_output_)
2020-09-08 10:40:53 +00:00
{
/// We need to update required and result columns by removing unused ones.
NamesAndTypesList new_required_columns;
ColumnsWithTypeAndName new_result_columns;
/// That's an input columns we need.
2021-03-10 08:41:24 +00:00
NameSet required_names = required_output_;
2021-09-06 10:59:18 +00:00
for (const auto & name : analyzed_join->getAllNames(JoinTableSide::Left))
required_names.emplace(name);
for (const auto & onexpr : analyzed_join->getClauses())
if (const auto & cond_name = onexpr.condColumnNames().first; !cond_name.empty())
required_names.emplace(cond_name);
2020-09-08 10:40:53 +00:00
for (const auto & column : required_columns)
{
if (required_names.count(column.name) != 0)
new_required_columns.emplace_back(column);
}
/// Result will also contain joined columns.
for (const auto & column_name : analyzed_join->columnsAddedByJoin())
required_names.emplace(column_name);
2020-09-08 10:40:53 +00:00
for (const auto & column : result_columns)
{
if (required_names.count(column.name) != 0)
new_result_columns.emplace_back(column);
}
std::swap(required_columns, new_required_columns);
std::swap(result_columns, new_result_columns);
}
2020-09-11 12:24:41 +00:00
ActionsDAGPtr & ExpressionActionsChain::Step::actions()
2020-08-12 08:55:16 +00:00
{
2020-11-11 16:52:27 +00:00
return typeid_cast<ExpressionActionsStep *>(this)->actions_dag;
2020-08-13 20:17:18 +00:00
}
2020-09-11 12:24:41 +00:00
const ActionsDAGPtr & ExpressionActionsChain::Step::actions() const
2020-08-13 20:17:18 +00:00
{
2020-11-11 16:52:27 +00:00
return typeid_cast<const ExpressionActionsStep *>(this)->actions_dag;
2020-08-12 08:55:16 +00:00
}
}