mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-22 07:31:57 +00:00
Fixed tests
This commit is contained in:
parent
14e01566a0
commit
d39be3ac9c
@ -110,8 +110,15 @@ ASTPtr ColumnNode::toASTImpl() const
|
||||
}
|
||||
else if (auto * table_node = column_source->as<TableNode>())
|
||||
{
|
||||
const auto & table_storage_id = table_node->getStorageID();
|
||||
column_identifier_parts = {table_storage_id.getDatabaseName(), table_storage_id.getTableName()};
|
||||
if (!table_node->getTemporaryTableName().empty())
|
||||
{
|
||||
column_identifier_parts = { table_node->getTemporaryTableName() };
|
||||
}
|
||||
else
|
||||
{
|
||||
const auto & table_storage_id = table_node->getStorageID();
|
||||
column_identifier_parts = { table_storage_id.getDatabaseName(), table_storage_id.getTableName() };
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
@ -40,15 +40,7 @@ void IdentifierNode::dumpTreeImpl(WriteBuffer & buffer, FormatState & format_sta
|
||||
bool IdentifierNode::isEqualImpl(const IQueryTreeNode & rhs) const
|
||||
{
|
||||
const auto & rhs_typed = assert_cast<const IdentifierNode &>(rhs);
|
||||
|
||||
if (table_expression_modifiers && rhs_typed.table_expression_modifiers && table_expression_modifiers != rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
else if (table_expression_modifiers && !rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
else if (!table_expression_modifiers && rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
|
||||
return identifier == rhs_typed.identifier;
|
||||
return identifier == rhs_typed.identifier && table_expression_modifiers == rhs_typed.table_expression_modifiers;
|
||||
}
|
||||
|
||||
void IdentifierNode::updateTreeHashImpl(HashState & state) const
|
||||
|
@ -1,5 +1,7 @@
|
||||
#include <Analyzer/Passes/CrossToInnerJoinPass.h>
|
||||
|
||||
#include <DataTypes/getLeastSupertype.h>
|
||||
|
||||
#include <Analyzer/InDepthQueryTreeVisitor.h>
|
||||
|
||||
#include <Analyzer/JoinNode.h>
|
||||
@ -152,25 +154,34 @@ public:
|
||||
QueryTreeNodes other_conditions;
|
||||
exctractJoinConditions(where_condition, equi_conditions, other_conditions);
|
||||
bool can_convert_cross_to_inner = false;
|
||||
for (auto & cond : equi_conditions)
|
||||
for (auto & condition : equi_conditions)
|
||||
{
|
||||
auto left_src = getExpressionSource(getEquiArgument(cond, 0));
|
||||
auto right_src = getExpressionSource(getEquiArgument(cond, 1));
|
||||
if (left_src.second && right_src.second && left_src.first && right_src.first)
|
||||
{
|
||||
bool can_join_on = (findInTableExpression(left_src.first, left_table) && findInTableExpression(right_src.first, right_table))
|
||||
|| (findInTableExpression(left_src.first, right_table) && findInTableExpression(right_src.first, left_table));
|
||||
const auto & lhs_equi_argument = getEquiArgument(condition, 0);
|
||||
const auto & rhs_equi_argument = getEquiArgument(condition, 1);
|
||||
|
||||
if (can_join_on)
|
||||
DataTypes key_types = {lhs_equi_argument->getResultType(), rhs_equi_argument->getResultType()};
|
||||
DataTypePtr common_key_type = tryGetLeastSupertype(key_types);
|
||||
|
||||
/// If there is common key type, we can join on this condition
|
||||
if (common_key_type)
|
||||
{
|
||||
auto left_src = getExpressionSource(lhs_equi_argument);
|
||||
auto right_src = getExpressionSource(rhs_equi_argument);
|
||||
|
||||
if (left_src.second && right_src.second && left_src.first && right_src.first)
|
||||
{
|
||||
can_convert_cross_to_inner = true;
|
||||
continue;
|
||||
if ((findInTableExpression(left_src.first, left_table) && findInTableExpression(right_src.first, right_table)) ||
|
||||
(findInTableExpression(left_src.first, right_table) && findInTableExpression(right_src.first, left_table)))
|
||||
{
|
||||
can_convert_cross_to_inner = true;
|
||||
continue;
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
/// Can't join on this condition, move it to other conditions
|
||||
other_conditions.push_back(cond);
|
||||
cond = nullptr;
|
||||
other_conditions.push_back(condition);
|
||||
condition = nullptr;
|
||||
}
|
||||
|
||||
if (!can_convert_cross_to_inner)
|
||||
|
@ -1,77 +0,0 @@
|
||||
#include <Analyzer/Passes/CustomizeFunctionsPass.h>
|
||||
|
||||
#include <AggregateFunctions/AggregateFunctionFactory.h>
|
||||
#include <AggregateFunctions/IAggregateFunction.h>
|
||||
|
||||
#include <Functions/FunctionFactory.h>
|
||||
|
||||
#include <Interpreters/Context.h>
|
||||
|
||||
#include <Analyzer/InDepthQueryTreeVisitor.h>
|
||||
#include <Analyzer/FunctionNode.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
namespace
|
||||
{
|
||||
|
||||
class CustomizeFunctionsVisitor : public InDepthQueryTreeVisitorWithContext<CustomizeFunctionsVisitor>
|
||||
{
|
||||
public:
|
||||
using Base = InDepthQueryTreeVisitorWithContext<CustomizeFunctionsVisitor>;
|
||||
using Base::Base;
|
||||
|
||||
void visitImpl(QueryTreeNodePtr & node) const
|
||||
{
|
||||
auto * function_node = node->as<FunctionNode>();
|
||||
if (!function_node)
|
||||
return;
|
||||
|
||||
const auto & settings = getSettings();
|
||||
|
||||
/// After successful function replacement function name and function name lowercase must be recalculated
|
||||
auto function_name = function_node->getFunctionName();
|
||||
auto function_name_lowercase = Poco::toLower(function_name);
|
||||
|
||||
if (settings.transform_null_in)
|
||||
{
|
||||
auto function_result_type = function_node->getResultType();
|
||||
|
||||
static constexpr std::array<std::pair<std::string_view, std::string_view>, 4> in_function_to_replace_null_in_function_map =
|
||||
{{
|
||||
{"in", "nullIn"},
|
||||
{"notin", "notNullIn"},
|
||||
{"globalin", "globalNullIn"},
|
||||
{"globalnotin", "globalNotNullIn"},
|
||||
}};
|
||||
|
||||
for (const auto & [in_function_name, in_function_name_to_replace] : in_function_to_replace_null_in_function_map)
|
||||
{
|
||||
if (function_name_lowercase == in_function_name)
|
||||
{
|
||||
resolveOrdinaryFunctionNode(*function_node, String(in_function_name_to_replace));
|
||||
function_name = function_node->getFunctionName();
|
||||
function_name_lowercase = Poco::toLower(function_name);
|
||||
break;
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
inline void resolveOrdinaryFunctionNode(FunctionNode & function_node, const String & function_name) const
|
||||
{
|
||||
auto function = FunctionFactory::instance().get(function_name, getContext());
|
||||
function_node.resolveAsFunction(function->build(function_node.getArgumentColumns()));
|
||||
}
|
||||
};
|
||||
|
||||
}
|
||||
|
||||
void CustomizeFunctionsPass::run(QueryTreeNodePtr query_tree_node, ContextPtr context)
|
||||
{
|
||||
CustomizeFunctionsVisitor visitor(std::move(context));
|
||||
visitor.visit(query_tree_node);
|
||||
}
|
||||
|
||||
}
|
@ -1,25 +0,0 @@
|
||||
#pragma once
|
||||
|
||||
#include <Analyzer/IQueryTreePass.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
/** Customize aggregate functions and `in` functions implementations.
|
||||
*
|
||||
* Example: SELECT countDistinct();
|
||||
* Result: SELECT countDistinctImplementation();
|
||||
* Function countDistinctImplementation is taken from settings.count_distinct_implementation.
|
||||
*/
|
||||
class CustomizeFunctionsPass final : public IQueryTreePass
|
||||
{
|
||||
public:
|
||||
String getName() override { return "CustomizeFunctions"; }
|
||||
|
||||
String getDescription() override { return "Customize implementation of aggregate functions, and in functions."; }
|
||||
|
||||
void run(QueryTreeNodePtr query_tree_node, ContextPtr context) override;
|
||||
|
||||
};
|
||||
|
||||
}
|
@ -37,14 +37,16 @@
|
||||
|
||||
#include <Storages/IStorage.h>
|
||||
#include <Storages/StorageSet.h>
|
||||
#include <Storages/StorageJoin.h>
|
||||
|
||||
#include <Interpreters/misc.h>
|
||||
#include <Interpreters/convertFieldToType.h>
|
||||
#include <Interpreters/StorageID.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Interpreters/SelectQueryOptions.h>
|
||||
#include <Interpreters/InterpreterSelectQueryAnalyzer.h>
|
||||
#include <Interpreters/Set.h>
|
||||
#include <Interpreters/misc.h>
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Interpreters/ExternalDictionariesLoader.h>
|
||||
#include <Interpreters/InterpreterSelectQueryAnalyzer.h>
|
||||
|
||||
#include <Processors/Executors/PullingAsyncPipelineExecutor.h>
|
||||
|
||||
@ -92,6 +94,7 @@ namespace ErrorCodes
|
||||
extern const int CYCLIC_ALIASES;
|
||||
extern const int INCORRECT_RESULT_OF_SCALAR_SUBQUERY;
|
||||
extern const int BAD_ARGUMENTS;
|
||||
extern const int ILLEGAL_TYPE_OF_ARGUMENT;
|
||||
extern const int MULTIPLE_EXPRESSIONS_FOR_ALIAS;
|
||||
extern const int TYPE_MISMATCH;
|
||||
extern const int AMBIGUOUS_IDENTIFIER;
|
||||
@ -1971,26 +1974,10 @@ void QueryAnalyzer::evaluateScalarSubqueryIfNeeded(QueryTreeNodePtr & node, Iden
|
||||
auto constant_node = std::make_shared<ConstantNode>(std::move(constant_value), node);
|
||||
|
||||
if (constant_node->getValue().isNull())
|
||||
{
|
||||
std::string cast_type = constant_node->getResultType()->getName();
|
||||
std::string cast_function_name = "_CAST";
|
||||
|
||||
auto cast_type_constant_value = std::make_shared<ConstantValue>(std::move(cast_type), std::make_shared<DataTypeString>());
|
||||
auto cast_type_constant_node = std::make_shared<ConstantNode>(std::move(cast_type_constant_value));
|
||||
|
||||
auto cast_function_node = std::make_shared<FunctionNode>(cast_function_name);
|
||||
cast_function_node->getArguments().getNodes().push_back(constant_node);
|
||||
cast_function_node->getArguments().getNodes().push_back(std::move(cast_type_constant_node));
|
||||
|
||||
auto cast_function = FunctionFactory::instance().get(cast_function_name, context);
|
||||
cast_function_node->resolveAsFunction(cast_function->build(cast_function_node->getArgumentColumns()));
|
||||
|
||||
node = std::move(cast_function_node);
|
||||
}
|
||||
node = buildCastFunction(constant_node, constant_node->getResultType(), context);
|
||||
else
|
||||
{
|
||||
node = std::move(constant_node);
|
||||
}
|
||||
|
||||
return;
|
||||
}
|
||||
|
||||
@ -2333,6 +2320,8 @@ QueryTreeNodePtr QueryAnalyzer::tryResolveTableIdentifierFromDatabaseCatalog(con
|
||||
|
||||
StorageID storage_id(database_name, table_name);
|
||||
storage_id = context->resolveStorageID(storage_id);
|
||||
bool is_temporary_table = storage_id.getDatabaseName() == DatabaseCatalog::TEMPORARY_DATABASE;
|
||||
|
||||
auto storage = DatabaseCatalog::instance().tryGetTable(storage_id, context);
|
||||
if (!storage)
|
||||
return {};
|
||||
@ -2340,7 +2329,11 @@ QueryTreeNodePtr QueryAnalyzer::tryResolveTableIdentifierFromDatabaseCatalog(con
|
||||
auto storage_lock = storage->lockForShare(context->getInitialQueryId(), context->getSettingsRef().lock_acquire_timeout);
|
||||
auto storage_snapshot = storage->getStorageSnapshot(storage->getInMemoryMetadataPtr(), context);
|
||||
|
||||
return std::make_shared<TableNode>(std::move(storage), std::move(storage_lock), std::move(storage_snapshot));
|
||||
auto result = std::make_shared<TableNode>(std::move(storage), std::move(storage_lock), std::move(storage_snapshot));
|
||||
if (is_temporary_table)
|
||||
result->setTemporaryTableName(table_name);
|
||||
|
||||
return result;
|
||||
}
|
||||
|
||||
/// Resolve identifier from compound expression
|
||||
@ -3961,17 +3954,16 @@ ProjectionNames QueryAnalyzer::resolveMatcher(QueryTreeNodePtr & matcher_node, I
|
||||
matched_expression_nodes_with_names = resolveUnqualifiedMatcher(matcher_node, scope);
|
||||
|
||||
std::unordered_map<const IColumnTransformerNode *, std::unordered_set<std::string>> strict_transformer_to_used_column_names;
|
||||
auto add_strict_transformer_column_name = [&](const IColumnTransformerNode * transformer, const std::string & column_name)
|
||||
for (const auto & transformer : matcher_node_typed.getColumnTransformers().getNodes())
|
||||
{
|
||||
auto it = strict_transformer_to_used_column_names.find(transformer);
|
||||
if (it == strict_transformer_to_used_column_names.end())
|
||||
{
|
||||
auto [inserted_it, _] = strict_transformer_to_used_column_names.emplace(transformer, std::unordered_set<std::string>());
|
||||
it = inserted_it;
|
||||
}
|
||||
auto * except_transformer = transformer->as<ExceptColumnTransformerNode>();
|
||||
auto * replace_transformer = transformer->as<ReplaceColumnTransformerNode>();
|
||||
|
||||
it->second.insert(column_name);
|
||||
};
|
||||
if (except_transformer && except_transformer->isStrict())
|
||||
strict_transformer_to_used_column_names.emplace(except_transformer, std::unordered_set<std::string>());
|
||||
else if (replace_transformer && replace_transformer->isStrict())
|
||||
strict_transformer_to_used_column_names.emplace(replace_transformer, std::unordered_set<std::string>());
|
||||
}
|
||||
|
||||
ListNodePtr list = std::make_shared<ListNode>();
|
||||
ProjectionNames result_projection_names;
|
||||
@ -4026,12 +4018,12 @@ ProjectionNames QueryAnalyzer::resolveMatcher(QueryTreeNodePtr & matcher_node, I
|
||||
else if (auto * except_transformer = transformer->as<ExceptColumnTransformerNode>())
|
||||
{
|
||||
if (apply_transformer_was_used || replace_transformer_was_used)
|
||||
break;
|
||||
continue;
|
||||
|
||||
if (except_transformer->isColumnMatching(column_name))
|
||||
{
|
||||
if (except_transformer->isStrict())
|
||||
add_strict_transformer_column_name(except_transformer, column_name);
|
||||
strict_transformer_to_used_column_names[except_transformer].insert(column_name);
|
||||
|
||||
node = {};
|
||||
break;
|
||||
@ -4040,7 +4032,7 @@ ProjectionNames QueryAnalyzer::resolveMatcher(QueryTreeNodePtr & matcher_node, I
|
||||
else if (auto * replace_transformer = transformer->as<ReplaceColumnTransformerNode>())
|
||||
{
|
||||
if (apply_transformer_was_used || replace_transformer_was_used)
|
||||
break;
|
||||
continue;
|
||||
|
||||
replace_transformer_was_used = true;
|
||||
|
||||
@ -4049,7 +4041,7 @@ ProjectionNames QueryAnalyzer::resolveMatcher(QueryTreeNodePtr & matcher_node, I
|
||||
continue;
|
||||
|
||||
if (replace_transformer->isStrict())
|
||||
add_strict_transformer_column_name(replace_transformer, column_name);
|
||||
strict_transformer_to_used_column_names[replace_transformer].insert(column_name);
|
||||
|
||||
node = replace_expression->clone();
|
||||
node_projection_names = resolveExpressionNode(node, scope, false /*allow_lambda_expression*/, false /*allow_table_expression*/);
|
||||
@ -4140,22 +4132,12 @@ ProjectionNames QueryAnalyzer::resolveMatcher(QueryTreeNodePtr & matcher_node, I
|
||||
non_matched_column_names.push_back(column_name);
|
||||
}
|
||||
|
||||
WriteBufferFromOwnString non_matched_column_names_buffer;
|
||||
size_t non_matched_column_names_size = non_matched_column_names.size();
|
||||
for (size_t i = 0; i < non_matched_column_names_size; ++i)
|
||||
{
|
||||
const auto & column_name = non_matched_column_names[i];
|
||||
|
||||
non_matched_column_names_buffer << column_name;
|
||||
if (i + 1 != non_matched_column_names_size)
|
||||
non_matched_column_names_buffer << ", ";
|
||||
}
|
||||
|
||||
throw Exception(ErrorCodes::BAD_ARGUMENTS,
|
||||
"Strict {} column transformer {} expects following column(s) {}",
|
||||
"Strict {} column transformer {} expects following column(s) : {}. In scope {}",
|
||||
toString(strict_transformer_type),
|
||||
strict_transformer->formatASTForErrorMessage(),
|
||||
non_matched_column_names_buffer.str());
|
||||
fmt::join(non_matched_column_names, ", "),
|
||||
scope.scope_node->formatASTForErrorMessage());
|
||||
}
|
||||
|
||||
matcher_node = std::move(list);
|
||||
@ -4449,17 +4431,26 @@ ProjectionNames QueryAnalyzer::resolveFunction(QueryTreeNodePtr & node, Identifi
|
||||
}
|
||||
|
||||
bool is_special_function_in = false;
|
||||
bool is_special_function_dict_get_or_join_get = false;
|
||||
bool is_special_function_dict_get = false;
|
||||
bool is_special_function_join_get = false;
|
||||
bool is_special_function_exists = false;
|
||||
|
||||
if (!lambda_expression_untyped)
|
||||
{
|
||||
is_special_function_in = isNameOfInFunction(function_name);
|
||||
is_special_function_dict_get_or_join_get = functionIsJoinGet(function_name) || functionIsDictGet(function_name);
|
||||
is_special_function_dict_get = functionIsDictGet(function_name);
|
||||
is_special_function_join_get = functionIsJoinGet(function_name);
|
||||
is_special_function_exists = function_name == "exists";
|
||||
|
||||
/// Handle SELECT count(*) FROM test_table
|
||||
if (Poco::toLower(function_name) == "count" && function_node_ptr->getArguments().getNodes().size() == 1)
|
||||
auto function_name_lowercase = Poco::toLower(function_name);
|
||||
|
||||
/** Special handling for count and countState functions.
|
||||
*
|
||||
* Example: SELECT count(*) FROM test_table
|
||||
* Example: SELECT countState(*) FROM test_table;
|
||||
*/
|
||||
if (function_node_ptr->getArguments().getNodes().size() == 1 &&
|
||||
(function_name_lowercase == "count" || function_name_lowercase == "countstate"))
|
||||
{
|
||||
auto * matcher_node = function_node_ptr->getArguments().getNodes().front()->as<MatcherNode>();
|
||||
if (matcher_node && matcher_node->isUnqualified())
|
||||
@ -4476,19 +4467,56 @@ ProjectionNames QueryAnalyzer::resolveFunction(QueryTreeNodePtr & node, Identifi
|
||||
* Otherwise replace identifier with identifier full name constant.
|
||||
* Validation that dictionary exists or table exists will be performed during function `getReturnType` method call.
|
||||
*/
|
||||
if (is_special_function_dict_get_or_join_get &&
|
||||
if ((is_special_function_dict_get || is_special_function_join_get) &&
|
||||
!function_node_ptr->getArguments().getNodes().empty() &&
|
||||
function_node_ptr->getArguments().getNodes()[0]->getNodeType() == QueryTreeNodeType::IDENTIFIER)
|
||||
{
|
||||
auto & first_argument = function_node_ptr->getArguments().getNodes()[0];
|
||||
auto & identifier_node = first_argument->as<IdentifierNode &>();
|
||||
IdentifierLookup identifier_lookup{identifier_node.getIdentifier(), IdentifierLookupContext::EXPRESSION};
|
||||
auto & first_argument_identifier = first_argument->as<IdentifierNode &>();
|
||||
auto identifier = first_argument_identifier.getIdentifier();
|
||||
|
||||
IdentifierLookup identifier_lookup{identifier, IdentifierLookupContext::EXPRESSION};
|
||||
auto resolve_result = tryResolveIdentifier(identifier_lookup, scope);
|
||||
|
||||
if (resolve_result.isResolved())
|
||||
{
|
||||
first_argument = std::move(resolve_result.resolved_identifier);
|
||||
}
|
||||
else
|
||||
first_argument = std::make_shared<ConstantNode>(identifier_node.getIdentifier().getFullName());
|
||||
{
|
||||
size_t parts_size = identifier.getPartsSize();
|
||||
if (parts_size < 1 || parts_size > 2)
|
||||
throw Exception(ErrorCodes::BAD_ARGUMENTS,
|
||||
"Expected {} function first argument identifier to contain 1 or 2 parts. Actual '{}'. In scope {}",
|
||||
function_name,
|
||||
identifier.getFullName(),
|
||||
scope.scope_node->formatASTForErrorMessage());
|
||||
|
||||
if (is_special_function_dict_get)
|
||||
{
|
||||
scope.context->getExternalDictionariesLoader().assertDictionaryStructureExists(identifier.getFullName(), scope.context);
|
||||
}
|
||||
else
|
||||
{
|
||||
auto table_node = tryResolveTableIdentifierFromDatabaseCatalog(identifier, scope.context);
|
||||
if (!table_node)
|
||||
throw Exception(ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
|
||||
"Function {} first argument expected table identifier '{}'. In scope {}",
|
||||
function_name,
|
||||
identifier.getFullName(),
|
||||
scope.scope_node->formatASTForErrorMessage());
|
||||
|
||||
auto & table_node_typed = table_node->as<TableNode &>();
|
||||
if (!std::dynamic_pointer_cast<StorageJoin>(table_node_typed.getStorage()))
|
||||
throw Exception(ErrorCodes::ILLEGAL_TYPE_OF_ARGUMENT,
|
||||
"Function {} table '{}' should have engine StorageJoin. In scope {}",
|
||||
function_name,
|
||||
identifier.getFullName(),
|
||||
scope.scope_node->formatASTForErrorMessage());
|
||||
}
|
||||
|
||||
first_argument = std::make_shared<ConstantNode>(identifier.getFullName());
|
||||
}
|
||||
}
|
||||
|
||||
/// Resolve function arguments
|
||||
@ -4526,6 +4554,26 @@ ProjectionNames QueryAnalyzer::resolveFunction(QueryTreeNodePtr & node, Identifi
|
||||
/// Replace right IN function argument if it is table or table function with subquery that read ordinary columns
|
||||
if (is_special_function_in)
|
||||
{
|
||||
if (scope.context->getSettingsRef().transform_null_in)
|
||||
{
|
||||
static constexpr std::array<std::pair<std::string_view, std::string_view>, 4> in_function_to_replace_null_in_function_map =
|
||||
{{
|
||||
{"in", "nullIn"},
|
||||
{"notIn", "notNullIn"},
|
||||
{"globalIn", "globalNullIn"},
|
||||
{"globalNotIn", "globalNotNullIn"},
|
||||
}};
|
||||
|
||||
for (const auto & [in_function_name, in_function_name_to_replace] : in_function_to_replace_null_in_function_map)
|
||||
{
|
||||
if (function_name == in_function_name)
|
||||
{
|
||||
function_name = in_function_name_to_replace;
|
||||
break;
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
auto & function_in_arguments_nodes = function_node.getArguments().getNodes();
|
||||
if (function_in_arguments_nodes.size() != 2)
|
||||
throw Exception(ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH, "Function '{}' expects 2 arguments", function_name);
|
||||
@ -4586,6 +4634,7 @@ ProjectionNames QueryAnalyzer::resolveFunction(QueryTreeNodePtr & node, Identifi
|
||||
auto & function_argument = function_arguments[function_argument_index];
|
||||
|
||||
ColumnWithTypeAndName argument_column;
|
||||
argument_column.name = arguments_projection_names[function_argument_index];
|
||||
|
||||
/** If function argument is lambda, save lambda argument index and initialize argument type as DataTypeFunction
|
||||
* where function argument types are initialized with empty array of lambda arguments size.
|
||||
@ -4627,7 +4676,7 @@ ProjectionNames QueryAnalyzer::resolveFunction(QueryTreeNodePtr & node, Identifi
|
||||
}
|
||||
|
||||
/// Calculate function projection name
|
||||
ProjectionNames result_projection_names = {calculateFunctionProjectionName(node, parameters_projection_names, arguments_projection_names)};
|
||||
ProjectionNames result_projection_names = { calculateFunctionProjectionName(node, parameters_projection_names, arguments_projection_names) };
|
||||
|
||||
/** Try to resolve function as
|
||||
* 1. Lambda function in current scope. Example: WITH (x -> x + 1) AS lambda SELECT lambda(1);
|
||||
@ -4939,7 +4988,7 @@ ProjectionNames QueryAnalyzer::resolveFunction(QueryTreeNodePtr & node, Identifi
|
||||
}
|
||||
|
||||
/// Recalculate function projection name after lambda resolution
|
||||
result_projection_names = {calculateFunctionProjectionName(node, parameters_projection_names, arguments_projection_names)};
|
||||
result_projection_names = { calculateFunctionProjectionName(node, parameters_projection_names, arguments_projection_names) };
|
||||
}
|
||||
|
||||
/** Create SET column for special function IN to allow constant folding
|
||||
@ -5765,10 +5814,19 @@ void QueryAnalyzer::initializeTableExpressionData(const QueryTreeNodePtr & table
|
||||
|
||||
if (table_node)
|
||||
{
|
||||
const auto & table_storage_id = table_node->getStorageID();
|
||||
table_expression_data.database_name = table_storage_id.database_name;
|
||||
table_expression_data.table_name = table_storage_id.table_name;
|
||||
table_expression_data.table_expression_name = table_storage_id.getFullNameNotQuoted();
|
||||
if (!table_node->getTemporaryTableName().empty())
|
||||
{
|
||||
table_expression_data.table_name = table_node->getTemporaryTableName();
|
||||
table_expression_data.table_expression_name = table_node->getTemporaryTableName();
|
||||
}
|
||||
else
|
||||
{
|
||||
const auto & table_storage_id = table_node->getStorageID();
|
||||
table_expression_data.database_name = table_storage_id.database_name;
|
||||
table_expression_data.table_name = table_storage_id.table_name;
|
||||
table_expression_data.table_expression_name = table_storage_id.getFullNameNotQuoted();
|
||||
}
|
||||
|
||||
table_expression_data.table_expression_description = "table";
|
||||
}
|
||||
else if (query_node || union_node)
|
||||
@ -5814,7 +5872,9 @@ void QueryAnalyzer::initializeTableExpressionData(const QueryTreeNodePtr & table
|
||||
|
||||
if (column_default && column_default->kind == ColumnDefaultKind::Alias)
|
||||
{
|
||||
auto column_node = std::make_shared<ColumnNode>(column_name_and_type, buildQueryTree(column_default->expression, scope.context), table_expression_node);
|
||||
auto alias_expression = buildQueryTree(column_default->expression, scope.context);
|
||||
alias_expression = buildCastFunction(alias_expression, column_name_and_type.type, scope.context, false /*resolve*/);
|
||||
auto column_node = std::make_shared<ColumnNode>(column_name_and_type, std::move(alias_expression), table_expression_node);
|
||||
column_name_to_column_node.emplace(column_name_and_type.name, column_node);
|
||||
alias_columns_to_resolve.emplace_back(column_name_and_type.name, column_node);
|
||||
}
|
||||
@ -6181,13 +6241,6 @@ void QueryAnalyzer::resolveJoin(QueryTreeNodePtr & join_node, IdentifierResolveS
|
||||
|
||||
IdentifierLookup identifier_lookup{identifier_node->getIdentifier(), IdentifierLookupContext::EXPRESSION};
|
||||
auto result_left_table_expression = tryResolveIdentifierFromJoinTreeNode(identifier_lookup, join_node_typed.getLeftTableExpression(), scope);
|
||||
if (!result_left_table_expression && identifier_node->hasAlias())
|
||||
{
|
||||
std::vector<std::string> alias_identifier_parts = {identifier_node->getAlias()};
|
||||
IdentifierLookup alias_identifier_lookup{Identifier(std::move(alias_identifier_parts)), IdentifierLookupContext::EXPRESSION};
|
||||
result_left_table_expression = tryResolveIdentifierFromJoinTreeNode(alias_identifier_lookup, join_node_typed.getLeftTableExpression(), scope);
|
||||
}
|
||||
|
||||
if (!result_left_table_expression)
|
||||
throw Exception(ErrorCodes::UNKNOWN_IDENTIFIER,
|
||||
"JOIN {} using identifier '{}' cannot be resolved from left table expression. In scope {}",
|
||||
@ -6203,13 +6256,6 @@ void QueryAnalyzer::resolveJoin(QueryTreeNodePtr & join_node, IdentifierResolveS
|
||||
scope.scope_node->formatASTForErrorMessage());
|
||||
|
||||
auto result_right_table_expression = tryResolveIdentifierFromJoinTreeNode(identifier_lookup, join_node_typed.getRightTableExpression(), scope);
|
||||
if (!result_right_table_expression && identifier_node->hasAlias())
|
||||
{
|
||||
std::vector<std::string> alias_identifier_parts = {identifier_node->getAlias()};
|
||||
IdentifierLookup alias_identifier_lookup{Identifier(std::move(alias_identifier_parts)), IdentifierLookupContext::EXPRESSION};
|
||||
result_right_table_expression = tryResolveIdentifierFromJoinTreeNode(alias_identifier_lookup, join_node_typed.getRightTableExpression(), scope);
|
||||
}
|
||||
|
||||
if (!result_right_table_expression)
|
||||
throw Exception(ErrorCodes::UNKNOWN_IDENTIFIER,
|
||||
"JOIN {} using identifier '{}' cannot be resolved from right table expression. In scope {}",
|
||||
|
@ -54,6 +54,7 @@ namespace DB
|
||||
* Replace `countDistinct` and `countIfDistinct` aggregate functions using setting count_distinct_implementation.
|
||||
* Add -OrNull suffix to aggregate functions if setting aggregate_functions_null_for_empty is true.
|
||||
* Function `exists` is converted into `in`.
|
||||
* Functions `in`, `notIn`, `globalIn`, `globalNotIn` converted into `nullIn`, `notNullIn`, `globalNullIn`, `globalNotNullIn` if setting transform_null_in is true.
|
||||
*
|
||||
* For function `grouping` arguments are resolved, but it is planner responsibility to initialize it with concrete grouping function
|
||||
* based on group by kind and group by keys positions.
|
||||
|
@ -26,7 +26,6 @@
|
||||
#include <Analyzer/Passes/IfChainToMultiIfPass.h>
|
||||
#include <Analyzer/Passes/OrderByTupleEliminationPass.h>
|
||||
#include <Analyzer/Passes/NormalizeCountVariantsPass.h>
|
||||
#include <Analyzer/Passes/CustomizeFunctionsPass.h>
|
||||
#include <Analyzer/Passes/AggregateFunctionsArithmericOperationsPass.h>
|
||||
#include <Analyzer/Passes/UniqInjectiveFunctionsEliminationPass.h>
|
||||
#include <Analyzer/Passes/OrderByLimitByDuplicateEliminationPass.h>
|
||||
@ -242,8 +241,6 @@ void addQueryTreePasses(QueryTreePassManager & manager)
|
||||
manager.addPass(std::make_unique<RewriteArrayExistsToHasPass>());
|
||||
manager.addPass(std::make_unique<NormalizeCountVariantsPass>());
|
||||
|
||||
manager.addPass(std::make_unique<CustomizeFunctionsPass>());
|
||||
|
||||
manager.addPass(std::make_unique<AggregateFunctionsArithmericOperationsPass>());
|
||||
manager.addPass(std::make_unique<UniqInjectiveFunctionsEliminationPass>());
|
||||
manager.addPass(std::make_unique<OptimizeGroupByFunctionKeysPass>());
|
||||
|
@ -82,14 +82,7 @@ bool TableFunctionNode::isEqualImpl(const IQueryTreeNode & rhs) const
|
||||
if (storage && rhs_typed.storage)
|
||||
return storage_id == rhs_typed.storage_id;
|
||||
|
||||
if (table_expression_modifiers && rhs_typed.table_expression_modifiers && table_expression_modifiers != rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
else if (table_expression_modifiers && !rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
else if (!table_expression_modifiers && rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
|
||||
return true;
|
||||
return table_expression_modifiers == rhs_typed.table_expression_modifiers;
|
||||
}
|
||||
|
||||
void TableFunctionNode::updateTreeHashImpl(HashState & state) const
|
||||
|
@ -42,6 +42,9 @@ void TableNode::dumpTreeImpl(WriteBuffer & buffer, FormatState & format_state, s
|
||||
|
||||
buffer << ", table_name: " << storage_id.getFullNameNotQuoted();
|
||||
|
||||
if (!temporary_table_name.empty())
|
||||
buffer << ", temporary_table_name: " << temporary_table_name;
|
||||
|
||||
if (table_expression_modifiers)
|
||||
{
|
||||
buffer << ", ";
|
||||
@ -52,15 +55,8 @@ void TableNode::dumpTreeImpl(WriteBuffer & buffer, FormatState & format_state, s
|
||||
bool TableNode::isEqualImpl(const IQueryTreeNode & rhs) const
|
||||
{
|
||||
const auto & rhs_typed = assert_cast<const TableNode &>(rhs);
|
||||
|
||||
if (table_expression_modifiers && rhs_typed.table_expression_modifiers && table_expression_modifiers != rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
else if (table_expression_modifiers && !rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
else if (!table_expression_modifiers && rhs_typed.table_expression_modifiers)
|
||||
return false;
|
||||
|
||||
return storage_id == rhs_typed.storage_id;
|
||||
return storage_id == rhs_typed.storage_id && table_expression_modifiers == rhs_typed.table_expression_modifiers &&
|
||||
temporary_table_name == rhs_typed.temporary_table_name;
|
||||
}
|
||||
|
||||
void TableNode::updateTreeHashImpl(HashState & state) const
|
||||
@ -69,6 +65,9 @@ void TableNode::updateTreeHashImpl(HashState & state) const
|
||||
state.update(full_name.size());
|
||||
state.update(full_name);
|
||||
|
||||
state.update(temporary_table_name.size());
|
||||
state.update(temporary_table_name);
|
||||
|
||||
if (table_expression_modifiers)
|
||||
table_expression_modifiers->updateTreeHash(state);
|
||||
}
|
||||
@ -77,12 +76,16 @@ QueryTreeNodePtr TableNode::cloneImpl() const
|
||||
{
|
||||
auto result_table_node = std::make_shared<TableNode>(storage, storage_id, storage_lock, storage_snapshot);
|
||||
result_table_node->table_expression_modifiers = table_expression_modifiers;
|
||||
result_table_node->temporary_table_name = temporary_table_name;
|
||||
|
||||
return result_table_node;
|
||||
}
|
||||
|
||||
ASTPtr TableNode::toASTImpl() const
|
||||
{
|
||||
if (!temporary_table_name.empty())
|
||||
return std::make_shared<ASTTableIdentifier>(temporary_table_name);
|
||||
|
||||
return std::make_shared<ASTTableIdentifier>(storage_id.getDatabaseName(), storage_id.getTableName());
|
||||
}
|
||||
|
||||
|
@ -56,6 +56,18 @@ public:
|
||||
return storage_lock;
|
||||
}
|
||||
|
||||
/// Get temporary table name
|
||||
const std::string & getTemporaryTableName() const
|
||||
{
|
||||
return temporary_table_name;
|
||||
}
|
||||
|
||||
/// Set temporary table name
|
||||
void setTemporaryTableName(std::string temporary_table_name_value)
|
||||
{
|
||||
temporary_table_name = std::move(temporary_table_name_value);
|
||||
}
|
||||
|
||||
/// Return true if table node has table expression modifiers, false otherwise
|
||||
bool hasTableExpressionModifiers() const
|
||||
{
|
||||
@ -102,6 +114,7 @@ private:
|
||||
TableLockHolder storage_lock;
|
||||
StorageSnapshotPtr storage_snapshot;
|
||||
std::optional<TableExpressionModifiers> table_expression_modifiers;
|
||||
std::string temporary_table_name;
|
||||
|
||||
static constexpr size_t children_size = 0;
|
||||
};
|
||||
|
@ -5,13 +5,18 @@
|
||||
#include <Parsers/ASTSubquery.h>
|
||||
#include <Parsers/ASTFunction.h>
|
||||
|
||||
#include <DataTypes/DataTypeString.h>
|
||||
#include <DataTypes/DataTypeTuple.h>
|
||||
#include <DataTypes/DataTypeArray.h>
|
||||
|
||||
#include <Functions/FunctionHelpers.h>
|
||||
#include <Functions/FunctionFactory.h>
|
||||
|
||||
#include <Interpreters/Context.h>
|
||||
|
||||
#include <Analyzer/InDepthQueryTreeVisitor.h>
|
||||
#include <Analyzer/IdentifierNode.h>
|
||||
#include <Analyzer/ConstantNode.h>
|
||||
#include <Analyzer/ColumnNode.h>
|
||||
#include <Analyzer/FunctionNode.h>
|
||||
#include <Analyzer/JoinNode.h>
|
||||
@ -74,6 +79,29 @@ bool isNameOfInFunction(const std::string & function_name)
|
||||
return is_special_function_in;
|
||||
}
|
||||
|
||||
QueryTreeNodePtr buildCastFunction(const QueryTreeNodePtr & expression,
|
||||
const DataTypePtr & type,
|
||||
const ContextPtr & context,
|
||||
bool resolve)
|
||||
{
|
||||
std::string cast_type = type->getName();
|
||||
auto cast_type_constant_value = std::make_shared<ConstantValue>(std::move(cast_type), std::make_shared<DataTypeString>());
|
||||
auto cast_type_constant_node = std::make_shared<ConstantNode>(std::move(cast_type_constant_value));
|
||||
|
||||
std::string cast_function_name = "_CAST";
|
||||
auto cast_function_node = std::make_shared<FunctionNode>(cast_function_name);
|
||||
cast_function_node->getArguments().getNodes().push_back(expression);
|
||||
cast_function_node->getArguments().getNodes().push_back(std::move(cast_type_constant_node));
|
||||
|
||||
if (resolve)
|
||||
{
|
||||
auto cast_function = FunctionFactory::instance().get(cast_function_name, context);
|
||||
cast_function_node->resolveAsFunction(cast_function->build(cast_function_node->getArgumentColumns()));
|
||||
}
|
||||
|
||||
return cast_function_node;
|
||||
}
|
||||
|
||||
static ASTPtr convertIntoTableExpressionAST(const QueryTreeNodePtr & table_expression_node)
|
||||
{
|
||||
ASTPtr table_expression_node_ast;
|
||||
|
@ -2,6 +2,8 @@
|
||||
|
||||
#include <Analyzer/IQueryTreeNode.h>
|
||||
|
||||
#include <Interpreters/Context_fwd.h>
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
@ -11,6 +13,15 @@ bool isNodePartOfTree(const IQueryTreeNode * node, const IQueryTreeNode * root);
|
||||
/// Returns true if function name is name of IN function or its variations, false otherwise
|
||||
bool isNameOfInFunction(const std::string & function_name);
|
||||
|
||||
/** Build cast function that cast expression into type.
|
||||
* If resolve = true, then result cast function is resolved during build, otherwise
|
||||
* result cast function is not resolved during build.
|
||||
*/
|
||||
QueryTreeNodePtr buildCastFunction(const QueryTreeNodePtr & expression,
|
||||
const DataTypePtr & type,
|
||||
const ContextPtr & context,
|
||||
bool resolve = true);
|
||||
|
||||
/** Add table expression in tables in select query children.
|
||||
* If table expression node is not of identifier node, table node, query node, table function node, join node or array join node type throws logical error exception.
|
||||
*/
|
||||
|
@ -675,27 +675,28 @@ private:
|
||||
return impl.isInjective(sample_columns);
|
||||
}
|
||||
|
||||
DataTypePtr getReturnTypeImpl(const DataTypes &) const override
|
||||
DataTypePtr getReturnTypeImpl(const ColumnsWithTypeAndName & arguments) const override
|
||||
{
|
||||
DataTypePtr result;
|
||||
DataTypePtr result_type;
|
||||
|
||||
if constexpr (IsDataTypeDecimal<DataType>)
|
||||
result = std::make_shared<DataType>(DataType::maxPrecision(), 0);
|
||||
result_type = std::make_shared<DataType>(DataType::maxPrecision(), 0);
|
||||
else
|
||||
result = std::make_shared<DataType>();
|
||||
result_type = std::make_shared<DataType>();
|
||||
|
||||
return result;
|
||||
auto return_type = impl.getReturnTypeImpl(arguments);
|
||||
if (!return_type->equals(*result_type))
|
||||
throw Exception(ErrorCodes::TYPE_MISMATCH, "Function {} dictionary attribute has different type {} expected {}",
|
||||
getName(),
|
||||
return_type->getName(),
|
||||
result_type->getName());
|
||||
|
||||
return result_type;
|
||||
}
|
||||
|
||||
ColumnPtr executeImpl(const ColumnsWithTypeAndName & arguments, const DataTypePtr & result_type, size_t input_rows_count) const override
|
||||
{
|
||||
auto return_type = impl.getReturnTypeImpl(arguments);
|
||||
|
||||
if (!return_type->equals(*result_type))
|
||||
throw Exception(ErrorCodes::TYPE_MISMATCH, "Dictionary attribute has different type {} expected {}",
|
||||
return_type->getName(), result_type->getName());
|
||||
|
||||
return impl.executeImpl(arguments, return_type, input_rows_count);
|
||||
return impl.executeImpl(arguments, result_type, input_rows_count);
|
||||
}
|
||||
|
||||
const FunctionDictGetNoType<dictionary_get_function_type> impl;
|
||||
|
@ -35,7 +35,10 @@ class FunctionFormatRow : public IFunction
|
||||
public:
|
||||
static constexpr auto name = no_newline ? "formatRowNoNewline" : "formatRow";
|
||||
|
||||
FunctionFormatRow(const String & format_name_, ContextPtr context_) : format_name(format_name_), context(context_)
|
||||
FunctionFormatRow(String format_name_, Names arguments_column_names_, ContextPtr context_)
|
||||
: format_name(std::move(format_name_))
|
||||
, arguments_column_names(std::move(arguments_column_names_))
|
||||
, context(std::move(context_))
|
||||
{
|
||||
if (!FormatFactory::instance().getAllFormats().contains(format_name))
|
||||
throw Exception(ErrorCodes::UNKNOWN_FORMAT, "Unknown format {}", format_name);
|
||||
@ -55,9 +58,17 @@ public:
|
||||
WriteBufferFromVector buffer(vec);
|
||||
ColumnString::Offsets & offsets = col_str->getOffsets();
|
||||
offsets.resize(input_rows_count);
|
||||
|
||||
Block arg_columns;
|
||||
for (auto i = 1u; i < arguments.size(); ++i)
|
||||
arg_columns.insert(arguments[i]);
|
||||
|
||||
size_t arguments_size = arguments.size();
|
||||
for (size_t i = 1; i < arguments_size; ++i)
|
||||
{
|
||||
auto argument_column = arguments[i];
|
||||
argument_column.name = arguments_column_names[i];
|
||||
arg_columns.insert(std::move(argument_column));
|
||||
}
|
||||
|
||||
materializeBlockInplace(arg_columns);
|
||||
auto format_settings = getFormatSettings(context);
|
||||
auto out = FormatFactory::instance().getOutputFormat(format_name, buffer, arg_columns, context, format_settings);
|
||||
@ -93,6 +104,7 @@ public:
|
||||
|
||||
private:
|
||||
String format_name;
|
||||
Names arguments_column_names;
|
||||
ContextPtr context;
|
||||
};
|
||||
|
||||
@ -115,9 +127,14 @@ public:
|
||||
throw Exception(ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH,
|
||||
"Function {} requires at least two arguments: the format name and its output expression(s)", getName());
|
||||
|
||||
Names arguments_column_names;
|
||||
arguments_column_names.reserve(arguments.size());
|
||||
for (const auto & argument : arguments)
|
||||
arguments_column_names.push_back(argument.name);
|
||||
|
||||
if (const auto * name_col = checkAndGetColumnConst<ColumnString>(arguments.at(0).column.get()))
|
||||
return std::make_unique<FunctionToFunctionBaseAdaptor>(
|
||||
std::make_shared<FunctionFormatRow<no_newline>>(name_col->getValue<String>(), context),
|
||||
std::make_shared<FunctionFormatRow<no_newline>>(name_col->getValue<String>(), std::move(arguments_column_names), context),
|
||||
collections::map<DataTypes>(arguments, [](const auto & elem) { return elem.type; }),
|
||||
return_type);
|
||||
else
|
||||
|
@ -85,6 +85,11 @@ DictionaryStructure ExternalDictionariesLoader::getDictionaryStructure(const std
|
||||
return ExternalDictionariesLoader::getDictionaryStructure(*load_result.config);
|
||||
}
|
||||
|
||||
void ExternalDictionariesLoader::assertDictionaryStructureExists(const std::string & dictionary_name, ContextPtr query_context) const
|
||||
{
|
||||
getDictionaryStructure(dictionary_name, query_context);
|
||||
}
|
||||
|
||||
QualifiedTableName ExternalDictionariesLoader::qualifyDictionaryNameWithDatabase(const std::string & dictionary_name, ContextPtr query_context) const
|
||||
{
|
||||
auto qualified_name = QualifiedTableName::tryParseFromString(dictionary_name);
|
||||
|
@ -31,6 +31,8 @@ public:
|
||||
|
||||
DictionaryStructure getDictionaryStructure(const std::string & dictionary_name, ContextPtr context) const;
|
||||
|
||||
void assertDictionaryStructureExists(const std::string & dictionary_name, ContextPtr context) const;
|
||||
|
||||
static DictionaryStructure getDictionaryStructure(const Poco::Util::AbstractConfiguration & config, const std::string & key_in_config = "dictionary");
|
||||
|
||||
static DictionaryStructure getDictionaryStructure(const ObjectConfig & config);
|
||||
|
@ -1474,7 +1474,7 @@ bool ParserColumnsTransformers::parseImpl(Pos & pos, ASTPtr & node, Expected & e
|
||||
auto opos = pos;
|
||||
if (ParserExpression().parse(pos, lambda, expected))
|
||||
{
|
||||
if (const auto * func = lambda->as<ASTFunction>(); func && func->name == "lambda")
|
||||
if (auto * func = lambda->as<ASTFunction>(); func && func->name == "lambda")
|
||||
{
|
||||
if (func->arguments->children.size() != 2)
|
||||
throw Exception(ErrorCodes::SYNTAX_ERROR, "lambda requires two arguments");
|
||||
@ -1491,6 +1491,8 @@ bool ParserColumnsTransformers::parseImpl(Pos & pos, ASTPtr & node, Expected & e
|
||||
lambda_arg = *opt_arg_name;
|
||||
else
|
||||
throw Exception(ErrorCodes::SYNTAX_ERROR, "lambda argument declarations must be identifiers");
|
||||
|
||||
func->is_lambda_function = true;
|
||||
}
|
||||
else
|
||||
{
|
||||
|
@ -404,15 +404,18 @@ JoinTreeQueryPlan buildQueryPlanForTableExpression(const QueryTreeNodePtr & tabl
|
||||
auto expected_header = planner.getQueryPlan().getCurrentDataStream().header;
|
||||
materializeBlockInplace(expected_header);
|
||||
|
||||
auto rename_actions_dag = ActionsDAG::makeConvertingActions(
|
||||
query_plan.getCurrentDataStream().header.getColumnsWithTypeAndName(),
|
||||
expected_header.getColumnsWithTypeAndName(),
|
||||
ActionsDAG::MatchColumnsMode::Position,
|
||||
true /*ignore_constant_values*/);
|
||||
auto rename_step = std::make_unique<ExpressionStep>(query_plan.getCurrentDataStream(), std::move(rename_actions_dag));
|
||||
std::string step_description = table_expression_data.isRemote() ? "Change remote column names to local column names" : "Change column names";
|
||||
rename_step->setStepDescription(std::move(step_description));
|
||||
query_plan.addStep(std::move(rename_step));
|
||||
if (!blocksHaveEqualStructure(query_plan.getCurrentDataStream().header, expected_header))
|
||||
{
|
||||
auto rename_actions_dag = ActionsDAG::makeConvertingActions(
|
||||
query_plan.getCurrentDataStream().header.getColumnsWithTypeAndName(),
|
||||
expected_header.getColumnsWithTypeAndName(),
|
||||
ActionsDAG::MatchColumnsMode::Position,
|
||||
true /*ignore_constant_values*/);
|
||||
auto rename_step = std::make_unique<ExpressionStep>(query_plan.getCurrentDataStream(), std::move(rename_actions_dag));
|
||||
std::string step_description = table_expression_data.isRemote() ? "Change remote column names to local column names" : "Change column names";
|
||||
rename_step->setStepDescription(std::move(step_description));
|
||||
query_plan.addStep(std::move(rename_step));
|
||||
}
|
||||
}
|
||||
|
||||
return {std::move(query_plan), from_stage};
|
||||
|
@ -647,9 +647,7 @@ QueryTreeNodePtr buildQueryTreeDistributedTableReplacedWithLocalTable(const Sele
|
||||
else
|
||||
{
|
||||
auto resolved_remote_storage_id = query_context->resolveStorageID(remote_storage_id);
|
||||
auto storage = DatabaseCatalog::instance().tryGetTable(resolved_remote_storage_id, query_context);
|
||||
if (!storage)
|
||||
storage = std::make_shared<StorageDummy>(resolved_remote_storage_id, distributed_storage_snapshot->metadata->getColumns());
|
||||
auto storage = std::make_shared<StorageDummy>(resolved_remote_storage_id, distributed_storage_snapshot->metadata->getColumns());
|
||||
|
||||
replacement_table_expression = std::make_shared<TableNode>(std::move(storage), query_context);
|
||||
}
|
||||
|
@ -5,16 +5,16 @@
|
||||
11
|
||||
40
|
||||
|
||||
40
|
||||
0
|
||||
41
|
||||
|
||||
41
|
||||
0
|
||||
2 42
|
||||
|
||||
2 42
|
||||
2 0
|
||||
43
|
||||
|
||||
43
|
||||
0
|
||||
11
|
||||
|
||||
11
|
||||
|
@ -1,6 +1,7 @@
|
||||
-- Tags: shard
|
||||
|
||||
set enable_positional_arguments=0;
|
||||
set allow_experimental_analyzer = 1;
|
||||
set enable_positional_arguments = 0;
|
||||
|
||||
select 40 as z from (select * from system.numbers limit 3) group by z;
|
||||
select 41 as z from remote('127.0.0.{2,3}', system.one) group by z;
|
||||
|
@ -1,6 +1,6 @@
|
||||
1 1
|
||||
|
||||
1 1
|
||||
0 1
|
||||
|
||||
1 1
|
||||
1 1
|
||||
@ -27,7 +27,7 @@
|
||||
|
||||
"totals":
|
||||
{
|
||||
"k": "1234567890123",
|
||||
"k": "0",
|
||||
"count()": "1"
|
||||
},
|
||||
|
||||
@ -65,7 +65,7 @@
|
||||
[1.23, "1"]
|
||||
],
|
||||
|
||||
"totals": [1.23, "1"],
|
||||
"totals": [0, "1"],
|
||||
|
||||
"extremes":
|
||||
{
|
||||
@ -77,7 +77,7 @@
|
||||
}
|
||||
2010-01-01 1
|
||||
|
||||
2010-01-01 1
|
||||
1970-01-01 1
|
||||
|
||||
2010-01-01 1
|
||||
2010-01-01 1
|
||||
@ -104,7 +104,7 @@
|
||||
|
||||
"totals":
|
||||
{
|
||||
"k": "2010-01-01 01:02:03",
|
||||
"k": "1970-01-01 01:00:00",
|
||||
"count()": "1"
|
||||
},
|
||||
|
||||
@ -142,7 +142,7 @@
|
||||
[1.1, "1"]
|
||||
],
|
||||
|
||||
"totals": [1.1, "1"],
|
||||
"totals": [0, "1"],
|
||||
|
||||
"extremes":
|
||||
{
|
||||
|
@ -4,10 +4,10 @@ CURDIR=$(cd "$(dirname "${BASH_SOURCE[0]}")" && pwd)
|
||||
# shellcheck source=../shell_config.sh
|
||||
. "$CURDIR"/../shell_config.sh
|
||||
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT 1 AS k, count() GROUP BY k WITH TOTALS";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT 1234567890123 AS k, count() GROUP BY k WITH TOTALS FORMAT JSON";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT toFloat32(1.23) AS k, count() GROUP BY k WITH TOTALS FORMAT JSONCompact";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT 1 AS k, count() GROUP BY k WITH TOTALS SETTINGS allow_experimental_analyzer = 1";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT 1234567890123 AS k, count() GROUP BY k WITH TOTALS SETTINGS allow_experimental_analyzer = 1 FORMAT JSON";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT toFloat32(1.23) AS k, count() GROUP BY k WITH TOTALS SETTINGS allow_experimental_analyzer = 1 FORMAT JSONCompact";
|
||||
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT toDate('2010-01-01') AS k, count() GROUP BY k WITH TOTALS";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT toDateTime('2010-01-01 01:02:03') AS k, count() GROUP BY k WITH TOTALS FORMAT JSON";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT 1.1 AS k, count() GROUP BY k WITH TOTALS FORMAT JSONCompact";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT toDate('2010-01-01') AS k, count() GROUP BY k WITH TOTALS SETTINGS allow_experimental_analyzer = 1";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT toDateTime('2010-01-01 01:02:03') AS k, count() GROUP BY k WITH TOTALS SETTINGS allow_experimental_analyzer = 1 FORMAT JSON";
|
||||
${CLICKHOUSE_CURL} -sS "${CLICKHOUSE_URL}&extremes=1&output_format_write_statistics=0" -d "SELECT 1.1 AS k, count() GROUP BY k WITH TOTALS SETTINGS allow_experimental_analyzer = 1 FORMAT JSONCompact";
|
||||
|
@ -1,7 +1,7 @@
|
||||
11
|
||||
|
||||
11
|
||||
12
|
||||
0
|
||||
12
|
||||
0
|
||||
13
|
||||
13
|
||||
0
|
||||
|
@ -1,3 +1,5 @@
|
||||
SET allow_experimental_analyzer = 1;
|
||||
|
||||
SELECT 11 AS n GROUP BY n WITH TOTALS;
|
||||
SELECT 12 AS n GROUP BY n WITH ROLLUP;
|
||||
SELECT 13 AS n GROUP BY n WITH CUBE;
|
||||
|
@ -88,11 +88,11 @@ SELECT * FROM (SELECT number + 2 AS key FROM numbers(4)) s1 FULL JOIN dict_flat
|
||||
SELECT '-';
|
||||
SELECT * FROM (SELECT number AS key FROM numbers(2)) s1 ANY INNER JOIN dict_flat d USING(key) ORDER BY s1.key;
|
||||
SELECT '-';
|
||||
SELECT * FROM (SELECT number AS key FROM numbers(2)) s1 ANY RIGHT JOIN dict_flat d USING(key) ORDER BY s1.key;
|
||||
SELECT * FROM (SELECT number AS key FROM numbers(2)) s1 ANY RIGHT JOIN dict_flat d USING(key) ORDER BY key;
|
||||
SELECT '-';
|
||||
SELECT * FROM (SELECT number AS key FROM numbers(2)) s1 SEMI RIGHT JOIN dict_flat d USING(key) ORDER BY s1.key;
|
||||
SELECT '-';
|
||||
SELECT * FROM (SELECT number AS key FROM numbers(2)) s1 ANTI RIGHT JOIN dict_flat d USING(key) ORDER BY s1.key;
|
||||
SELECT * FROM (SELECT number AS key FROM numbers(2)) s1 ANTI RIGHT JOIN dict_flat d USING(key) ORDER BY key;
|
||||
|
||||
SET join_use_nulls = 0;
|
||||
|
||||
|
@ -1,3 +1,5 @@
|
||||
SET allow_experimental_analyzer = 1;
|
||||
|
||||
DROP TABLE IF EXISTS columns_transformers;
|
||||
|
||||
CREATE TABLE columns_transformers (i Int64, j Int16, k Int64) Engine=TinyLog;
|
||||
@ -17,15 +19,15 @@ SELECT a.* APPLY(toDate) EXCEPT(i, j) APPLY(any) from columns_transformers a;
|
||||
SELECT * EXCEPT STRICT i from columns_transformers;
|
||||
SELECT * EXCEPT STRICT (i, j) from columns_transformers;
|
||||
SELECT * EXCEPT STRICT i, j1 from columns_transformers; -- { serverError 47 }
|
||||
SELECT * EXCEPT STRICT(i, j1) from columns_transformers; -- { serverError 16 }
|
||||
SELECT * EXCEPT STRICT(i, j1) from columns_transformers; -- { serverError 36 }
|
||||
SELECT * REPLACE STRICT i + 1 AS i from columns_transformers;
|
||||
SELECT * REPLACE STRICT(i + 1 AS col) from columns_transformers; -- { serverError 16 }
|
||||
SELECT * REPLACE STRICT(i + 1 AS col) from columns_transformers; -- { serverError 36 }
|
||||
SELECT * REPLACE(i + 1 AS i) APPLY(sum) from columns_transformers;
|
||||
SELECT columns_transformers.* REPLACE(j + 2 AS j, i + 1 AS i) APPLY(avg) from columns_transformers;
|
||||
SELECT columns_transformers.* REPLACE(j + 1 AS j, j + 2 AS j) APPLY(avg) from columns_transformers; -- { serverError 43 }
|
||||
-- REPLACE after APPLY will not match anything
|
||||
SELECT a.* APPLY(toDate) REPLACE(i + 1 AS i) APPLY(any) from columns_transformers a;
|
||||
SELECT a.* APPLY(toDate) REPLACE STRICT(i + 1 AS i) APPLY(any) from columns_transformers a; -- { serverError 16 }
|
||||
SELECT a.* APPLY(toDate) REPLACE STRICT(i + 1 AS i) APPLY(any) from columns_transformers a; -- { serverError 36 }
|
||||
|
||||
EXPLAIN SYNTAX SELECT * APPLY(sum) from columns_transformers;
|
||||
EXPLAIN SYNTAX SELECT columns_transformers.* APPLY(avg) from columns_transformers;
|
||||
|
@ -3,7 +3,7 @@
|
||||
0.00012000000000000002
|
||||
150.16500000000002
|
||||
7.775900000000001
|
||||
56.62269
|
||||
56.622689999999984
|
||||
598.8376688440277
|
||||
299.41883695311844
|
||||
0.7485470860550345
|
||||
|
@ -1,3 +1,5 @@
|
||||
SET optimize_arithmetic_operations_in_aggregate_functions = 0;
|
||||
|
||||
SELECT toDecimal32(2, 2) * 1.2;
|
||||
SELECT toDecimal64(0.5, 2) * 20.33;
|
||||
SELECT 0.00001 * toDecimal32(12, 2);
|
||||
|
@ -1,2 +1 @@
|
||||
0
|
||||
0
|
||||
|
@ -1,2 +1 @@
|
||||
SELECT (NULL, NULL) = (8, 0) OR (NULL, NULL) = (3, 2) OR (NULL, NULL) = (0, 0) OR (NULL, NULL) = (3, 1);
|
||||
SELECT (NULL, NULL) IN ((NULL, 0), (3, 1), (3, 2), (8, 0), (NULL, NULL));
|
||||
|
@ -1,7 +1,9 @@
|
||||
SELECT dictGet(t.nest.a, concat(currentDatabase(), '.dict.dict'), 's', number) FROM numbers(5); -- { serverError 47 }
|
||||
SET allow_experimental_analyzer = 1;
|
||||
|
||||
SELECT dictGetFloat64(t.b.s, 'database_for_dict.dict1', dictGetFloat64('Ta\0', toUInt64('databas\0_for_dict.dict1databas\0_for_dict.dict1', dictGetFloat64('', '', toUInt64(1048577), toDate(NULL)), NULL), toDate(dictGetFloat64(257, 'database_for_dict.dict1database_for_dict.dict1', '', toUInt64(NULL), 2, toDate(NULL)), '2019-05-2\0')), NULL, toUInt64(dictGetFloat64('', '', toUInt64(-9223372036854775808), toDate(NULL)), NULL)); -- { serverError 47 }
|
||||
SELECT dictGet(t.nest.a, concat(currentDatabase(), '.dict.dict'), 's', number) FROM numbers(5); -- { serverError 36 }
|
||||
|
||||
SELECT NULL AND (2147483648 AND NULL) AND -2147483647, toUUID(((1048576 AND NULL) AND (2147483647 AND 257 AND NULL AND -2147483649) AND NULL) IN (test_01103.t1_distr.id), '00000000-e1fe-11e\0-bb8f\0853d60c00749'), stringToH3('89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff'); -- { serverError 47 }
|
||||
SELECT dictGetFloat64(t.b.s, 'database_for_dict.dict1', dictGetFloat64('Ta\0', toUInt64('databas\0_for_dict.dict1databas\0_for_dict.dict1', dictGetFloat64('', '', toUInt64(1048577), toDate(NULL)), NULL), toDate(dictGetFloat64(257, 'database_for_dict.dict1database_for_dict.dict1', '', toUInt64(NULL), 2, toDate(NULL)), '2019-05-2\0')), NULL, toUInt64(dictGetFloat64('', '', toUInt64(-9223372036854775808), toDate(NULL)), NULL)); -- { serverError 36 }
|
||||
|
||||
SELECT NULL AND (2147483648 AND NULL) AND -2147483647, toUUID(((1048576 AND NULL) AND (2147483647 AND 257 AND NULL AND -2147483649) AND NULL) IN (test_01103.t1_distr.id), '00000000-e1fe-11e\0-bb8f\0853d60c00749'), stringToH3('89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff89184926cc3ffff'); -- { serverError 36 }
|
||||
|
||||
SELECT 'still alive';
|
||||
|
@ -1 +0,0 @@
|
||||
1
|
@ -1,5 +1,3 @@
|
||||
WITH x AS y SELECT 1;
|
||||
|
||||
DROP TEMPORARY TABLE IF EXISTS t1;
|
||||
DROP TEMPORARY TABLE IF EXISTS t2;
|
||||
|
||||
|
@ -12,16 +12,16 @@
|
||||
523264
|
||||
1 523264
|
||||
|
||||
1 523264
|
||||
0 523264
|
||||
1 523264
|
||||
|
||||
1 523264
|
||||
0 523264
|
||||
1 523264
|
||||
|
||||
1 523264
|
||||
0 523264
|
||||
1 523264
|
||||
|
||||
1 523264
|
||||
0 523264
|
||||
-1 86614
|
||||
0 413533
|
||||
1 9984
|
||||
|
@ -1,3 +1,5 @@
|
||||
SET allow_experimental_analyzer = 1;
|
||||
|
||||
SELECT count() AS c FROM test.hits WHERE CounterID = 1704509 WITH TOTALS SETTINGS totals_mode = 'before_having', max_rows_to_group_by = 100000, group_by_overflow_mode = 'any';
|
||||
SELECT count() AS c FROM test.hits WHERE CounterID = 1704509 WITH TOTALS SETTINGS totals_mode = 'after_having_inclusive', max_rows_to_group_by = 100000, group_by_overflow_mode = 'any';
|
||||
SELECT count() AS c FROM test.hits WHERE CounterID = 1704509 WITH TOTALS SETTINGS totals_mode = 'after_having_exclusive', max_rows_to_group_by = 100000, group_by_overflow_mode = 'any';
|
||||
|
Loading…
Reference in New Issue
Block a user