2017-04-01 09:19:00 +00:00
|
|
|
#include <Storages/StorageJoin.h>
|
2017-12-30 00:36:06 +00:00
|
|
|
#include <Storages/StorageFactory.h>
|
2020-04-07 09:48:47 +00:00
|
|
|
#include <Interpreters/HashJoin.h>
|
2020-02-10 15:50:12 +00:00
|
|
|
#include <Interpreters/Context.h>
|
2018-11-30 14:49:35 +00:00
|
|
|
#include <Parsers/ASTCreateQuery.h>
|
2019-03-23 22:45:28 +00:00
|
|
|
#include <Parsers/ASTSetQuery.h>
|
2017-12-30 00:36:06 +00:00
|
|
|
#include <Parsers/ASTIdentifier.h>
|
2018-11-30 14:49:35 +00:00
|
|
|
#include <Core/ColumnNumbers.h>
|
2019-01-23 14:48:50 +00:00
|
|
|
#include <DataStreams/IBlockInputStream.h>
|
2018-11-30 14:49:35 +00:00
|
|
|
#include <DataTypes/NestedUtils.h>
|
2020-12-15 16:45:13 +00:00
|
|
|
#include <Disks/IDisk.h>
|
2019-07-02 14:38:31 +00:00
|
|
|
#include <Interpreters/joinDispatch.h>
|
2020-04-07 09:48:47 +00:00
|
|
|
#include <Interpreters/TableJoin.h>
|
2020-11-25 12:41:12 +00:00
|
|
|
#include <Interpreters/castColumn.h>
|
2019-08-21 02:28:04 +00:00
|
|
|
#include <Common/assert_cast.h>
|
2019-12-30 21:30:25 +00:00
|
|
|
#include <Common/quoteString.h>
|
2018-03-11 00:15:26 +00:00
|
|
|
#include <Poco/String.h> /// toLower
|
2020-01-31 14:51:09 +00:00
|
|
|
#include <Processors/Sources/SourceWithProgress.h>
|
|
|
|
#include <Processors/Pipe.h>
|
2015-01-27 21:24:24 +00:00
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2016-01-11 21:46:36 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2020-02-25 18:02:41 +00:00
|
|
|
extern const int NOT_IMPLEMENTED;
|
|
|
|
extern const int LOGICAL_ERROR;
|
2019-08-05 14:03:14 +00:00
|
|
|
extern const int UNSUPPORTED_JOIN_KEYS;
|
2017-04-01 07:20:54 +00:00
|
|
|
extern const int NO_SUCH_COLUMN_IN_TABLE;
|
|
|
|
extern const int INCOMPATIBLE_TYPE_OF_JOIN;
|
2017-12-30 00:36:06 +00:00
|
|
|
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
|
|
|
|
extern const int BAD_ARGUMENTS;
|
2016-01-11 21:46:36 +00:00
|
|
|
}
|
|
|
|
|
2015-01-27 21:24:24 +00:00
|
|
|
StorageJoin::StorageJoin(
|
2020-12-15 16:45:13 +00:00
|
|
|
DiskPtr disk_,
|
2019-10-25 19:07:47 +00:00
|
|
|
const String & relative_path_,
|
2019-12-04 16:06:55 +00:00
|
|
|
const StorageID & table_id_,
|
2017-04-01 07:20:54 +00:00
|
|
|
const Names & key_names_,
|
2018-11-30 14:49:35 +00:00
|
|
|
bool use_nulls_,
|
|
|
|
SizeLimits limits_,
|
|
|
|
ASTTableJoin::Kind kind_,
|
|
|
|
ASTTableJoin::Strictness strictness_,
|
2018-12-30 15:54:45 +00:00
|
|
|
const ColumnsDescription & columns_,
|
2019-08-24 21:20:20 +00:00
|
|
|
const ConstraintsDescription & constraints_,
|
2021-04-23 12:18:23 +00:00
|
|
|
const String & comment,
|
2020-03-29 10:07:51 +00:00
|
|
|
bool overwrite_,
|
2020-09-18 12:58:27 +00:00
|
|
|
bool persistent_)
|
2021-04-23 12:18:23 +00:00
|
|
|
: StorageSetOrJoinBase{disk_, relative_path_, table_id_, columns_, constraints_, comment, persistent_}
|
2018-11-30 14:49:35 +00:00
|
|
|
, key_names(key_names_)
|
|
|
|
, use_nulls(use_nulls_)
|
|
|
|
, limits(limits_)
|
|
|
|
, kind(kind_)
|
|
|
|
, strictness(strictness_)
|
2020-03-29 10:07:51 +00:00
|
|
|
, overwrite(overwrite_)
|
2015-01-27 21:24:24 +00:00
|
|
|
{
|
2020-06-16 15:51:29 +00:00
|
|
|
auto metadata_snapshot = getInMemoryMetadataPtr();
|
2017-04-01 07:20:54 +00:00
|
|
|
for (const auto & key : key_names)
|
2020-06-16 15:51:29 +00:00
|
|
|
if (!metadata_snapshot->getColumns().hasPhysical(key))
|
2018-05-07 02:01:11 +00:00
|
|
|
throw Exception{"Key column (" + key + ") does not exist in table declaration.", ErrorCodes::NO_SUCH_COLUMN_IN_TABLE};
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-04-07 09:48:47 +00:00
|
|
|
table_join = std::make_shared<TableJoin>(limits, use_nulls, kind, strictness, key_names);
|
2020-06-16 15:51:29 +00:00
|
|
|
join = std::make_shared<HashJoin>(table_join, metadata_snapshot->getSampleBlock().sortColumns(), overwrite);
|
2017-04-01 07:20:54 +00:00
|
|
|
restore();
|
2015-01-27 21:24:24 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2020-06-18 10:29:13 +00:00
|
|
|
void StorageJoin::truncate(
|
2021-04-10 23:33:54 +00:00
|
|
|
const ASTPtr &, const StorageMetadataPtr & metadata_snapshot, ContextPtr, TableExclusiveLockHolder&)
|
2018-06-09 16:09:37 +00:00
|
|
|
{
|
2020-12-15 16:45:13 +00:00
|
|
|
disk->removeRecursive(path);
|
|
|
|
disk->createDirectories(path);
|
|
|
|
disk->createDirectories(path + "tmp/");
|
2018-06-09 16:09:37 +00:00
|
|
|
|
|
|
|
increment = 0;
|
2020-06-16 15:51:29 +00:00
|
|
|
join = std::make_shared<HashJoin>(table_join, metadata_snapshot->getSampleBlock().sortColumns(), overwrite);
|
2018-08-10 04:02:56 +00:00
|
|
|
}
|
2018-06-09 16:09:37 +00:00
|
|
|
|
|
|
|
|
2021-02-25 09:31:22 +00:00
|
|
|
HashJoinPtr StorageJoin::getJoinLocked(std::shared_ptr<TableJoin> analyzed_join) const
|
2015-01-28 02:37:05 +00:00
|
|
|
{
|
2020-06-16 15:51:29 +00:00
|
|
|
auto metadata_snapshot = getInMemoryMetadataPtr();
|
2020-02-02 16:12:48 +00:00
|
|
|
if (!analyzed_join->sameStrictnessAndKind(strictness, kind))
|
2019-12-03 16:25:32 +00:00
|
|
|
throw Exception("Table " + getStorageID().getNameForLogs() + " has incompatible type of JOIN.", ErrorCodes::INCOMPATIBLE_TYPE_OF_JOIN);
|
2019-12-19 15:50:28 +00:00
|
|
|
|
2019-12-30 20:08:03 +00:00
|
|
|
if ((analyzed_join->forceNullableRight() && !use_nulls) ||
|
|
|
|
(!analyzed_join->forceNullableRight() && isLeftOrFull(analyzed_join->kind()) && use_nulls))
|
2020-01-09 14:33:58 +00:00
|
|
|
throw Exception("Table " + getStorageID().getNameForLogs() + " needs the same join_use_nulls setting as present in LEFT or FULL JOIN.",
|
2019-12-30 19:43:40 +00:00
|
|
|
ErrorCodes::INCOMPATIBLE_TYPE_OF_JOIN);
|
|
|
|
|
2019-12-19 15:50:28 +00:00
|
|
|
/// TODO: check key columns
|
|
|
|
|
2021-05-10 13:39:32 +00:00
|
|
|
/// Set names qualifiers: table.column -> column
|
|
|
|
/// It's required because storage join stores non-qualified names
|
|
|
|
/// Qualifies will be added by join implementation (HashJoin)
|
2019-12-19 20:37:10 +00:00
|
|
|
analyzed_join->setRightKeys(key_names);
|
|
|
|
|
2020-06-16 15:51:29 +00:00
|
|
|
HashJoinPtr join_clone = std::make_shared<HashJoin>(analyzed_join, metadata_snapshot->getSampleBlock().sortColumns());
|
2021-02-20 15:00:59 +00:00
|
|
|
join_clone->setLock(rwlock);
|
2021-02-24 06:48:12 +00:00
|
|
|
join_clone->reuseJoinedData(*join);
|
2021-02-20 15:00:59 +00:00
|
|
|
|
2019-12-19 15:50:28 +00:00
|
|
|
return join_clone;
|
2015-01-28 02:37:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2021-02-20 15:00:59 +00:00
|
|
|
void StorageJoin::insertBlock(const Block & block)
|
|
|
|
{
|
|
|
|
std::unique_lock<std::shared_mutex> lock(rwlock);
|
|
|
|
join->addJoinedBlock(block, true);
|
|
|
|
}
|
2020-10-23 18:11:55 +00:00
|
|
|
|
2021-02-24 16:19:04 +00:00
|
|
|
size_t StorageJoin::getSize() const
|
|
|
|
{
|
|
|
|
std::shared_lock<std::shared_mutex> lock(rwlock);
|
|
|
|
return join->getTotalRowCount();
|
|
|
|
}
|
|
|
|
|
|
|
|
std::optional<UInt64> StorageJoin::totalRows(const Settings &) const
|
|
|
|
{
|
|
|
|
std::shared_lock<std::shared_mutex> lock(rwlock);
|
|
|
|
return join->getTotalRowCount();
|
|
|
|
}
|
|
|
|
|
|
|
|
std::optional<UInt64> StorageJoin::totalBytes(const Settings &) const
|
|
|
|
{
|
|
|
|
std::shared_lock<std::shared_mutex> lock(rwlock);
|
|
|
|
return join->getTotalByteCount();
|
|
|
|
}
|
2017-01-14 09:00:19 +00:00
|
|
|
|
2021-02-25 09:31:22 +00:00
|
|
|
DataTypePtr StorageJoin::joinGetCheckAndGetReturnType(const DataTypes & data_types, const String & column_name, bool or_null) const
|
|
|
|
{
|
|
|
|
return join->joinGetCheckAndGetReturnType(data_types, column_name, or_null);
|
|
|
|
}
|
|
|
|
|
|
|
|
ColumnWithTypeAndName StorageJoin::joinGet(const Block & block, const Block & block_with_columns_to_add) const
|
|
|
|
{
|
|
|
|
std::shared_lock<std::shared_mutex> lock(rwlock);
|
|
|
|
return join->joinGet(block, block_with_columns_to_add);
|
|
|
|
}
|
2017-12-30 00:36:06 +00:00
|
|
|
|
|
|
|
void registerStorageJoin(StorageFactory & factory)
|
|
|
|
{
|
2020-01-25 19:14:33 +00:00
|
|
|
auto creator_fn = [](const StorageFactory::Arguments & args)
|
2017-12-30 00:36:06 +00:00
|
|
|
{
|
|
|
|
/// Join(ANY, LEFT, k1, k2, ...)
|
|
|
|
|
|
|
|
ASTs & engine_args = args.engine_args;
|
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
const auto & settings = args.getContext()->getSettingsRef();
|
2019-12-25 17:32:55 +00:00
|
|
|
|
2018-11-30 14:49:35 +00:00
|
|
|
auto join_use_nulls = settings.join_use_nulls;
|
|
|
|
auto max_rows_in_join = settings.max_rows_in_join;
|
|
|
|
auto max_bytes_in_join = settings.max_bytes_in_join;
|
|
|
|
auto join_overflow_mode = settings.join_overflow_mode;
|
2019-01-24 17:12:05 +00:00
|
|
|
auto join_any_take_last_row = settings.join_any_take_last_row;
|
2019-12-25 17:32:55 +00:00
|
|
|
auto old_any_join = settings.any_join_distinct_right_table_keys;
|
2020-09-19 16:38:36 +00:00
|
|
|
bool persistent = true;
|
2020-12-15 16:45:13 +00:00
|
|
|
String disk_name = "default";
|
2018-11-30 14:49:35 +00:00
|
|
|
|
|
|
|
if (args.storage_def && args.storage_def->settings)
|
|
|
|
{
|
2019-04-18 23:29:32 +00:00
|
|
|
for (const auto & setting : args.storage_def->settings->changes)
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
2018-12-18 01:26:12 +00:00
|
|
|
if (setting.name == "join_use_nulls")
|
2020-07-27 14:26:18 +00:00
|
|
|
join_use_nulls = setting.value;
|
2018-12-18 01:26:12 +00:00
|
|
|
else if (setting.name == "max_rows_in_join")
|
2020-07-27 14:26:18 +00:00
|
|
|
max_rows_in_join = setting.value;
|
2018-12-18 01:26:12 +00:00
|
|
|
else if (setting.name == "max_bytes_in_join")
|
2020-07-27 14:26:18 +00:00
|
|
|
max_bytes_in_join = setting.value;
|
2018-12-18 01:26:12 +00:00
|
|
|
else if (setting.name == "join_overflow_mode")
|
2020-07-27 14:26:18 +00:00
|
|
|
join_overflow_mode = setting.value;
|
2019-01-24 17:12:05 +00:00
|
|
|
else if (setting.name == "join_any_take_last_row")
|
2020-07-27 14:26:18 +00:00
|
|
|
join_any_take_last_row = setting.value;
|
2019-12-25 17:32:55 +00:00
|
|
|
else if (setting.name == "any_join_distinct_right_table_keys")
|
2020-07-27 14:26:18 +00:00
|
|
|
old_any_join = setting.value;
|
2020-12-15 16:45:13 +00:00
|
|
|
else if (setting.name == "disk")
|
|
|
|
disk_name = setting.value.get<String>();
|
2020-09-18 12:58:27 +00:00
|
|
|
else if (setting.name == "persistent")
|
2020-09-15 09:16:10 +00:00
|
|
|
{
|
|
|
|
auto join_settings = std::make_unique<JoinSettings>();
|
|
|
|
join_settings->loadFromQuery(*args.storage_def);
|
2020-09-19 16:38:36 +00:00
|
|
|
persistent = join_settings->persistent;
|
2020-09-15 09:16:10 +00:00
|
|
|
}
|
2018-11-30 14:49:35 +00:00
|
|
|
else
|
2020-12-15 16:45:13 +00:00
|
|
|
throw Exception("Unknown setting " + setting.name + " for storage " + args.engine_name, ErrorCodes::BAD_ARGUMENTS);
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
DiskPtr disk = args.getContext()->getDisk(disk_name);
|
2020-12-15 16:45:13 +00:00
|
|
|
|
2019-12-25 17:32:55 +00:00
|
|
|
if (engine_args.size() < 3)
|
|
|
|
throw Exception(
|
|
|
|
"Storage Join requires at least 3 parameters: Join(ANY|ALL|SEMI|ANTI, LEFT|INNER|RIGHT, keys...).",
|
|
|
|
ErrorCodes::NUMBER_OF_ARGUMENTS_DOESNT_MATCH);
|
|
|
|
|
|
|
|
ASTTableJoin::Strictness strictness = ASTTableJoin::Strictness::Unspecified;
|
|
|
|
ASTTableJoin::Kind kind = ASTTableJoin::Kind::Comma;
|
|
|
|
|
|
|
|
if (auto opt_strictness_id = tryGetIdentifierName(engine_args[0]))
|
|
|
|
{
|
|
|
|
const String strictness_str = Poco::toLower(*opt_strictness_id);
|
|
|
|
|
2019-12-27 12:41:55 +00:00
|
|
|
if (strictness_str == "any")
|
2019-12-25 17:32:55 +00:00
|
|
|
{
|
|
|
|
if (old_any_join)
|
|
|
|
strictness = ASTTableJoin::Strictness::RightAny;
|
|
|
|
else
|
|
|
|
strictness = ASTTableJoin::Strictness::Any;
|
|
|
|
}
|
2019-12-27 12:41:55 +00:00
|
|
|
else if (strictness_str == "all")
|
2019-12-25 17:32:55 +00:00
|
|
|
strictness = ASTTableJoin::Strictness::All;
|
2019-12-27 12:41:55 +00:00
|
|
|
else if (strictness_str == "semi")
|
2019-12-25 17:32:55 +00:00
|
|
|
strictness = ASTTableJoin::Strictness::Semi;
|
2019-12-27 12:41:55 +00:00
|
|
|
else if (strictness_str == "anti")
|
2019-12-25 17:32:55 +00:00
|
|
|
strictness = ASTTableJoin::Strictness::Anti;
|
|
|
|
}
|
|
|
|
|
|
|
|
if (strictness == ASTTableJoin::Strictness::Unspecified)
|
2019-12-27 12:41:55 +00:00
|
|
|
throw Exception("First parameter of storage Join must be ANY or ALL or SEMI or ANTI (without quotes).",
|
|
|
|
ErrorCodes::BAD_ARGUMENTS);
|
2019-12-25 17:32:55 +00:00
|
|
|
|
|
|
|
if (auto opt_kind_id = tryGetIdentifierName(engine_args[1]))
|
|
|
|
{
|
|
|
|
const String kind_str = Poco::toLower(*opt_kind_id);
|
|
|
|
|
2019-12-27 12:41:55 +00:00
|
|
|
if (kind_str == "left")
|
2019-12-25 17:32:55 +00:00
|
|
|
kind = ASTTableJoin::Kind::Left;
|
2019-12-27 12:41:55 +00:00
|
|
|
else if (kind_str == "inner")
|
2019-12-25 17:32:55 +00:00
|
|
|
kind = ASTTableJoin::Kind::Inner;
|
2019-12-27 12:41:55 +00:00
|
|
|
else if (kind_str == "right")
|
2019-12-25 17:32:55 +00:00
|
|
|
kind = ASTTableJoin::Kind::Right;
|
2019-12-27 12:41:55 +00:00
|
|
|
else if (kind_str == "full")
|
2019-12-25 17:32:55 +00:00
|
|
|
{
|
|
|
|
if (strictness == ASTTableJoin::Strictness::Any)
|
|
|
|
strictness = ASTTableJoin::Strictness::RightAny;
|
|
|
|
kind = ASTTableJoin::Kind::Full;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (kind == ASTTableJoin::Kind::Comma)
|
2019-12-27 12:41:55 +00:00
|
|
|
throw Exception("Second parameter of storage Join must be LEFT or INNER or RIGHT or FULL (without quotes).",
|
|
|
|
ErrorCodes::BAD_ARGUMENTS);
|
2019-12-25 17:32:55 +00:00
|
|
|
|
|
|
|
Names key_names;
|
|
|
|
key_names.reserve(engine_args.size() - 2);
|
|
|
|
for (size_t i = 2, size = engine_args.size(); i < size; ++i)
|
|
|
|
{
|
|
|
|
auto opt_key = tryGetIdentifierName(engine_args[i]);
|
|
|
|
if (!opt_key)
|
|
|
|
throw Exception("Parameter №" + toString(i + 1) + " of storage Join don't look like column name.", ErrorCodes::BAD_ARGUMENTS);
|
|
|
|
|
|
|
|
key_names.push_back(*opt_key);
|
|
|
|
}
|
|
|
|
|
2017-12-30 00:36:06 +00:00
|
|
|
return StorageJoin::create(
|
2020-12-15 16:45:13 +00:00
|
|
|
disk,
|
2019-10-25 19:07:47 +00:00
|
|
|
args.relative_data_path,
|
2019-12-04 16:06:55 +00:00
|
|
|
args.table_id,
|
2018-11-30 14:49:35 +00:00
|
|
|
key_names,
|
2019-08-09 13:02:01 +00:00
|
|
|
join_use_nulls,
|
|
|
|
SizeLimits{max_rows_in_join, max_bytes_in_join, join_overflow_mode},
|
2018-11-30 14:49:35 +00:00
|
|
|
kind,
|
|
|
|
strictness,
|
2018-12-30 15:54:45 +00:00
|
|
|
args.columns,
|
2019-08-24 21:20:20 +00:00
|
|
|
args.constraints,
|
2021-04-23 12:18:23 +00:00
|
|
|
args.comment,
|
2019-10-25 19:07:47 +00:00
|
|
|
join_any_take_last_row,
|
2020-09-19 16:38:36 +00:00
|
|
|
persistent);
|
2020-01-25 19:14:33 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
factory.registerStorage("Join", creator_fn, StorageFactory::StorageFeatures{ .supports_settings = true, });
|
2017-12-30 00:36:06 +00:00
|
|
|
}
|
|
|
|
|
2018-11-30 14:49:35 +00:00
|
|
|
template <typename T>
|
|
|
|
static const char * rawData(T & t)
|
|
|
|
{
|
|
|
|
return reinterpret_cast<const char *>(&t);
|
|
|
|
}
|
|
|
|
template <typename T>
|
|
|
|
static size_t rawSize(T &)
|
|
|
|
{
|
|
|
|
return sizeof(T);
|
|
|
|
}
|
|
|
|
template <>
|
|
|
|
const char * rawData(const StringRef & t)
|
|
|
|
{
|
|
|
|
return t.data;
|
|
|
|
}
|
|
|
|
template <>
|
|
|
|
size_t rawSize(const StringRef & t)
|
|
|
|
{
|
|
|
|
return t.size;
|
|
|
|
}
|
|
|
|
|
2020-01-31 14:51:09 +00:00
|
|
|
class JoinSource : public SourceWithProgress
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
|
|
|
public:
|
2021-02-25 09:31:22 +00:00
|
|
|
JoinSource(HashJoinPtr join_, std::shared_mutex & rwlock, UInt64 max_block_size_, Block sample_block_)
|
2020-01-31 14:51:09 +00:00
|
|
|
: SourceWithProgress(sample_block_)
|
2021-02-25 09:31:22 +00:00
|
|
|
, join(join_)
|
|
|
|
, lock(rwlock)
|
2020-01-31 14:51:09 +00:00
|
|
|
, max_block_size(max_block_size_)
|
|
|
|
, sample_block(std::move(sample_block_))
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
|
|
|
column_indices.resize(sample_block.columns());
|
2020-06-01 17:27:22 +00:00
|
|
|
|
2021-02-25 09:31:22 +00:00
|
|
|
auto & saved_block = join->getJoinedData()->sample_block;
|
2020-06-01 17:27:22 +00:00
|
|
|
|
2018-11-30 14:49:35 +00:00
|
|
|
for (size_t i = 0; i < sample_block.columns(); ++i)
|
|
|
|
{
|
|
|
|
auto & [_, type, name] = sample_block.getByPosition(i);
|
2021-02-25 09:31:22 +00:00
|
|
|
if (join->right_table_keys.has(name))
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
|
|
|
key_pos = i;
|
2021-02-25 09:31:22 +00:00
|
|
|
const auto & column = join->right_table_keys.getByName(name);
|
2020-06-01 17:27:22 +00:00
|
|
|
restored_block.insert(column);
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
2020-06-01 17:27:22 +00:00
|
|
|
size_t pos = saved_block.getPositionByName(name);
|
2018-11-30 14:49:35 +00:00
|
|
|
column_indices[i] = pos;
|
2020-06-01 17:27:22 +00:00
|
|
|
|
2020-06-01 19:38:51 +00:00
|
|
|
const auto & column = saved_block.getByPosition(pos);
|
2020-06-01 17:27:22 +00:00
|
|
|
restored_block.insert(column);
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
String getName() const override { return "Join"; }
|
|
|
|
|
|
|
|
protected:
|
2020-01-31 14:51:09 +00:00
|
|
|
Chunk generate() override
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
2021-02-25 09:31:22 +00:00
|
|
|
if (join->data->blocks.empty())
|
2020-01-31 14:51:09 +00:00
|
|
|
return {};
|
2018-11-30 14:49:35 +00:00
|
|
|
|
2020-01-31 14:51:09 +00:00
|
|
|
Chunk chunk;
|
2021-02-25 09:31:22 +00:00
|
|
|
if (!joinDispatch(join->kind, join->strictness, join->data->maps,
|
2020-01-31 14:51:09 +00:00
|
|
|
[&](auto kind, auto strictness, auto & map) { chunk = createChunk<kind, strictness>(map); }))
|
2019-12-25 17:32:55 +00:00
|
|
|
throw Exception("Logical error: unknown JOIN strictness", ErrorCodes::LOGICAL_ERROR);
|
2020-01-31 14:51:09 +00:00
|
|
|
return chunk;
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
private:
|
2021-02-25 09:31:22 +00:00
|
|
|
HashJoinPtr join;
|
|
|
|
std::shared_lock<std::shared_mutex> lock;
|
|
|
|
|
2019-02-10 16:55:12 +00:00
|
|
|
UInt64 max_block_size;
|
2018-11-30 14:49:35 +00:00
|
|
|
Block sample_block;
|
2020-06-01 17:27:22 +00:00
|
|
|
Block restored_block; /// sample_block with parent column types
|
2018-11-30 14:49:35 +00:00
|
|
|
|
|
|
|
ColumnNumbers column_indices;
|
|
|
|
std::optional<size_t> key_pos;
|
|
|
|
|
|
|
|
std::unique_ptr<void, std::function<void(void *)>> position; /// type erasure
|
|
|
|
|
|
|
|
|
2019-12-25 17:32:55 +00:00
|
|
|
template <ASTTableJoin::Kind KIND, ASTTableJoin::Strictness STRICTNESS, typename Maps>
|
2020-01-31 14:51:09 +00:00
|
|
|
Chunk createChunk(const Maps & maps)
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
2020-11-25 12:41:12 +00:00
|
|
|
MutableColumns mut_columns = restored_block.cloneEmpty().mutateColumns();
|
2018-11-30 14:49:35 +00:00
|
|
|
|
|
|
|
size_t rows_added = 0;
|
|
|
|
|
2021-02-25 09:31:22 +00:00
|
|
|
switch (join->data->type)
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
|
|
|
#define M(TYPE) \
|
2020-04-07 09:48:47 +00:00
|
|
|
case HashJoin::Type::TYPE: \
|
2020-11-25 12:41:12 +00:00
|
|
|
rows_added = fillColumns<KIND, STRICTNESS>(*maps.TYPE, mut_columns); \
|
2018-11-30 14:49:35 +00:00
|
|
|
break;
|
|
|
|
APPLY_FOR_JOIN_VARIANTS_LIMITED(M)
|
|
|
|
#undef M
|
|
|
|
|
|
|
|
default:
|
2021-02-25 09:31:22 +00:00
|
|
|
throw Exception("Unsupported JOIN keys in StorageJoin. Type: " + toString(static_cast<UInt32>(join->data->type)),
|
2019-08-05 14:03:14 +00:00
|
|
|
ErrorCodes::UNSUPPORTED_JOIN_KEYS);
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if (!rows_added)
|
|
|
|
return {};
|
|
|
|
|
2020-11-25 12:41:12 +00:00
|
|
|
Columns columns;
|
|
|
|
columns.reserve(mut_columns.size());
|
|
|
|
for (auto & col : mut_columns)
|
|
|
|
columns.emplace_back(std::move(col));
|
|
|
|
|
|
|
|
/// Correct nullability and LowCardinality types
|
2018-11-30 14:49:35 +00:00
|
|
|
for (size_t i = 0; i < columns.size(); ++i)
|
2020-06-01 17:27:22 +00:00
|
|
|
{
|
2020-11-25 12:41:12 +00:00
|
|
|
const auto & src = restored_block.getByPosition(i);
|
|
|
|
const auto & dst = sample_block.getByPosition(i);
|
2020-06-01 17:27:22 +00:00
|
|
|
|
2020-11-25 12:41:12 +00:00
|
|
|
if (!src.type->equals(*dst.type))
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
2020-11-25 12:41:12 +00:00
|
|
|
auto arg = src;
|
|
|
|
arg.column = std::move(columns[i]);
|
|
|
|
columns[i] = castColumn(arg, dst.type);
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
2020-06-01 17:27:22 +00:00
|
|
|
}
|
2018-11-30 14:49:35 +00:00
|
|
|
|
2020-06-01 17:27:22 +00:00
|
|
|
UInt64 num_rows = columns.at(0)->size();
|
|
|
|
return Chunk(std::move(columns), num_rows);
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
|
|
|
|
2019-12-25 17:32:55 +00:00
|
|
|
template <ASTTableJoin::Kind KIND, ASTTableJoin::Strictness STRICTNESS, typename Map>
|
2020-06-01 17:27:22 +00:00
|
|
|
size_t fillColumns(const Map & map, MutableColumns & columns)
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
|
|
|
size_t rows_added = 0;
|
|
|
|
|
|
|
|
if (!position)
|
|
|
|
position = decltype(position)(
|
2021-05-08 20:54:49 +00:00
|
|
|
static_cast<void *>(new typename Map::const_iterator(map.begin())), //-V572
|
2018-11-30 14:49:35 +00:00
|
|
|
[](void * ptr) { delete reinterpret_cast<typename Map::const_iterator *>(ptr); });
|
|
|
|
|
|
|
|
auto & it = *reinterpret_cast<typename Map::const_iterator *>(position.get());
|
|
|
|
auto end = map.end();
|
|
|
|
|
|
|
|
for (; it != end; ++it)
|
|
|
|
{
|
2019-11-07 21:32:44 +00:00
|
|
|
if constexpr (STRICTNESS == ASTTableJoin::Strictness::RightAny)
|
2018-11-30 14:49:35 +00:00
|
|
|
{
|
2019-12-25 17:32:55 +00:00
|
|
|
fillOne<Map>(columns, column_indices, it, key_pos, rows_added);
|
|
|
|
}
|
|
|
|
else if constexpr (STRICTNESS == ASTTableJoin::Strictness::All)
|
|
|
|
{
|
|
|
|
fillAll<Map>(columns, column_indices, it, key_pos, rows_added);
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
2019-11-07 21:32:44 +00:00
|
|
|
else if constexpr (STRICTNESS == ASTTableJoin::Strictness::Any)
|
|
|
|
{
|
2019-12-25 17:32:55 +00:00
|
|
|
if constexpr (KIND == ASTTableJoin::Kind::Left || KIND == ASTTableJoin::Kind::Inner)
|
|
|
|
fillOne<Map>(columns, column_indices, it, key_pos, rows_added);
|
|
|
|
else if constexpr (KIND == ASTTableJoin::Kind::Right)
|
|
|
|
fillAll<Map>(columns, column_indices, it, key_pos, rows_added);
|
|
|
|
}
|
|
|
|
else if constexpr (STRICTNESS == ASTTableJoin::Strictness::Semi)
|
|
|
|
{
|
|
|
|
if constexpr (KIND == ASTTableJoin::Kind::Left)
|
|
|
|
fillOne<Map>(columns, column_indices, it, key_pos, rows_added);
|
|
|
|
else if constexpr (KIND == ASTTableJoin::Kind::Right)
|
|
|
|
fillAll<Map>(columns, column_indices, it, key_pos, rows_added);
|
2019-11-07 21:32:44 +00:00
|
|
|
}
|
2019-12-25 17:32:55 +00:00
|
|
|
else if constexpr (STRICTNESS == ASTTableJoin::Strictness::Anti)
|
2019-03-26 22:05:51 +00:00
|
|
|
{
|
2019-12-25 17:32:55 +00:00
|
|
|
if constexpr (KIND == ASTTableJoin::Kind::Left)
|
|
|
|
fillOne<Map>(columns, column_indices, it, key_pos, rows_added);
|
|
|
|
else if constexpr (KIND == ASTTableJoin::Kind::Right)
|
|
|
|
fillAll<Map>(columns, column_indices, it, key_pos, rows_added);
|
2019-03-20 16:58:28 +00:00
|
|
|
}
|
2018-11-30 14:49:35 +00:00
|
|
|
else
|
2019-12-25 17:32:55 +00:00
|
|
|
throw Exception("This JOIN is not implemented yet", ErrorCodes::NOT_IMPLEMENTED);
|
2018-11-30 14:49:35 +00:00
|
|
|
|
|
|
|
if (rows_added >= max_block_size)
|
|
|
|
{
|
|
|
|
++it;
|
|
|
|
break;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
return rows_added;
|
|
|
|
}
|
2019-12-25 17:32:55 +00:00
|
|
|
|
|
|
|
template <typename Map>
|
|
|
|
static void fillOne(MutableColumns & columns, const ColumnNumbers & column_indices, typename Map::const_iterator & it,
|
|
|
|
const std::optional<size_t> & key_pos, size_t & rows_added)
|
|
|
|
{
|
|
|
|
for (size_t j = 0; j < columns.size(); ++j)
|
|
|
|
if (j == key_pos)
|
|
|
|
columns[j]->insertData(rawData(it->getKey()), rawSize(it->getKey()));
|
|
|
|
else
|
|
|
|
columns[j]->insertFrom(*it->getMapped().block->getByPosition(column_indices[j]).column.get(), it->getMapped().row_num);
|
|
|
|
++rows_added;
|
|
|
|
}
|
|
|
|
|
|
|
|
template <typename Map>
|
|
|
|
static void fillAll(MutableColumns & columns, const ColumnNumbers & column_indices, typename Map::const_iterator & it,
|
|
|
|
const std::optional<size_t> & key_pos, size_t & rows_added)
|
|
|
|
{
|
|
|
|
for (auto ref_it = it->getMapped().begin(); ref_it.ok(); ++ref_it)
|
|
|
|
{
|
|
|
|
for (size_t j = 0; j < columns.size(); ++j)
|
|
|
|
if (j == key_pos)
|
|
|
|
columns[j]->insertData(rawData(it->getKey()), rawSize(it->getKey()));
|
|
|
|
else
|
|
|
|
columns[j]->insertFrom(*ref_it->block->getByPosition(column_indices[j]).column.get(), ref_it->row_num);
|
|
|
|
++rows_added;
|
|
|
|
}
|
|
|
|
}
|
2018-11-30 14:49:35 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
|
|
|
|
// TODO: multiple stream read and index read
|
2020-08-03 13:54:14 +00:00
|
|
|
Pipe StorageJoin::read(
|
2018-11-30 14:49:35 +00:00
|
|
|
const Names & column_names,
|
2020-06-16 14:25:08 +00:00
|
|
|
const StorageMetadataPtr & metadata_snapshot,
|
2020-09-20 17:52:17 +00:00
|
|
|
SelectQueryInfo & /*query_info*/,
|
2021-04-10 23:33:54 +00:00
|
|
|
ContextPtr /*context*/,
|
2018-11-30 14:49:35 +00:00
|
|
|
QueryProcessingStage::Enum /*processed_stage*/,
|
2019-02-18 23:38:44 +00:00
|
|
|
size_t max_block_size,
|
2018-11-30 14:49:35 +00:00
|
|
|
unsigned /*num_streams*/)
|
|
|
|
{
|
2020-06-19 17:17:13 +00:00
|
|
|
metadata_snapshot->check(column_names, getVirtuals(), getStorageID());
|
2020-01-31 14:51:09 +00:00
|
|
|
|
2021-02-25 09:31:22 +00:00
|
|
|
Block source_sample_block = metadata_snapshot->getSampleBlockForColumns(column_names, getVirtuals(), getStorageID());
|
|
|
|
return Pipe(std::make_shared<JoinSource>(join, rwlock, max_block_size, source_sample_block));
|
2018-11-30 14:49:35 +00:00
|
|
|
}
|
|
|
|
|
2015-01-27 21:24:24 +00:00
|
|
|
}
|