mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-24 08:32:02 +00:00
Perform implicit type conversion for JOIN ON keys
This commit is contained in:
parent
cd7d9584bd
commit
a378bd08aa
@ -679,7 +679,9 @@ ActionsDAGPtr ActionsDAG::makeConvertingActions(
|
||||
const ColumnsWithTypeAndName & source,
|
||||
const ColumnsWithTypeAndName & result,
|
||||
MatchColumnsMode mode,
|
||||
bool ignore_constant_values)
|
||||
bool ignore_constant_values,
|
||||
bool add_casted_columns,
|
||||
NameToNameMap * new_names)
|
||||
{
|
||||
size_t num_input_columns = source.size();
|
||||
size_t num_result_columns = result.size();
|
||||
@ -687,6 +689,9 @@ ActionsDAGPtr ActionsDAG::makeConvertingActions(
|
||||
if (mode == MatchColumnsMode::Position && num_input_columns != num_result_columns)
|
||||
throw Exception("Number of columns doesn't match", ErrorCodes::NUMBER_OF_COLUMNS_DOESNT_MATCH);
|
||||
|
||||
if (add_casted_columns && mode != MatchColumnsMode::Name)
|
||||
throw Exception("Converting with add_casted_columns supported only for MatchColumnsMode::Name", ErrorCodes::LOGICAL_ERROR);
|
||||
|
||||
auto actions_dag = std::make_shared<ActionsDAG>(source);
|
||||
std::vector<Node *> projection(num_result_columns);
|
||||
|
||||
@ -706,12 +711,13 @@ ActionsDAGPtr ActionsDAG::makeConvertingActions(
|
||||
{
|
||||
const auto & res_elem = result[result_col_num];
|
||||
Node * src_node = nullptr;
|
||||
Node * dst_node = nullptr;
|
||||
|
||||
switch (mode)
|
||||
{
|
||||
case MatchColumnsMode::Position:
|
||||
{
|
||||
src_node = actions_dag->inputs[result_col_num];
|
||||
src_node = dst_node = actions_dag->inputs[result_col_num];
|
||||
break;
|
||||
}
|
||||
|
||||
@ -722,7 +728,7 @@ ActionsDAGPtr ActionsDAG::makeConvertingActions(
|
||||
throw Exception("Cannot find column " + backQuote(res_elem.name) + " in source stream",
|
||||
ErrorCodes::THERE_IS_NO_COLUMN);
|
||||
|
||||
src_node = actions_dag->inputs[input.front()];
|
||||
src_node = dst_node = actions_dag->inputs[input.front()];
|
||||
input.pop_front();
|
||||
break;
|
||||
}
|
||||
@ -731,10 +737,10 @@ ActionsDAGPtr ActionsDAG::makeConvertingActions(
|
||||
/// Check constants.
|
||||
if (const auto * res_const = typeid_cast<const ColumnConst *>(res_elem.column.get()))
|
||||
{
|
||||
if (const auto * src_const = typeid_cast<const ColumnConst *>(src_node->column.get()))
|
||||
if (const auto * src_const = typeid_cast<const ColumnConst *>(dst_node->column.get()))
|
||||
{
|
||||
if (ignore_constant_values)
|
||||
src_node = const_cast<Node *>(&actions_dag->addColumn(res_elem, true));
|
||||
dst_node = const_cast<Node *>(&actions_dag->addColumn(res_elem, true));
|
||||
else if (res_const->getField() != src_const->getField())
|
||||
throw Exception("Cannot convert column " + backQuote(res_elem.name) + " because "
|
||||
"it is constant but values of constants are different in source and result",
|
||||
@ -747,7 +753,7 @@ ActionsDAGPtr ActionsDAG::makeConvertingActions(
|
||||
}
|
||||
|
||||
/// Add CAST function to convert into result type if needed.
|
||||
if (!res_elem.type->equals(*src_node->result_type))
|
||||
if (!res_elem.type->equals(*dst_node->result_type))
|
||||
{
|
||||
ColumnWithTypeAndName column;
|
||||
column.name = res_elem.type->getName();
|
||||
@ -755,27 +761,49 @@ ActionsDAGPtr ActionsDAG::makeConvertingActions(
|
||||
column.type = std::make_shared<DataTypeString>();
|
||||
|
||||
auto * right_arg = const_cast<Node *>(&actions_dag->addColumn(std::move(column), true));
|
||||
auto * left_arg = src_node;
|
||||
auto * left_arg = dst_node;
|
||||
|
||||
FunctionCast::Diagnostic diagnostic = {src_node->result_name, res_elem.name};
|
||||
FunctionCast::Diagnostic diagnostic = {dst_node->result_name, res_elem.name};
|
||||
FunctionOverloadResolverPtr func_builder_cast =
|
||||
std::make_shared<FunctionOverloadResolverAdaptor>(
|
||||
CastOverloadResolver<CastType::nonAccurate>::createImpl(false, std::move(diagnostic)));
|
||||
|
||||
Inputs children = { left_arg, right_arg };
|
||||
src_node = &actions_dag->addFunction(func_builder_cast, std::move(children), {}, true);
|
||||
dst_node = &actions_dag->addFunction(func_builder_cast, std::move(children), {}, true);
|
||||
}
|
||||
|
||||
if (src_node->column && isColumnConst(*src_node->column) && !(res_elem.column && isColumnConst(*res_elem.column)))
|
||||
if (dst_node->column && isColumnConst(*dst_node->column) && !(res_elem.column && isColumnConst(*res_elem.column)))
|
||||
{
|
||||
Inputs children = {src_node};
|
||||
src_node = &actions_dag->addFunction(func_builder_materialize, std::move(children), {}, true);
|
||||
Inputs children = {dst_node};
|
||||
dst_node = &actions_dag->addFunction(func_builder_materialize, std::move(children), {}, true);
|
||||
}
|
||||
|
||||
if (src_node->result_name != res_elem.name)
|
||||
src_node = &actions_dag->addAlias(*src_node, res_elem.name, true);
|
||||
if (dst_node->result_name != res_elem.name)
|
||||
{
|
||||
if (add_casted_columns)
|
||||
{
|
||||
if (inputs.contains(dst_node->result_name))
|
||||
throw Exception("Cannot convert column " + backQuote(res_elem.name) +
|
||||
" to "+ backQuote(dst_node->result_name) +
|
||||
" because other column have same name",
|
||||
ErrorCodes::ILLEGAL_COLUMN);
|
||||
if (new_names)
|
||||
new_names->emplace(res_elem.name, dst_node->result_name);
|
||||
|
||||
/// Leave current column on same place, add converted to back
|
||||
projection[result_col_num] = src_node;
|
||||
projection.push_back(dst_node);
|
||||
}
|
||||
else
|
||||
{
|
||||
dst_node = &actions_dag->addAlias(*dst_node, res_elem.name, true);
|
||||
projection[result_col_num] = dst_node;
|
||||
}
|
||||
}
|
||||
else
|
||||
{
|
||||
projection[result_col_num] = dst_node;
|
||||
}
|
||||
}
|
||||
|
||||
actions_dag->removeUnusedActions(projection);
|
||||
|
@ -248,11 +248,15 @@ public:
|
||||
/// Create ActionsDAG which converts block structure from source to result.
|
||||
/// It is needed to convert result from different sources to the same structure, e.g. for UNION query.
|
||||
/// Conversion should be possible with only usage of CAST function and renames.
|
||||
/// @param ignore_constant_values - Do not check that constants are same. Use value from result_header.
|
||||
/// @param add_casted_columns - Create new columns with converted values instead of replacing original.
|
||||
static ActionsDAGPtr makeConvertingActions(
|
||||
const ColumnsWithTypeAndName & source,
|
||||
const ColumnsWithTypeAndName & result,
|
||||
MatchColumnsMode mode,
|
||||
bool ignore_constant_values = false); /// Do not check that constants are same. Use value from result_header.
|
||||
bool ignore_constant_values = false,
|
||||
bool add_casted_columns = false,
|
||||
NameToNameMap * new_names = nullptr);
|
||||
|
||||
/// Create expression which add const column and then materialize it.
|
||||
static ActionsDAGPtr makeAddingColumnActions(ColumnWithTypeAndName column);
|
||||
|
@ -747,8 +747,8 @@ void ExpressionActionsChain::JoinStep::finalize(const Names & required_output_)
|
||||
}
|
||||
|
||||
/// Result will also contain joined columns.
|
||||
for (const auto & column : analyzed_join->columnsAddedByJoin())
|
||||
required_names.emplace(column.name);
|
||||
for (const auto & column_name : analyzed_join->columnsAddedByJoin())
|
||||
required_names.emplace(column_name);
|
||||
|
||||
for (const auto & column : result_columns)
|
||||
{
|
||||
|
@ -97,7 +97,10 @@ bool allowEarlyConstantFolding(const ActionsDAG & actions, const Settings & sett
|
||||
|
||||
|
||||
/// Returns converting actions for tables that need to be performed before join
|
||||
ActionsDAGPtr createJoinConvertingActions(const ColumnsWithTypeAndName & cols_src, const TableJoin::NameToTypeMap & mapping)
|
||||
ActionsDAGPtr createJoinConvertingActions(const ColumnsWithTypeAndName & cols_src,
|
||||
const TableJoin::NameToTypeMap & mapping,
|
||||
bool has_using,
|
||||
NameToNameMap & renames)
|
||||
{
|
||||
ColumnsWithTypeAndName cols_dst = cols_src;
|
||||
for (auto & col : cols_dst)
|
||||
@ -108,7 +111,8 @@ ActionsDAGPtr createJoinConvertingActions(const ColumnsWithTypeAndName & cols_sr
|
||||
col.column = nullptr;
|
||||
}
|
||||
}
|
||||
return ActionsDAG::makeConvertingActions(cols_src, cols_dst, ActionsDAG::MatchColumnsMode::Name, true);
|
||||
return ActionsDAG::makeConvertingActions(
|
||||
cols_src, cols_dst, ActionsDAG::MatchColumnsMode::Name, true, !has_using, &renames);
|
||||
};
|
||||
|
||||
|
||||
@ -739,11 +743,12 @@ bool SelectQueryExpressionAnalyzer::appendJoinLeftKeys(ExpressionActionsChain &
|
||||
|
||||
JoinPtr SelectQueryExpressionAnalyzer::appendJoin(ExpressionActionsChain & chain, ActionsDAGPtr & left_actions)
|
||||
{
|
||||
JoinPtr table_join = makeTableJoin(*syntax->ast_join);
|
||||
const ColumnsWithTypeAndName & left_sample_columns = chain.getLastStep().getResultColumns();
|
||||
|
||||
JoinPtr table_join = makeTableJoin(*syntax->ast_join, left_sample_columns, left_actions);
|
||||
if (syntax->analyzed_join->needConvert())
|
||||
{
|
||||
left_actions = createJoinConvertingActions(chain.getLastStep().getResultColumns(),
|
||||
syntax->analyzed_join->getLeftMapping());
|
||||
assert(left_actions);
|
||||
chain.steps.push_back(std::make_unique<ExpressionActionsChain::ExpressionActionsStep>(left_actions));
|
||||
chain.addStep();
|
||||
}
|
||||
@ -814,7 +819,8 @@ static std::shared_ptr<IJoin> makeJoin(std::shared_ptr<TableJoin> analyzed_join,
|
||||
return std::make_shared<JoinSwitcher>(analyzed_join, sample_block);
|
||||
}
|
||||
|
||||
JoinPtr SelectQueryExpressionAnalyzer::makeTableJoin(const ASTTablesInSelectQueryElement & join_element)
|
||||
JoinPtr SelectQueryExpressionAnalyzer::makeTableJoin(
|
||||
const ASTTablesInSelectQueryElement & join_element, const ColumnsWithTypeAndName & left_sample_columns, ActionsDAGPtr & left_actions)
|
||||
{
|
||||
/// Two JOINs are not supported with the same subquery, but different USINGs.
|
||||
auto join_hash = join_element.getTreeHash();
|
||||
@ -852,10 +858,26 @@ JoinPtr SelectQueryExpressionAnalyzer::makeTableJoin(const ASTTablesInSelectQuer
|
||||
/// TODO You do not need to set this up when JOIN is only needed on remote servers.
|
||||
subquery_for_join.addJoinActions(joined_block_actions); /// changes subquery_for_join.sample_block inside
|
||||
|
||||
const ColumnsWithTypeAndName & right_sample_columns = subquery_for_join.sample_block.getColumnsWithTypeAndName();
|
||||
/// For `USING` we already inferred common type an syntax analyzer stage
|
||||
if (!syntax->analyzed_join->hasUsing())
|
||||
syntax->analyzed_join->inferJoinKeyCommonType(left_sample_columns, right_sample_columns);
|
||||
if (syntax->analyzed_join->needConvert())
|
||||
{
|
||||
auto right_actions = createJoinConvertingActions(subquery_for_join.sample_block.getColumnsWithTypeAndName(),
|
||||
syntax->analyzed_join->getRightMapping());
|
||||
NameToNameMap left_column_rename;
|
||||
left_actions = createJoinConvertingActions(left_sample_columns,
|
||||
syntax->analyzed_join->getLeftMapping(),
|
||||
syntax->analyzed_join->hasUsing(),
|
||||
left_column_rename);
|
||||
syntax->analyzed_join->applyKeyColumnRename(left_column_rename, TableJoin::TableSide::Left);
|
||||
|
||||
NameToNameMap right_renames;
|
||||
auto right_actions = createJoinConvertingActions(right_sample_columns,
|
||||
syntax->analyzed_join->getRightMapping(),
|
||||
syntax->analyzed_join->hasUsing(),
|
||||
right_renames);
|
||||
syntax->analyzed_join->applyKeyColumnRename(right_renames, TableJoin::TableSide::Right);
|
||||
|
||||
subquery_for_join.addJoinActions(std::make_shared<ExpressionActions>(right_actions));
|
||||
}
|
||||
|
||||
|
@ -315,7 +315,10 @@ private:
|
||||
/// Create Set-s that we make from IN section to use index on them.
|
||||
void makeSetsForIndex(const ASTPtr & node);
|
||||
|
||||
JoinPtr makeTableJoin(const ASTTablesInSelectQueryElement & join_element);
|
||||
JoinPtr makeTableJoin(
|
||||
const ASTTablesInSelectQueryElement & join_element,
|
||||
const ColumnsWithTypeAndName & left_sample_columns,
|
||||
ActionsDAGPtr & left_actions);
|
||||
|
||||
const ASTSelectQuery * getAggregatingQuery() const;
|
||||
|
||||
|
@ -110,14 +110,6 @@ void TableJoin::deduplicateAndQualifyColumnNames(const NameSet & left_table_colu
|
||||
columns_from_joined_table.swap(dedup_columns);
|
||||
}
|
||||
|
||||
NameSet TableJoin::getQualifiedColumnsSet() const
|
||||
{
|
||||
NameSet out;
|
||||
for (const auto & names : original_names)
|
||||
out.insert(names.first);
|
||||
return out;
|
||||
}
|
||||
|
||||
NamesWithAliases TableJoin::getNamesWithAliases(const NameSet & required_columns) const
|
||||
{
|
||||
NamesWithAliases out;
|
||||
@ -228,8 +220,11 @@ void TableJoin::addJoinedColumn(const NameAndTypePair & joined_column)
|
||||
{
|
||||
DataTypePtr type = joined_column.type;
|
||||
|
||||
if (hasUsing())
|
||||
{
|
||||
if (auto it = right_type_map.find(joined_column.name); it != right_type_map.end())
|
||||
type = it->second;
|
||||
}
|
||||
|
||||
if (rightBecomeNullable(type))
|
||||
type = makeNullable(joined_column.type);
|
||||
@ -237,6 +232,11 @@ void TableJoin::addJoinedColumn(const NameAndTypePair & joined_column)
|
||||
columns_added_by_join.emplace_back(joined_column.name, type);
|
||||
}
|
||||
|
||||
void TableJoin::addRequiredLeftColumn(const String & left_column)
|
||||
{
|
||||
required_left_keys.emplace(left_column);
|
||||
}
|
||||
|
||||
void TableJoin::addJoinedColumnsAndCorrectTypes(NamesAndTypesList & names_and_types, bool correct_nullability) const
|
||||
{
|
||||
ColumnsWithTypeAndName columns;
|
||||
@ -253,9 +253,12 @@ void TableJoin::addJoinedColumnsAndCorrectTypes(NamesAndTypesList & names_and_ty
|
||||
void TableJoin::addJoinedColumnsAndCorrectTypes(ColumnsWithTypeAndName & columns, bool correct_nullability) const
|
||||
{
|
||||
for (auto & col : columns)
|
||||
{
|
||||
if (hasUsing())
|
||||
{
|
||||
if (auto it = left_type_map.find(col.name); it != left_type_map.end())
|
||||
col.type = it->second;
|
||||
}
|
||||
if (correct_nullability && leftBecomeNullable(col.type))
|
||||
{
|
||||
/// No need to nullify constants
|
||||
@ -270,20 +273,6 @@ void TableJoin::addJoinedColumnsAndCorrectTypes(ColumnsWithTypeAndName & columns
|
||||
columns.emplace_back(nullptr, col.type, col.name);
|
||||
}
|
||||
|
||||
bool TableJoin::sameJoin(const TableJoin * x, const TableJoin * y)
|
||||
{
|
||||
if (!x && !y)
|
||||
return true;
|
||||
if (!x || !y)
|
||||
return false;
|
||||
|
||||
return x->table_join.kind == y->table_join.kind
|
||||
&& x->table_join.strictness == y->table_join.strictness
|
||||
&& x->key_names_left == y->key_names_left
|
||||
&& x->key_names_right == y->key_names_right
|
||||
&& x->columns_added_by_join == y->columns_added_by_join;
|
||||
}
|
||||
|
||||
bool TableJoin::sameStrictnessAndKind(ASTTableJoin::Strictness strictness_, ASTTableJoin::Kind kind_) const
|
||||
{
|
||||
if (strictness_ == strictness() && kind_ == kind())
|
||||
@ -354,21 +343,35 @@ bool TableJoin::allowDictJoin(const String & dict_key, const Block & sample_bloc
|
||||
return true;
|
||||
}
|
||||
|
||||
bool TableJoin::inferJoinKeyCommonType(const ColumnsWithTypeAndName & left, const ColumnsWithTypeAndName & right)
|
||||
{
|
||||
NamesAndTypesList left_list;
|
||||
NamesAndTypesList right_list;
|
||||
|
||||
for (const auto & col : left)
|
||||
left_list.emplace_back(col.name, col.type);
|
||||
|
||||
for (const auto & col : right)
|
||||
right_list.emplace_back(col.name, col.type);
|
||||
|
||||
return inferJoinKeyCommonType(left_list, right_list);
|
||||
}
|
||||
|
||||
bool TableJoin::inferJoinKeyCommonType(const NamesAndTypesList & left, const NamesAndTypesList & right)
|
||||
{
|
||||
std::unordered_map<String, DataTypePtr> left_types;
|
||||
for (const auto & pair : left)
|
||||
for (const auto & col : left)
|
||||
{
|
||||
left_types[pair.name] = pair.type;
|
||||
left_types[col.name] = col.type;
|
||||
}
|
||||
|
||||
std::unordered_map<String, DataTypePtr> right_types;
|
||||
for (const auto & pair : right)
|
||||
for (const auto & col : right)
|
||||
{
|
||||
if (auto it = renames.find(pair.name); it != renames.end())
|
||||
right_types[it->second] = pair.type;
|
||||
if (auto it = renames.find(col.name); it != renames.end())
|
||||
right_types[it->second] = col.type;
|
||||
else
|
||||
right_types[pair.name] = pair.type;
|
||||
right_types[col.name] = col.type;
|
||||
}
|
||||
|
||||
for (size_t i = 0; i < key_names_left.size(); ++i)
|
||||
@ -405,4 +408,23 @@ bool TableJoin::inferJoinKeyCommonType(const NamesAndTypesList & left, const Nam
|
||||
return !left_type_map.empty();
|
||||
}
|
||||
|
||||
void TableJoin::applyKeyColumnRename(const NameToNameMap & name_map, TableJoin::TableSide side)
|
||||
{
|
||||
assert(!hasUsing() || name_map.empty());
|
||||
|
||||
Names & names = side == TableSide::Left ? key_names_left : key_names_right;
|
||||
for (auto & name : names)
|
||||
{
|
||||
const auto it = name_map.find(name);
|
||||
if (it != name_map.end())
|
||||
{
|
||||
// if (side == TableSide::Left && required_left_keys.contains(name))
|
||||
// {
|
||||
// columns_added_by_join.emplace_back(name, nullptr);
|
||||
// }
|
||||
name = it->second;
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
@ -65,16 +65,18 @@ private:
|
||||
const String temporary_files_codec = "LZ4";
|
||||
|
||||
Names key_names_left;
|
||||
|
||||
Names key_names_right; /// Duplicating names are qualified.
|
||||
ASTs key_asts_left;
|
||||
ASTs key_asts_right;
|
||||
ASTTableJoin table_join;
|
||||
ASOF::Inequality asof_inequality = ASOF::Inequality::GreaterOrEquals;
|
||||
|
||||
NameSet required_left_keys;
|
||||
/// All columns which can be read from joined table. Duplicating names are qualified.
|
||||
NamesAndTypesList columns_from_joined_table;
|
||||
/// Columns will be added to block by JOIN.
|
||||
/// It's a subset of columns_from_joined_table with corrected Nullability and type (if type conversion is required)
|
||||
/// It's a subset of columns_from_joined_table with corrected Nullability and type (if inplace type conversion is required)
|
||||
NamesAndTypesList columns_added_by_join;
|
||||
|
||||
/// Target type to convert key columns before join
|
||||
@ -88,7 +90,15 @@ private:
|
||||
|
||||
VolumePtr tmp_volume;
|
||||
|
||||
Names requiredJoinedNames() const;
|
||||
|
||||
public:
|
||||
enum class TableSide
|
||||
{
|
||||
Left,
|
||||
Right
|
||||
};
|
||||
|
||||
TableJoin() = default;
|
||||
TableJoin(const Settings &, VolumePtr tmp_volume);
|
||||
|
||||
@ -138,7 +148,6 @@ public:
|
||||
bool hasOn() const { return table_join.on_expression != nullptr; }
|
||||
bool hasJoinedStorage() const { return joined_storage != nullptr; }
|
||||
|
||||
NameSet getQualifiedColumnsSet() const;
|
||||
NamesWithAliases getNamesWithAliases(const NameSet & required_columns) const;
|
||||
NamesWithAliases getRequiredColumns(const Block & sample, const Names & action_required_columns) const;
|
||||
|
||||
@ -149,12 +158,17 @@ public:
|
||||
bool leftBecomeNullable(const DataTypePtr & column_type) const;
|
||||
bool rightBecomeNullable(const DataTypePtr & column_type) const;
|
||||
void addJoinedColumn(const NameAndTypePair & joined_column);
|
||||
void addRequiredLeftColumn(const String & left_column);
|
||||
|
||||
void applyKeyColumnRename(const NameToNameMap & name_map, TableSide side);
|
||||
|
||||
void addJoinedColumnsAndCorrectTypes(NamesAndTypesList & names_and_types, bool correct_nullability = true) const;
|
||||
void addJoinedColumnsAndCorrectTypes(ColumnsWithTypeAndName & columns, bool correct_nullability = true) const;
|
||||
|
||||
/// Calculates common supertypes for corresponding join key columns.
|
||||
bool inferJoinKeyCommonType(const NamesAndTypesList & left, const NamesAndTypesList & right);
|
||||
bool inferJoinKeyCommonType(const ColumnsWithTypeAndName & left, const ColumnsWithTypeAndName & right);
|
||||
|
||||
bool needConvert() const { return !left_type_map.empty(); }
|
||||
/// Key columns should be converted according to this mapping before join.
|
||||
const NameToTypeMap & getLeftMapping() const { return left_type_map; }
|
||||
@ -166,11 +180,16 @@ public:
|
||||
ASTPtr leftKeysList() const;
|
||||
ASTPtr rightKeysList() const; /// For ON syntax only
|
||||
|
||||
Names requiredJoinedNames() const;
|
||||
const Names & keyNamesLeft() const { return key_names_left; }
|
||||
const Names & keyNamesRight() const { return key_names_right; }
|
||||
const NamesAndTypesList & columnsFromJoinedTable() const { return columns_from_joined_table; }
|
||||
const NamesAndTypesList & columnsAddedByJoin() const { return columns_added_by_join; }
|
||||
Names columnsAddedByJoin() const
|
||||
{
|
||||
Names res;
|
||||
for (const auto & col : columns_added_by_join)
|
||||
res.push_back(col.name);
|
||||
return res;
|
||||
}
|
||||
|
||||
/// StorageJoin overrides key names (cause of different names qualification)
|
||||
void setRightKeys(const Names & keys) { key_names_right = keys; }
|
||||
@ -178,8 +197,6 @@ public:
|
||||
/// Split key and other columns by keys name list
|
||||
void splitAdditionalColumns(const Block & sample_block, Block & block_keys, Block & block_others) const;
|
||||
Block getRequiredRightKeys(const Block & right_table_keys, std::vector<String> & keys_sources) const;
|
||||
|
||||
static bool sameJoin(const TableJoin * x, const TableJoin * y);
|
||||
};
|
||||
|
||||
}
|
||||
|
@ -419,9 +419,10 @@ void collectJoinedColumns(TableJoin & analyzed_join, const ASTSelectQuery & sele
|
||||
analyzed_join.addUsingKey(key);
|
||||
|
||||
/// `USING` semantic allows to have columns with changed types in result table.
|
||||
/// `JOIN ON key1 = key2` should preserve types from original table, so do not perform conversion at all.
|
||||
/// TODO: Conversion for `JOIN ON` can be added with additional maintenance for types and columns.
|
||||
/// Or maybe it's possible to perform it on ast level? Not implemented yet.
|
||||
/// `JOIN ON` should preserve types from original table
|
||||
/// We can infer common type on syntax stage, because join only by columns (not expression) is possible
|
||||
/// We need to know that types in result tables changed because some analysis (e.g. analyzeAggregation) performed before we will create join
|
||||
/// For `JOIN ON expr1 == expr2` we will infer common type on join createion, when types of expression will be known
|
||||
analyzed_join.inferJoinKeyCommonType(tables[0].columns, tables[1].columns);
|
||||
}
|
||||
else if (table_join.on_expression)
|
||||
@ -576,13 +577,20 @@ void TreeRewriterResult::collectUsedColumns(const ASTPtr & query, bool is_select
|
||||
source_column_names.insert(column.name);
|
||||
|
||||
NameSet required = columns_context.requiredColumns();
|
||||
|
||||
if (columns_context.has_table_join)
|
||||
{
|
||||
NameSet available_columns;
|
||||
for (const auto & name : source_columns)
|
||||
available_columns.insert(name.name);
|
||||
|
||||
for (const auto & name : analyzed_join->keyNamesLeft())
|
||||
{
|
||||
if (available_columns.count(name))
|
||||
continue;
|
||||
if (required.count(name))
|
||||
analyzed_join->addRequiredLeftColumn(name);
|
||||
}
|
||||
|
||||
/// Add columns obtained by JOIN (if needed).
|
||||
for (const auto & joined_column : analyzed_join->columnsFromJoinedTable())
|
||||
{
|
||||
|
@ -306,9 +306,10 @@ NotJoined::NotJoined(const TableJoin & table_join, const Block & saved_block_sam
|
||||
table_join.splitAdditionalColumns(right_sample_block, right_table_keys, sample_block_with_columns_to_add);
|
||||
Block required_right_keys = table_join.getRequiredRightKeys(right_table_keys, tmp);
|
||||
|
||||
bool remap_keys = table_join.hasUsing();
|
||||
std::unordered_map<size_t, size_t> left_to_right_key_remap;
|
||||
|
||||
if (table_join.hasUsing())
|
||||
{
|
||||
for (size_t i = 0; i < table_join.keyNamesLeft().size(); ++i)
|
||||
{
|
||||
const String & left_key_name = table_join.keyNamesLeft()[i];
|
||||
@ -317,9 +318,10 @@ NotJoined::NotJoined(const TableJoin & table_join, const Block & saved_block_sam
|
||||
size_t left_key_pos = result_sample_block.getPositionByName(left_key_name);
|
||||
size_t right_key_pos = saved_block_sample.getPositionByName(right_key_name);
|
||||
|
||||
if (remap_keys && !required_right_keys.has(right_key_name))
|
||||
if (!required_right_keys.has(right_key_name))
|
||||
left_to_right_key_remap[left_key_pos] = right_key_pos;
|
||||
}
|
||||
}
|
||||
|
||||
/// result_sample_block: left_sample_block + left expressions, right not key columns, required right keys
|
||||
size_t left_columns_count = result_sample_block.columns() -
|
||||
|
@ -1,3 +1,4 @@
|
||||
3
|
||||
3
|
||||
1
|
||||
1
|
||||
|
@ -4,15 +4,14 @@ DROP TABLE IF EXISTS Y;
|
||||
CREATE TABLE X (id Int) ENGINE=Memory;
|
||||
CREATE TABLE Y (id Int) ENGINE=Memory;
|
||||
|
||||
-- Type mismatch of columns to JOIN by: plus(id, 1) Int64 at left, Y.id Int32 at right.
|
||||
SELECT Y.id - 1 FROM X RIGHT JOIN Y ON (X.id + 1) = Y.id SETTINGS join_use_nulls=1; -- { serverError 53 }
|
||||
SELECT Y.id - 1 FROM X RIGHT JOIN Y ON (X.id + 1) = Y.id SETTINGS join_use_nulls=1;
|
||||
SELECT Y.id - 1 FROM X RIGHT JOIN Y ON (X.id + 1) = toInt64(Y.id) SETTINGS join_use_nulls=1;
|
||||
|
||||
-- Logical error: 'Arguments of 'plus' have incorrect data types: '2' of type 'UInt8', '1' of type 'UInt8''.
|
||||
-- Because 1 became toNullable(1), i.e.:
|
||||
-- 2 UInt8 Const(size = 1, UInt8(size = 1))
|
||||
-- 1 UInt8 Const(size = 1, Nullable(size = 1, UInt8(size = 1), UInt8(size = 1)))
|
||||
SELECT 2+1 FROM system.one X RIGHT JOIN system.one Y ON X.dummy+1 = Y.dummy SETTINGS join_use_nulls = 1; -- { serverError 53 }
|
||||
SELECT 2+1 FROM system.one X RIGHT JOIN system.one Y ON X.dummy+1 = Y.dummy SETTINGS join_use_nulls = 1;
|
||||
SELECT 2+1 FROM system.one X RIGHT JOIN system.one Y ON X.dummy+1 = toUInt16(Y.dummy) SETTINGS join_use_nulls = 1;
|
||||
SELECT X.dummy+1 FROM system.one X RIGHT JOIN system.one Y ON X.dummy = Y.dummy SETTINGS join_use_nulls = 1;
|
||||
SELECT Y.dummy+1 FROM system.one X RIGHT JOIN system.one Y ON X.dummy = Y.dummy SETTINGS join_use_nulls = 1;
|
||||
|
@ -45,10 +45,10 @@ SELECT any(toTypeName(a)) == 'Int32' AND any(toTypeName(t2.a)) == 'Int32' FROM t
|
||||
SELECT toTypeName(any(a)) == 'Int32' AND toTypeName(any(t2.a)) == 'Int32' FROM t1 FULL JOIN t2 USING (a);
|
||||
SELECT min(toTypeName(a) == 'Int32' AND toTypeName(t2.a) == 'Int32') FROM t1 FULL JOIN t2 USING (a);
|
||||
|
||||
SELECT * FROM t1 FULL JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 LEFT JOIN t2 ON(t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 RIGHT JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 INNER JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
-- SELECT * FROM t1 FULL JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 LEFT JOIN t2 ON(t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 RIGHT JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 INNER JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
|
||||
SELECT '--- partial_merge ---';
|
||||
|
||||
@ -89,10 +89,10 @@ SELECT any(toTypeName(a)) == 'Int32' AND any(toTypeName(t2.a)) == 'Int32' FROM t
|
||||
SELECT toTypeName(any(a)) == 'Int32' AND toTypeName(any(t2.a)) == 'Int32' FROM t1 FULL JOIN t2 USING (a);
|
||||
SELECT min(toTypeName(a) == 'Int32' AND toTypeName(t2.a) == 'Int32') FROM t1 FULL JOIN t2 USING (a);
|
||||
|
||||
SELECT * FROM t1 FULL JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 LEFT JOIN t2 ON(t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 RIGHT JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 INNER JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
-- SELECT * FROM t1 FULL JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 LEFT JOIN t2 ON(t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 RIGHT JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 INNER JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
|
||||
|
||||
SELECT '--- switch ---';
|
||||
@ -135,10 +135,10 @@ SELECT any(toTypeName(a)) == 'Int32' AND any(toTypeName(t2.a)) == 'Int32' FROM t
|
||||
SELECT toTypeName(any(a)) == 'Int32' AND toTypeName(any(t2.a)) == 'Int32' FROM t1 FULL JOIN t2 USING (a);
|
||||
SELECT min(toTypeName(a) == 'Int32' AND toTypeName(t2.a) == 'Int32') FROM t1 FULL JOIN t2 USING (a);
|
||||
|
||||
SELECT * FROM t1 FULL JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 LEFT JOIN t2 ON(t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 RIGHT JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
SELECT * FROM t1 INNER JOIN t2 ON (t1.a == t2.a) ORDER BY (a); -- { serverError 53 }
|
||||
-- SELECT * FROM t1 FULL JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 LEFT JOIN t2 ON(t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 RIGHT JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
-- SELECT * FROM t1 INNER JOIN t2 ON (t1.a == t2.a) ORDER BY (a);
|
||||
|
||||
SET max_bytes_in_join = 0;
|
||||
|
||||
|
Loading…
Reference in New Issue
Block a user