ClickHouse/dbms/src/Storages/StorageJoin.cpp

454 lines
17 KiB
C++
Raw Normal View History

#include <Storages/StorageJoin.h>
#include <Storages/StorageFactory.h>
#include <Interpreters/Join.h>
#include <Parsers/ASTCreateQuery.h>
2019-03-23 22:45:28 +00:00
#include <Parsers/ASTSetQuery.h>
#include <Parsers/ASTIdentifier.h>
#include <Core/ColumnNumbers.h>
#include <DataStreams/IBlockInputStream.h>
#include <DataTypes/NestedUtils.h>
#include <Interpreters/joinDispatch.h>
2019-09-09 19:43:37 +00:00
#include <Interpreters/AnalyzedJoin.h>
#include <Common/assert_cast.h>
2019-12-30 21:30:25 +00:00
#include <Common/quoteString.h>
#include <Poco/String.h> /// toLower
2018-06-09 16:09:37 +00:00
#include <Poco/File.h>
namespace DB
{
namespace ErrorCodes
{
2019-08-05 14:03:14 +00:00
extern const int UNSUPPORTED_JOIN_KEYS;
extern const int NO_SUCH_COLUMN_IN_TABLE;
extern const int INCOMPATIBLE_TYPE_OF_JOIN;
extern const int NUMBER_OF_ARGUMENTS_DOESNT_MATCH;
extern const int BAD_ARGUMENTS;
}
StorageJoin::StorageJoin(
2019-10-25 19:07:47 +00:00
const String & relative_path_,
2019-12-04 16:06:55 +00:00
const StorageID & table_id_,
const Names & key_names_,
bool use_nulls_,
SizeLimits limits_,
ASTTableJoin::Kind kind_,
ASTTableJoin::Strictness strictness_,
const ColumnsDescription & columns_,
2019-08-24 21:20:20 +00:00
const ConstraintsDescription & constraints_,
2019-10-25 19:07:47 +00:00
bool overwrite,
const Context & context_)
2019-12-04 16:06:55 +00:00
: StorageSetOrJoinBase{relative_path_, table_id_, columns_, constraints_, context_}
, key_names(key_names_)
, use_nulls(use_nulls_)
, limits(limits_)
, kind(kind_)
, strictness(strictness_)
{
for (const auto & key : key_names)
2018-03-13 15:00:28 +00:00
if (!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};
2019-09-09 19:43:37 +00:00
table_join = std::make_shared<AnalyzedJoin>(limits, use_nulls, kind, strictness, key_names);
join = std::make_shared<Join>(table_join, getSampleBlock().sortColumns(), overwrite);
restore();
}
2019-08-27 20:43:08 +00:00
void StorageJoin::truncate(const ASTPtr &, const Context &, TableStructureWriteLockHolder &)
2018-06-09 16:09:37 +00:00
{
Poco::File(path).remove(true);
Poco::File(path).createDirectories();
Poco::File(path + "tmp/").createDirectories();
2018-06-09 16:09:37 +00:00
increment = 0;
join = std::make_shared<Join>(table_join, getSampleBlock().sortColumns());
}
2018-06-09 16:09:37 +00:00
2019-12-19 15:50:28 +00:00
HashJoinPtr StorageJoin::getJoin(std::shared_ptr<AnalyzedJoin> analyzed_join) const
{
if (kind != analyzed_join->kind() || strictness != analyzed_join->strictness())
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))
throw Exception("Table " + getStorageID().getNameForLogs() + " needs the same join_use_nulls setting as present in LEFT or FULL JOIN.",
ErrorCodes::INCOMPATIBLE_TYPE_OF_JOIN);
2019-12-19 15:50:28 +00:00
/// TODO: check key columns
/// Some HACK to remove wrong names qualifiers: table.column -> column.
analyzed_join->setRightKeys(key_names);
2019-12-19 15:50:28 +00:00
HashJoinPtr join_clone = std::make_shared<Join>(analyzed_join, getSampleBlock().sortColumns());
join_clone->reuseJoinedData(*join);
return join_clone;
}
2019-09-09 19:43:37 +00:00
void StorageJoin::insertBlock(const Block & block) { join->addJoinedBlock(block); }
size_t StorageJoin::getSize() const { return join->getTotalRowCount(); }
Squashed commit of the following: commit e712f469a55ff34ad34b482b15cc4153b7ad7233 Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:59:13 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 2a002823084e3a79bffcc17d479620a68eb0644b Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:58:30 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 9e06f407c8ee781ed8ddf98bdfcc31846bf2a0fe Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:55:14 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 9581620f1e839f456fa7894aa1f996d5162ac6cd Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:54:22 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 2a8564c68cb6cc3649fafaf401256d43c9a2e777 Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:47:34 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit cf60632d78ec656be3304ef4565e859bb6ce80ba Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:40:09 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit ee3d1dc6e0c4ca60e3ac1e0c30d4b3ed1e66eca0 Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:22:49 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 65592ef7116a90104fcd524b53ef8b7cf22640f2 Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:18:17 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 37972c257320d3b7e7b294e0fdeffff218647bfd Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:17:06 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit dd909d149974ce5bed2456de1261aa5a368fd3ff Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:16:28 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 3cf43266ca7e30adf01212b1a739ba5fe43639fd Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:15:42 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 6731a3df96d1609286e2536b6432916af7743f0f Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:13:35 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 1b5727e0d56415b7add4cb76110105358663602c Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:11:18 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit bbcf726a55685b8e72f5b40ba0bf1904bd1c0407 Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:09:04 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit c03b477d5e2e65014e8906ecfa2efb67ee295af1 Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:06:30 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 2986e2fb0466bc18d73693dcdded28fccc0dc66b Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:05:44 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit 5d6cdef13d2e02bd5c4954983334e9162ab2635b Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:04:53 2017 +0300 Less dependencies [#CLICKHOUSE-2] commit f2b819b25ce8b2ccdcb201eefb03e1e6f5aab590 Author: Alexey Milovidov <milovidov@yandex-team.ru> Date: Sat Jan 14 11:01:47 2017 +0300 Less dependencies [#CLICKHOUSE-2]
2017-01-14 09:00:19 +00:00
void registerStorageJoin(StorageFactory & factory)
{
auto creator_fn = [](const StorageFactory::Arguments & args)
{
/// Join(ANY, LEFT, k1, k2, ...)
ASTs & engine_args = args.engine_args;
auto & settings = args.context.getSettingsRef();
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;
auto old_any_join = settings.any_join_distinct_right_table_keys;
if (args.storage_def && args.storage_def->settings)
{
for (const auto & setting : args.storage_def->settings->changes)
{
2018-12-18 01:26:12 +00:00
if (setting.name == "join_use_nulls")
join_use_nulls.set(setting.value);
else if (setting.name == "max_rows_in_join")
max_rows_in_join.set(setting.value);
else if (setting.name == "max_bytes_in_join")
max_bytes_in_join.set(setting.value);
else if (setting.name == "join_overflow_mode")
join_overflow_mode.set(setting.value);
2019-01-24 17:12:05 +00:00
else if (setting.name == "join_any_take_last_row")
join_any_take_last_row.set(setting.value);
else if (setting.name == "any_join_distinct_right_table_keys")
old_any_join.set(setting.value);
else
throw Exception(
"Unknown setting " + setting.name + " for storage " + args.engine_name,
ErrorCodes::BAD_ARGUMENTS);
}
}
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")
{
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")
strictness = ASTTableJoin::Strictness::All;
2019-12-27 12:41:55 +00:00
else if (strictness_str == "semi")
strictness = ASTTableJoin::Strictness::Semi;
2019-12-27 12:41:55 +00:00
else if (strictness_str == "anti")
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);
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")
kind = ASTTableJoin::Kind::Left;
2019-12-27 12:41:55 +00:00
else if (kind_str == "inner")
kind = ASTTableJoin::Kind::Inner;
2019-12-27 12:41:55 +00:00
else if (kind_str == "right")
kind = ASTTableJoin::Kind::Right;
2019-12-27 12:41:55 +00:00
else if (kind_str == "full")
{
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);
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);
}
return StorageJoin::create(
2019-10-25 19:07:47 +00:00
args.relative_data_path,
2019-12-04 16:06:55 +00:00
args.table_id,
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},
kind,
strictness,
args.columns,
2019-08-24 21:20:20 +00:00
args.constraints,
2019-10-25 19:07:47 +00:00
join_any_take_last_row,
args.context);
};
factory.registerStorage("Join", creator_fn, StorageFactory::StorageFeatures{ .supports_settings = true, });
}
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;
}
class JoinBlockInputStream : public IBlockInputStream
{
public:
2019-02-10 16:55:12 +00:00
JoinBlockInputStream(const Join & parent_, UInt64 max_block_size_, Block && sample_block_)
2019-12-19 15:50:28 +00:00
: parent(parent_), lock(parent.data->rwlock), max_block_size(max_block_size_), sample_block(std::move(sample_block_))
{
columns.resize(sample_block.columns());
column_indices.resize(sample_block.columns());
column_with_null.resize(sample_block.columns());
for (size_t i = 0; i < sample_block.columns(); ++i)
{
auto & [_, type, name] = sample_block.getByPosition(i);
2019-09-11 15:57:09 +00:00
if (parent.right_table_keys.has(name))
{
key_pos = i;
2019-09-11 15:57:09 +00:00
column_with_null[i] = parent.right_table_keys.getByName(name).type->isNullable();
}
else
{
auto pos = parent.sample_block_with_columns_to_add.getPositionByName(name);
column_indices[i] = pos;
column_with_null[i] = !parent.sample_block_with_columns_to_add.getByPosition(pos).type->equals(*type);
}
}
}
String getName() const override { return "Join"; }
Block getHeader() const override { return sample_block; }
protected:
Block readImpl() override
{
2019-12-19 15:50:28 +00:00
if (parent.data->blocks.empty())
return Block();
Block block;
2019-12-19 15:50:28 +00:00
if (!joinDispatch(parent.kind, parent.strictness, parent.data->maps,
[&](auto kind, auto strictness, auto & map) { block = createBlock<kind, strictness>(map); }))
throw Exception("Logical error: unknown JOIN strictness", ErrorCodes::LOGICAL_ERROR);
return block;
}
private:
const Join & parent;
std::shared_lock<std::shared_mutex> lock;
2019-02-10 16:55:12 +00:00
UInt64 max_block_size;
Block sample_block;
ColumnNumbers column_indices;
std::vector<bool> column_with_null;
std::optional<size_t> key_pos;
MutableColumns columns;
std::unique_ptr<void, std::function<void(void *)>> position; /// type erasure
template <ASTTableJoin::Kind KIND, ASTTableJoin::Strictness STRICTNESS, typename Maps>
Block createBlock(const Maps & maps)
{
for (size_t i = 0; i < sample_block.columns(); ++i)
{
const auto & src_col = sample_block.safeGetByPosition(i);
columns[i] = src_col.type->createColumn();
if (column_with_null[i])
{
if (key_pos == i)
{
// unwrap null key column
ColumnNullable & nullable_col = assert_cast<ColumnNullable &>(*columns[i]);
columns[i] = nullable_col.getNestedColumnPtr()->assumeMutable();
}
else
// wrap non key column with null
columns[i] = makeNullable(std::move(columns[i]))->assumeMutable();
}
}
size_t rows_added = 0;
2019-12-19 15:50:28 +00:00
switch (parent.data->type)
{
#define M(TYPE) \
case Join::Type::TYPE: \
rows_added = fillColumns<KIND, STRICTNESS>(*maps.TYPE); \
break;
APPLY_FOR_JOIN_VARIANTS_LIMITED(M)
#undef M
default:
2019-12-19 15:50:28 +00:00
throw Exception("Unsupported JOIN keys in StorageJoin. Type: " + toString(static_cast<UInt32>(parent.data->type)),
2019-08-05 14:03:14 +00:00
ErrorCodes::UNSUPPORTED_JOIN_KEYS);
}
if (!rows_added)
return {};
Block res = sample_block.cloneEmpty();
for (size_t i = 0; i < columns.size(); ++i)
if (column_with_null[i])
{
if (key_pos == i)
res.getByPosition(i).column = makeNullable(std::move(columns[i]));
else
{
const ColumnNullable & nullable_col = assert_cast<const ColumnNullable &>(*columns[i]);
res.getByPosition(i).column = nullable_col.getNestedColumnPtr();
}
}
else
res.getByPosition(i).column = std::move(columns[i]);
return res;
}
template <ASTTableJoin::Kind KIND, ASTTableJoin::Strictness STRICTNESS, typename Map>
size_t fillColumns(const Map & map)
{
size_t rows_added = 0;
if (!position)
position = decltype(position)(
static_cast<void *>(new typename Map::const_iterator(map.begin())),
[](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)
{
if constexpr (STRICTNESS == ASTTableJoin::Strictness::RightAny)
{
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);
}
else if constexpr (STRICTNESS == ASTTableJoin::Strictness::Any)
{
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);
}
else if constexpr (STRICTNESS == ASTTableJoin::Strictness::Anti)
{
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);
}
else
throw Exception("This JOIN is not implemented yet", ErrorCodes::NOT_IMPLEMENTED);
if (rows_added >= max_block_size)
{
++it;
break;
}
}
return rows_added;
}
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;
}
}
};
// TODO: multiple stream read and index read
BlockInputStreams StorageJoin::read(
const Names & column_names,
const SelectQueryInfo & /*query_info*/,
const Context & /*context*/,
QueryProcessingStage::Enum /*processed_stage*/,
size_t max_block_size,
unsigned /*num_streams*/)
{
check(column_names);
return {std::make_shared<JoinBlockInputStream>(*join, max_block_size, getSampleBlockForColumns(column_names))};
}
}