2018-11-02 18:53:23 +00:00
|
|
|
#pragma once
|
|
|
|
|
|
|
|
#include <Core/Names.h>
|
|
|
|
#include <Core/NamesAndTypes.h>
|
2020-07-16 22:01:08 +00:00
|
|
|
#include <Core/SettingsEnums.h>
|
2019-09-02 19:58:45 +00:00
|
|
|
#include <Parsers/ASTTablesInSelectQuery.h>
|
2019-09-09 19:43:37 +00:00
|
|
|
#include <Interpreters/IJoin.h>
|
2022-08-04 15:20:19 +00:00
|
|
|
#include <Interpreters/JoinUtils.h>
|
2021-10-15 20:18:20 +00:00
|
|
|
#include <QueryPipeline/SizeLimits.h>
|
2021-02-09 13:17:42 +00:00
|
|
|
#include <DataTypes/getLeastSupertype.h>
|
2022-08-04 15:39:28 +00:00
|
|
|
#include <Interpreters/IKeyValueEntity.h>
|
2022-01-21 05:36:36 +00:00
|
|
|
|
2021-09-09 09:47:08 +00:00
|
|
|
#include <Common/Exception.h>
|
2021-09-15 11:16:10 +00:00
|
|
|
#include <Parsers/IAST_fwd.h>
|
|
|
|
|
|
|
|
#include <cstddef>
|
|
|
|
#include <unordered_map>
|
2018-11-02 18:53:23 +00:00
|
|
|
|
|
|
|
#include <utility>
|
|
|
|
#include <memory>
|
2021-10-02 07:13:14 +00:00
|
|
|
#include <base/types.h>
|
2021-06-11 14:47:30 +00:00
|
|
|
|
2018-11-02 18:53:23 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
class Context;
|
|
|
|
class ASTSelectQuery;
|
2019-02-13 19:00:52 +00:00
|
|
|
struct DatabaseAndTableWithAlias;
|
2019-09-02 19:58:45 +00:00
|
|
|
class Block;
|
2022-07-07 12:26:34 +00:00
|
|
|
class DictionaryJoinAdapter;
|
2021-06-29 09:22:53 +00:00
|
|
|
class StorageJoin;
|
|
|
|
class StorageDictionary;
|
2022-08-04 15:39:28 +00:00
|
|
|
class IKeyValueEntity;
|
2020-09-08 11:06:36 +00:00
|
|
|
|
2020-09-08 11:07:26 +00:00
|
|
|
struct ColumnWithTypeAndName;
|
2020-09-08 11:06:36 +00:00
|
|
|
using ColumnsWithTypeAndName = std::vector<ColumnWithTypeAndName>;
|
2019-09-02 19:58:45 +00:00
|
|
|
|
2019-09-09 19:43:37 +00:00
|
|
|
struct Settings;
|
2018-11-02 18:53:23 +00:00
|
|
|
|
2020-07-08 14:25:23 +00:00
|
|
|
class IVolume;
|
|
|
|
using VolumePtr = std::shared_ptr<IVolume>;
|
2020-01-19 14:26:28 +00:00
|
|
|
|
2020-04-07 09:48:47 +00:00
|
|
|
class TableJoin
|
2018-11-02 18:53:23 +00:00
|
|
|
{
|
2021-02-09 13:17:42 +00:00
|
|
|
public:
|
|
|
|
using NameToTypeMap = std::unordered_map<String, DataTypePtr>;
|
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
/// Corresponds to one disjunct
|
|
|
|
struct JoinOnClause
|
|
|
|
{
|
|
|
|
Names key_names_left;
|
2022-03-21 15:01:34 +00:00
|
|
|
Names key_names_right; /// Duplicating right key names are qualified
|
2021-09-06 10:59:18 +00:00
|
|
|
|
2023-08-23 15:46:56 +00:00
|
|
|
/** JOIN ON a1 == a2 AND b1 <=> b2 AND c1 == c2 AND d1 <=> d2
|
|
|
|
* key_names_left: [a1, b1, c1, d1]
|
|
|
|
* key_names_right: [a2, b2, c2, d2]
|
|
|
|
* nullsafe_compare_key_indexes: {1, 3}
|
|
|
|
*/
|
|
|
|
std::unordered_set<size_t> nullsafe_compare_key_indexes;
|
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
ASTPtr on_filter_condition_left;
|
|
|
|
ASTPtr on_filter_condition_right;
|
|
|
|
|
2022-08-15 16:34:10 +00:00
|
|
|
std::string analyzer_left_filter_condition_column_name;
|
|
|
|
std::string analyzer_right_filter_condition_column_name;
|
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
JoinOnClause() = default;
|
|
|
|
|
2023-08-28 14:03:10 +00:00
|
|
|
void addKey(const String & left_name, const String & right_name, bool null_safe_comparison)
|
2023-08-23 15:46:56 +00:00
|
|
|
{
|
|
|
|
key_names_left.push_back(left_name);
|
|
|
|
key_names_right.push_back(right_name);
|
|
|
|
if (null_safe_comparison)
|
|
|
|
nullsafe_compare_key_indexes.insert(key_names_left.size() - 1);
|
|
|
|
}
|
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
std::pair<String, String> condColumnNames() const
|
|
|
|
{
|
|
|
|
std::pair<String, String> res;
|
2022-08-15 16:34:10 +00:00
|
|
|
|
|
|
|
if (!analyzer_left_filter_condition_column_name.empty())
|
|
|
|
res.first = analyzer_left_filter_condition_column_name;
|
|
|
|
|
|
|
|
if (!analyzer_right_filter_condition_column_name.empty())
|
|
|
|
res.second = analyzer_right_filter_condition_column_name;
|
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
if (on_filter_condition_left)
|
|
|
|
res.first = on_filter_condition_left->getColumnName();
|
|
|
|
if (on_filter_condition_right)
|
|
|
|
res.second = on_filter_condition_right->getColumnName();
|
2022-08-15 16:34:10 +00:00
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
return res;
|
|
|
|
}
|
|
|
|
|
|
|
|
size_t keysCount() const
|
|
|
|
{
|
|
|
|
assert(key_names_left.size() == key_names_right.size());
|
|
|
|
return key_names_right.size();
|
|
|
|
}
|
2021-09-09 09:47:08 +00:00
|
|
|
|
2022-07-27 13:35:22 +00:00
|
|
|
String formatDebug(bool short_format = false) const
|
2021-09-09 09:47:08 +00:00
|
|
|
{
|
2022-07-27 13:35:22 +00:00
|
|
|
const auto & [left_cond, right_cond] = condColumnNames();
|
|
|
|
|
|
|
|
if (short_format)
|
|
|
|
{
|
|
|
|
return fmt::format("({}) = ({}){}{}", fmt::join(key_names_left, ", "), fmt::join(key_names_right, ", "),
|
|
|
|
!left_cond.empty() ? " AND " + left_cond : "", !right_cond.empty() ? " AND " + right_cond : "");
|
|
|
|
}
|
|
|
|
|
|
|
|
return fmt::format(
|
|
|
|
"Left keys: [{}] Right keys [{}] Condition columns: '{}', '{}'",
|
|
|
|
fmt::join(key_names_left, ", "), fmt::join(key_names_right, ", "), left_cond, right_cond);
|
2021-09-09 09:47:08 +00:00
|
|
|
}
|
2021-09-06 10:59:18 +00:00
|
|
|
};
|
|
|
|
|
2021-09-15 11:16:10 +00:00
|
|
|
using Clauses = std::vector<JoinOnClause>;
|
|
|
|
|
2022-07-27 13:35:22 +00:00
|
|
|
static std::string formatClauses(const Clauses & clauses, bool short_format = false)
|
|
|
|
{
|
|
|
|
std::vector<std::string> res;
|
|
|
|
for (const auto & clause : clauses)
|
|
|
|
res.push_back("[" + clause.formatDebug(short_format) + "]");
|
|
|
|
return fmt::format("{}", fmt::join(res, "; "));
|
|
|
|
}
|
|
|
|
|
2021-02-09 13:17:42 +00:00
|
|
|
private:
|
2018-11-02 18:53:23 +00:00
|
|
|
/** Query of the form `SELECT expr(x) AS k FROM t1 ANY LEFT JOIN (SELECT expr(x) AS k FROM t2) USING k`
|
|
|
|
* The join is made by column k.
|
|
|
|
* During the JOIN,
|
|
|
|
* - in the "right" table, it will be available by alias `k`, since `Project` action for the subquery was executed.
|
|
|
|
* - in the "left" table, it will be accessible by the name `expr(x)`, since `Project` action has not been executed yet.
|
|
|
|
* You must remember both of these options.
|
|
|
|
*
|
|
|
|
* Query of the form `SELECT ... from t1 ANY LEFT JOIN (SELECT ... from t2) ON expr(t1 columns) = expr(t2 columns)`
|
|
|
|
* to the subquery will be added expression `expr(t2 columns)`.
|
|
|
|
* It's possible to use name `expr(t2 columns)`.
|
|
|
|
*/
|
2022-03-13 11:59:20 +00:00
|
|
|
SizeLimits size_limits;
|
2021-03-05 14:34:43 +00:00
|
|
|
const size_t default_max_bytes = 0;
|
|
|
|
const bool join_use_nulls = false;
|
|
|
|
const size_t max_joined_block_rows = 0;
|
2022-03-17 12:51:44 +00:00
|
|
|
MultiEnum<JoinAlgorithm> join_algorithm = MultiEnum<JoinAlgorithm>(JoinAlgorithm::AUTO);
|
2021-03-05 14:34:43 +00:00
|
|
|
const size_t partial_merge_join_rows_in_right_blocks = 0;
|
|
|
|
const size_t partial_merge_join_left_table_buffer_bytes = 0;
|
|
|
|
const size_t max_files_to_merge = 0;
|
|
|
|
const String temporary_files_codec = "LZ4";
|
2019-09-09 19:43:37 +00:00
|
|
|
|
2023-09-14 12:03:30 +00:00
|
|
|
/// Value if setting max_memory_usage for query, can be used when max_bytes_in_join is not specified.
|
|
|
|
size_t max_memory_usage = 0;
|
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
ASTs key_asts_left;
|
|
|
|
ASTs key_asts_right;
|
2021-09-02 11:40:04 +00:00
|
|
|
|
2021-09-15 11:16:10 +00:00
|
|
|
Clauses clauses;
|
|
|
|
|
2021-03-05 14:34:43 +00:00
|
|
|
ASTTableJoin table_join;
|
2021-03-05 14:41:39 +00:00
|
|
|
|
2022-07-29 16:30:50 +00:00
|
|
|
ASOFJoinInequality asof_inequality = ASOFJoinInequality::GreaterOrEquals;
|
2018-11-02 18:53:23 +00:00
|
|
|
|
|
|
|
/// All columns which can be read from joined table. Duplicating names are qualified.
|
2019-07-30 18:39:37 +00:00
|
|
|
NamesAndTypesList columns_from_joined_table;
|
2021-02-09 13:17:42 +00:00
|
|
|
/// Columns will be added to block by JOIN.
|
2021-06-28 13:44:19 +00:00
|
|
|
/// It's a subset of columns_from_joined_table
|
|
|
|
/// Note: without corrected Nullability or type, see correctedColumnsAddedByJoin
|
2019-09-02 19:58:45 +00:00
|
|
|
NamesAndTypesList columns_added_by_join;
|
2021-02-09 13:17:42 +00:00
|
|
|
|
|
|
|
/// Target type to convert key columns before join
|
|
|
|
NameToTypeMap left_type_map;
|
|
|
|
NameToTypeMap right_type_map;
|
2019-09-02 19:58:45 +00:00
|
|
|
|
2019-07-30 18:39:37 +00:00
|
|
|
/// Name -> original name. Names are the same as in columns_from_joined_table list.
|
|
|
|
std::unordered_map<String, String> original_names;
|
2021-02-01 13:53:54 +00:00
|
|
|
/// Original name -> name. Only renamed columns.
|
2019-07-30 18:39:37 +00:00
|
|
|
std::unordered_map<String, String> renames;
|
|
|
|
|
2023-04-25 11:28:54 +00:00
|
|
|
/// Map column name to actual key name that can be an alias.
|
|
|
|
/// Example: SELECT r.id as rid from t JOIN r ON t.id = rid
|
|
|
|
/// Map: r.id -> rid
|
|
|
|
/// Required only for StorageJoin to map join keys back to original column names.
|
|
|
|
/// (workaround for ExpressionAnalyzer)
|
|
|
|
std::unordered_map<String, String> right_key_aliases;
|
|
|
|
|
2020-07-08 14:25:23 +00:00
|
|
|
VolumePtr tmp_volume;
|
2019-10-15 16:31:49 +00:00
|
|
|
|
2021-06-29 09:22:53 +00:00
|
|
|
std::shared_ptr<StorageJoin> right_storage_join;
|
|
|
|
|
2022-08-08 10:58:28 +00:00
|
|
|
std::shared_ptr<const IKeyValueEntity> right_kv_storage;
|
2022-01-21 05:36:36 +00:00
|
|
|
|
2022-07-07 12:26:34 +00:00
|
|
|
std::string right_storage_name;
|
|
|
|
|
2022-09-28 11:20:24 +00:00
|
|
|
bool is_join_with_constant = false;
|
|
|
|
|
2021-02-18 11:49:32 +00:00
|
|
|
Names requiredJoinedNames() const;
|
|
|
|
|
2021-03-05 14:34:43 +00:00
|
|
|
/// Create converting actions and change key column names if required
|
|
|
|
ActionsDAGPtr applyKeyConvertToTable(
|
2023-08-23 15:46:56 +00:00
|
|
|
const ColumnsWithTypeAndName & cols_src,
|
|
|
|
const NameToTypeMap & type_mapping,
|
|
|
|
JoinTableSide table_side,
|
|
|
|
NameToNameMap & key_column_rename);
|
|
|
|
|
|
|
|
ActionsDAGPtr applyNullsafeWrapper(
|
|
|
|
const ColumnsWithTypeAndName & cols_src,
|
|
|
|
const NameSet & columns_for_nullsafe_comparison,
|
|
|
|
JoinTableSide table_side,
|
|
|
|
NameToNameMap & key_column_rename);
|
2021-03-05 14:34:43 +00:00
|
|
|
|
2023-08-23 15:46:56 +00:00
|
|
|
ActionsDAGPtr applyJoinUseNullsConversion(
|
|
|
|
const ColumnsWithTypeAndName & cols_src,
|
|
|
|
const NameToNameMap & key_column_rename);
|
|
|
|
|
|
|
|
void applyRename(JoinTableSide side, const NameToNameMap & name_map);
|
|
|
|
|
|
|
|
void addKey(const String & left_name, const String & right_name, const ASTPtr & left_ast, const ASTPtr & right_ast, bool null_safe_comparison = false);
|
2021-09-06 10:59:18 +00:00
|
|
|
|
2022-09-06 15:25:52 +00:00
|
|
|
void assertHasOneOnExpr() const;
|
2021-06-25 12:03:10 +00:00
|
|
|
|
2021-09-15 11:16:10 +00:00
|
|
|
/// Calculates common supertypes for corresponding join key columns.
|
|
|
|
template <typename LeftNamesAndTypes, typename RightNamesAndTypes>
|
2022-05-02 14:00:24 +00:00
|
|
|
void inferJoinKeyCommonType(const LeftNamesAndTypes & left, const RightNamesAndTypes & right, bool allow_right, bool strict);
|
2021-09-15 11:16:10 +00:00
|
|
|
|
2022-08-15 16:34:10 +00:00
|
|
|
void deduplicateAndQualifyColumnNames(const NameSet & left_table_columns, const String & right_table_prefix);
|
|
|
|
|
2023-08-23 15:46:56 +00:00
|
|
|
std::pair<NameSet, NameSet>
|
|
|
|
getKeysForNullSafeComparion(const ColumnsWithTypeAndName & left_sample_columns, const ColumnsWithTypeAndName & right_sample_columns);
|
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
public:
|
2021-09-24 16:21:05 +00:00
|
|
|
TableJoin() = default;
|
2021-09-06 10:59:18 +00:00
|
|
|
|
2021-06-25 12:03:10 +00:00
|
|
|
TableJoin(const Settings & settings, VolumePtr tmp_volume_);
|
2021-03-05 14:34:43 +00:00
|
|
|
|
|
|
|
/// for StorageJoin
|
2022-07-29 16:30:50 +00:00
|
|
|
TableJoin(SizeLimits limits, bool use_nulls, JoinKind kind, JoinStrictness strictness,
|
2021-09-02 11:40:04 +00:00
|
|
|
const Names & key_names_right)
|
2021-03-05 14:34:43 +00:00
|
|
|
: size_limits(limits)
|
|
|
|
, default_max_bytes(0)
|
|
|
|
, join_use_nulls(use_nulls)
|
2023-01-16 12:33:45 +00:00
|
|
|
, join_algorithm(JoinAlgorithm::DEFAULT)
|
2021-03-05 14:34:43 +00:00
|
|
|
{
|
2021-09-10 14:52:44 +00:00
|
|
|
clauses.emplace_back().key_names_right = key_names_right;
|
2021-03-05 14:34:43 +00:00
|
|
|
table_join.kind = kind;
|
|
|
|
table_join.strictness = strictness;
|
|
|
|
}
|
2019-09-09 19:43:37 +00:00
|
|
|
|
2022-07-29 16:30:50 +00:00
|
|
|
JoinKind kind() const { return table_join.kind; }
|
|
|
|
JoinStrictness strictness() const { return table_join.strictness; }
|
|
|
|
bool sameStrictnessAndKind(JoinStrictness, JoinKind) const;
|
2021-03-05 14:34:43 +00:00
|
|
|
const SizeLimits & sizeLimits() const { return size_limits; }
|
2023-09-14 12:03:30 +00:00
|
|
|
size_t getMaxMemoryUsage() const;
|
|
|
|
|
2023-04-05 13:42:13 +00:00
|
|
|
VolumePtr getGlobalTemporaryVolume() { return tmp_volume; }
|
2022-03-17 12:51:44 +00:00
|
|
|
|
2023-06-08 17:56:35 +00:00
|
|
|
ActionsDAGPtr createJoinedBlockActions(ContextPtr context) const;
|
2023-06-08 15:30:23 +00:00
|
|
|
|
2022-07-15 14:57:58 +00:00
|
|
|
bool isEnabledAlgorithm(JoinAlgorithm val) const
|
2021-04-20 09:52:52 +00:00
|
|
|
{
|
2022-07-15 14:57:58 +00:00
|
|
|
/// When join_algorithm = 'default' (not specified by user) we use hash or direct algorithm.
|
|
|
|
/// It's behaviour that was initially supported by clickhouse.
|
2023-07-18 05:51:55 +00:00
|
|
|
bool is_enabled_by_default = val == JoinAlgorithm::DEFAULT
|
2022-07-15 14:57:58 +00:00
|
|
|
|| val == JoinAlgorithm::HASH
|
|
|
|
|| val == JoinAlgorithm::DIRECT;
|
2023-07-18 05:51:55 +00:00
|
|
|
if (join_algorithm.isSet(JoinAlgorithm::DEFAULT) && is_enabled_by_default)
|
2022-07-15 14:57:58 +00:00
|
|
|
return true;
|
|
|
|
return join_algorithm.isSet(val);
|
2021-04-20 09:52:52 +00:00
|
|
|
}
|
2021-03-05 14:34:43 +00:00
|
|
|
|
2022-07-15 14:57:58 +00:00
|
|
|
bool allowParallelHashJoin() const;
|
|
|
|
|
2022-05-02 15:54:42 +00:00
|
|
|
bool joinUseNulls() const { return join_use_nulls; }
|
2023-06-06 19:03:23 +00:00
|
|
|
|
|
|
|
bool forceNullableRight() const
|
|
|
|
{
|
2023-06-07 13:25:06 +00:00
|
|
|
return join_use_nulls && isLeftOrFull(kind());
|
2023-06-06 19:03:23 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
bool forceNullableLeft() const
|
|
|
|
{
|
2023-06-07 13:25:06 +00:00
|
|
|
return join_use_nulls && isRightOrFull(kind());
|
2023-06-06 19:03:23 +00:00
|
|
|
}
|
|
|
|
|
2021-03-05 14:34:43 +00:00
|
|
|
size_t defaultMaxBytes() const { return default_max_bytes; }
|
|
|
|
size_t maxJoinedBlockRows() const { return max_joined_block_rows; }
|
|
|
|
size_t maxRowsInRightBlock() const { return partial_merge_join_rows_in_right_blocks; }
|
|
|
|
size_t maxBytesInLeftBuffer() const { return partial_merge_join_left_table_buffer_bytes; }
|
|
|
|
size_t maxFilesToMerge() const { return max_files_to_merge; }
|
|
|
|
const String & temporaryFilesCodec() const { return temporary_files_codec; }
|
2020-07-10 18:10:06 +00:00
|
|
|
bool needStreamWithNonJoinedRows() const;
|
2019-09-09 19:43:37 +00:00
|
|
|
|
2021-09-06 10:59:18 +00:00
|
|
|
bool oneDisjunct() const;
|
|
|
|
|
2022-08-15 16:34:10 +00:00
|
|
|
ASTTableJoin & getTableJoin() { return table_join; }
|
|
|
|
const ASTTableJoin & getTableJoin() const { return table_join; }
|
|
|
|
|
2022-09-06 15:25:52 +00:00
|
|
|
JoinOnClause & getOnlyClause() { assertHasOneOnExpr(); return clauses[0]; }
|
|
|
|
const JoinOnClause & getOnlyClause() const { assertHasOneOnExpr(); return clauses[0]; }
|
2021-09-06 10:59:18 +00:00
|
|
|
|
|
|
|
std::vector<JoinOnClause> & getClauses() { return clauses; }
|
|
|
|
const std::vector<JoinOnClause> & getClauses() const { return clauses; }
|
|
|
|
|
|
|
|
Names getAllNames(JoinTableSide side) const;
|
|
|
|
|
2021-03-05 14:34:43 +00:00
|
|
|
void resetCollected();
|
2019-02-13 15:18:02 +00:00
|
|
|
void addUsingKey(const ASTPtr & ast);
|
2021-09-27 12:57:26 +00:00
|
|
|
|
2021-09-24 16:21:05 +00:00
|
|
|
void addDisjunct();
|
2021-09-27 12:57:26 +00:00
|
|
|
|
2023-08-23 15:46:56 +00:00
|
|
|
void addOnKeys(ASTPtr & left_table_ast, ASTPtr & right_table_ast, bool null_safe_comparison);
|
2019-02-06 16:44:47 +00:00
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
/* Conditions for left/right table from JOIN ON section.
|
|
|
|
*
|
|
|
|
* Conditions for left and right tables stored separately and united with 'and' function into one column.
|
|
|
|
* For example for query:
|
|
|
|
* SELECT ... JOIN ... ON t1.id == t2.id AND expr11(t1) AND expr21(t2) AND expr12(t1) AND expr22(t2)
|
|
|
|
*
|
|
|
|
* We will build two new ASTs: `expr11(t1) AND expr12(t1)`, `expr21(t2) AND expr22(t2)`
|
|
|
|
* Such columns will be added and calculated for left and right tables respectively.
|
|
|
|
* Only rows where conditions are met (where new columns have non-zero value) will be joined.
|
|
|
|
*
|
|
|
|
* NOTE: non-equi condition containing columns from different tables (like `... ON t1.id = t2.id AND t1.val > t2.val)
|
|
|
|
* doesn't supported yet, it can be added later.
|
|
|
|
*/
|
|
|
|
void addJoinCondition(const ASTPtr & ast, bool is_left);
|
|
|
|
|
2021-03-05 14:34:43 +00:00
|
|
|
bool hasUsing() const { return table_join.using_expression_list != nullptr; }
|
|
|
|
bool hasOn() const { return table_join.on_expression != nullptr; }
|
2019-01-30 15:51:39 +00:00
|
|
|
|
2022-03-21 15:01:34 +00:00
|
|
|
String getOriginalName(const String & column_name) const;
|
2019-09-04 16:20:02 +00:00
|
|
|
NamesWithAliases getNamesWithAliases(const NameSet & required_columns) const;
|
2020-03-08 23:48:08 +00:00
|
|
|
NamesWithAliases getRequiredColumns(const Block & sample, const Names & action_required_columns) const;
|
2018-11-02 18:53:23 +00:00
|
|
|
|
2019-05-13 18:58:15 +00:00
|
|
|
size_t rightKeyInclusion(const String & name) const;
|
2019-09-11 15:57:09 +00:00
|
|
|
NameSet requiredRightKeys() const;
|
2019-09-02 19:58:45 +00:00
|
|
|
|
2022-09-28 11:20:24 +00:00
|
|
|
bool isJoinWithConstant() const
|
|
|
|
{
|
|
|
|
return is_join_with_constant;
|
|
|
|
}
|
|
|
|
|
|
|
|
void setIsJoinWithConstant(bool is_join_with_constant_value)
|
|
|
|
{
|
|
|
|
is_join_with_constant = is_join_with_constant_value;
|
|
|
|
}
|
|
|
|
|
2021-03-05 14:34:43 +00:00
|
|
|
bool leftBecomeNullable(const DataTypePtr & column_type) const;
|
|
|
|
bool rightBecomeNullable(const DataTypePtr & column_type) const;
|
2019-09-02 19:58:45 +00:00
|
|
|
void addJoinedColumn(const NameAndTypePair & joined_column);
|
2022-09-06 15:25:52 +00:00
|
|
|
void setColumnsAddedByJoin(const NamesAndTypesList & columns_added_by_join_value)
|
|
|
|
{
|
|
|
|
columns_added_by_join = columns_added_by_join_value;
|
|
|
|
}
|
2021-02-18 11:49:32 +00:00
|
|
|
|
2022-05-19 18:47:26 +00:00
|
|
|
template <typename TColumns>
|
|
|
|
void addJoinedColumnsAndCorrectTypesImpl(TColumns & left_columns, bool correct_nullability);
|
|
|
|
|
2021-06-28 13:44:19 +00:00
|
|
|
void addJoinedColumnsAndCorrectTypes(NamesAndTypesList & left_columns, bool correct_nullability);
|
2022-05-19 18:47:26 +00:00
|
|
|
void addJoinedColumnsAndCorrectTypes(ColumnsWithTypeAndName & left_columns, bool correct_nullability);
|
2021-02-18 16:43:41 +00:00
|
|
|
|
2021-03-05 14:34:43 +00:00
|
|
|
/// Calculate converting actions, rename key columns in required
|
|
|
|
/// For `USING` join we will convert key columns inplace and affect into types in the result table
|
|
|
|
/// For `JOIN ON` we will create new columns with converted keys to join by.
|
2021-06-29 09:52:19 +00:00
|
|
|
std::pair<ActionsDAGPtr, ActionsDAGPtr>
|
2022-04-05 10:14:42 +00:00
|
|
|
createConvertingActions(
|
|
|
|
const ColumnsWithTypeAndName & left_sample_columns,
|
|
|
|
const ColumnsWithTypeAndName & right_sample_columns);
|
2019-09-02 19:58:45 +00:00
|
|
|
|
2022-07-29 16:30:50 +00:00
|
|
|
void setAsofInequality(ASOFJoinInequality inequality) { asof_inequality = inequality; }
|
2023-08-03 15:55:15 +00:00
|
|
|
ASOFJoinInequality getAsofInequality() const { return asof_inequality; }
|
2019-10-11 17:56:26 +00:00
|
|
|
|
2019-09-02 19:58:45 +00:00
|
|
|
ASTPtr leftKeysList() const;
|
|
|
|
ASTPtr rightKeysList() const; /// For ON syntax only
|
|
|
|
|
2022-09-06 15:25:52 +00:00
|
|
|
void setColumnsFromJoinedTable(NamesAndTypesList columns_from_joined_table_value, const NameSet & left_table_columns, const String & right_table_prefix)
|
|
|
|
{
|
|
|
|
columns_from_joined_table = std::move(columns_from_joined_table_value);
|
|
|
|
deduplicateAndQualifyColumnNames(left_table_columns, right_table_prefix);
|
|
|
|
}
|
2019-09-02 19:58:45 +00:00
|
|
|
const NamesAndTypesList & columnsFromJoinedTable() const { return columns_from_joined_table; }
|
2022-08-15 16:34:10 +00:00
|
|
|
const NamesAndTypesList & columnsAddedByJoin() const { return columns_added_by_join; }
|
2021-03-05 14:34:43 +00:00
|
|
|
|
|
|
|
/// StorageJoin overrides key names (cause of different names qualification)
|
2021-09-09 09:47:08 +00:00
|
|
|
void setRightKeys(const Names & keys) { getOnlyClause().key_names_right = keys; }
|
2022-12-07 15:02:37 +00:00
|
|
|
void setLeftKeys(const Names & keys) { getOnlyClause().key_names_left = keys; }
|
2021-03-05 14:34:43 +00:00
|
|
|
|
|
|
|
Block getRequiredRightKeys(const Block & right_table_keys, std::vector<String> & keys_sources) const;
|
2021-04-29 14:30:02 +00:00
|
|
|
|
|
|
|
String renamedRightColumnName(const String & name) const;
|
2023-04-25 11:28:54 +00:00
|
|
|
String renamedRightColumnNameWithAlias(const String & name) const;
|
2022-12-06 11:45:41 +00:00
|
|
|
void setRename(const String & from, const String & to);
|
2021-11-08 12:44:13 +00:00
|
|
|
|
|
|
|
void resetKeys();
|
|
|
|
void resetToCross();
|
|
|
|
|
2021-08-06 14:15:11 +00:00
|
|
|
std::unordered_map<String, String> leftToRightKeyRemap() const;
|
2021-06-29 09:22:53 +00:00
|
|
|
|
2022-07-07 12:26:34 +00:00
|
|
|
/// Remember storage name in case of joining with dictionary or another special storage
|
|
|
|
void setRightStorageName(const std::string & storage_name);
|
|
|
|
const std::string & getRightStorageName() const;
|
|
|
|
|
2022-08-08 10:58:28 +00:00
|
|
|
void setStorageJoin(std::shared_ptr<const IKeyValueEntity> storage);
|
2021-06-29 09:22:53 +00:00
|
|
|
void setStorageJoin(std::shared_ptr<StorageJoin> storage);
|
|
|
|
|
2022-12-05 17:54:12 +00:00
|
|
|
std::shared_ptr<StorageJoin> getStorageJoin() const { return right_storage_join; }
|
2021-06-29 09:22:53 +00:00
|
|
|
|
2022-07-08 13:11:27 +00:00
|
|
|
bool isSpecialStorage() const { return !right_storage_name.empty() || right_storage_join || right_kv_storage; }
|
2022-01-21 05:36:36 +00:00
|
|
|
|
2022-08-08 10:58:28 +00:00
|
|
|
std::shared_ptr<const IKeyValueEntity> getStorageKeyValue() { return right_kv_storage; }
|
2023-08-04 11:07:33 +00:00
|
|
|
|
|
|
|
NamesAndTypesList correctedColumnsAddedByJoin() const;
|
2018-11-02 18:53:23 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
}
|