ClickHouse/src/Interpreters/ConcurrentHashJoin.cpp

207 lines
6.2 KiB
C++
Raw Normal View History

#include <memory>
#include <mutex>
2022-04-19 08:53:24 +00:00
#include <Columns/FilterDescription.h>
#include <Columns/IColumn.h>
#include <Core/ColumnsWithTypeAndName.h>
#include <Core/NamesAndTypes.h>
#include <Interpreters/ConcurrentHashJoin.h>
#include <Interpreters/Context.h>
2022-04-19 08:53:24 +00:00
#include <Interpreters/ExpressionActions.h>
#include <Interpreters/PreparedSets.h>
#include <Interpreters/SubqueryForSet.h>
#include <Interpreters/TableJoin.h>
2022-04-21 08:59:30 +00:00
#include <Interpreters/createBlockSelector.h>
2022-04-19 08:53:24 +00:00
#include <Parsers/DumpASTNode.h>
#include <Parsers/ExpressionListParsers.h>
#include <Parsers/IAST_fwd.h>
#include <Parsers/parseQuery.h>
#include <Common/Exception.h>
2022-04-21 08:59:30 +00:00
#include <Common/typeid_cast.h>
namespace DB
{
namespace ErrorCodes
{
extern const int LOGICAL_ERROR;
extern const int SET_SIZE_LIMIT_EXCEEDED;
extern const int BAD_ARGUMENTS;
}
namespace JoinStuff
{
2022-05-05 04:04:11 +00:00
ConcurrentHashJoin::ConcurrentHashJoin(ContextPtr context_, std::shared_ptr<TableJoin> table_join_, size_t slots_, const Block & right_sample_block, bool any_take_last_row_)
: context(context_)
, table_join(table_join_)
, slots(slots_)
{
2022-04-21 08:59:30 +00:00
if (!slots_ || slots_ >= 256)
{
throw Exception(ErrorCodes::BAD_ARGUMENTS, "Invalid argument slot : {}", slots_);
}
for (size_t i = 0; i < slots; ++i)
{
2022-04-21 04:14:36 +00:00
auto inner_hash_join = std::make_shared<InternalHashJoin>();
inner_hash_join->data = std::make_unique<HashJoin>(table_join_, right_sample_block, any_take_last_row_);
hash_joins.emplace_back(std::move(inner_hash_join));
}
2022-04-21 04:14:36 +00:00
}
bool ConcurrentHashJoin::addJoinedBlock(const Block & block, bool check_limits)
{
2022-05-05 04:04:11 +00:00
Blocks dispatched_blocks = dispatchBlock(table_join->getOnlyClause().key_names_right, block);
2022-04-21 05:19:33 +00:00
std::list<size_t> pending_blocks;
for (size_t i = 0; i < dispatched_blocks.size(); ++i)
2022-04-21 05:19:33 +00:00
pending_blocks.emplace_back(i);
while (!pending_blocks.empty())
{
2022-04-21 05:19:33 +00:00
for (auto iter = pending_blocks.begin(); iter != pending_blocks.end();)
{
auto & i = *iter;
auto & hash_join = hash_joins[i];
auto & dispatched_block = dispatched_blocks[i];
if (hash_join->mutex.try_lock())
{
2022-05-05 02:02:50 +00:00
if (!hash_join->data->addJoinedBlock(dispatched_block, check_limits))
{
hash_join->mutex.unlock();
return false;
}
2022-04-21 04:14:36 +00:00
2022-04-21 05:19:33 +00:00
hash_join->mutex.unlock();
iter = pending_blocks.erase(iter);
}
2022-04-21 08:59:30 +00:00
else
2022-04-21 05:19:33 +00:00
iter++;
}
}
if (check_limits)
2022-04-21 05:19:33 +00:00
return table_join->sizeLimits().check(getTotalRowCount(), getTotalByteCount(), "JOIN", ErrorCodes::SET_SIZE_LIMIT_EXCEEDED);
return true;
}
2022-04-21 05:19:33 +00:00
void ConcurrentHashJoin::joinBlock(Block & block, std::shared_ptr<ExtraBlock> & /*not_processed*/)
{
2022-05-05 04:04:11 +00:00
Blocks dispatched_blocks = dispatchBlock(table_join->getOnlyClause().key_names_left, block);
for (size_t i = 0; i < dispatched_blocks.size(); ++i)
{
std::shared_ptr<ExtraBlock> none_extra_block;
auto & hash_join = hash_joins[i];
auto & dispatched_block = dispatched_blocks[i];
hash_join->data->joinBlock(dispatched_block, none_extra_block);
if (none_extra_block && !none_extra_block->empty())
throw Exception(ErrorCodes::LOGICAL_ERROR, "not_processed should be empty");
}
2022-04-21 06:10:09 +00:00
block = concatenateBlocks(dispatched_blocks);
}
void ConcurrentHashJoin::checkTypesOfKeys(const Block & block) const
{
hash_joins[0]->data->checkTypesOfKeys(block);
}
void ConcurrentHashJoin::setTotals(const Block & block)
{
if (block)
{
std::lock_guard lock(totals_mutex);
totals = block;
}
}
const Block & ConcurrentHashJoin::getTotals() const
{
return totals;
}
size_t ConcurrentHashJoin::getTotalRowCount() const
{
size_t res = 0;
for (const auto & hash_join : hash_joins)
{
2022-04-25 08:16:50 +00:00
std::lock_guard lock(hash_join->mutex);
res += hash_join->data->getTotalRowCount();
}
return res;
}
size_t ConcurrentHashJoin::getTotalByteCount() const
{
size_t res = 0;
for (const auto & hash_join : hash_joins)
{
2022-04-25 08:16:50 +00:00
std::lock_guard lock(hash_join->mutex);
res += hash_join->data->getTotalByteCount();
}
return res;
}
bool ConcurrentHashJoin::alwaysReturnsEmptySet() const
{
for (const auto & hash_join : hash_joins)
{
2022-04-25 08:16:50 +00:00
std::lock_guard lock(hash_join->mutex);
2022-04-21 05:19:33 +00:00
if (!hash_join->data->alwaysReturnsEmptySet())
return false;
}
return true;
}
std::shared_ptr<NotJoinedBlocks> ConcurrentHashJoin::getNonJoinedBlocks(
const Block & /*left_sample_block*/, const Block & /*result_sample_block*/, UInt64 /*max_block_size*/) const
{
if (table_join->strictness() == ASTTableJoin::Strictness::Asof ||
table_join->strictness() == ASTTableJoin::Strictness::Semi ||
!isRightOrFull(table_join->kind()))
{
return {};
}
throw Exception(ErrorCodes::LOGICAL_ERROR, "Invalid join type. join kind: {}, strictness: {}", table_join->kind(), table_join->strictness());
}
2022-05-05 04:04:11 +00:00
Blocks ConcurrentHashJoin::dispatchBlock(const Strings & key_columns_names, const Block & from_block)
{
2022-05-05 04:04:11 +00:00
Blocks result;
size_t num_shards = hash_joins.size();
size_t num_rows = from_block.rows();
size_t num_cols = from_block.columns();
ColumnRawPtrs key_cols;
for (const auto & key_name : key_columns_names)
{
2022-05-05 04:04:11 +00:00
key_cols.push_back(from_block.getByName(key_name).column.get());
}
2022-05-05 04:04:11 +00:00
IColumn::Selector selector(num_rows);
for (size_t i = 0; i < num_rows; ++i)
{
2022-05-05 04:04:11 +00:00
SipHash hash;
for (const auto & key_col : key_cols)
{
key_col->updateHashWithValue(i, hash);
}
selector[i] = hash.get64() % num_shards;
2022-04-21 08:59:30 +00:00
}
2022-05-05 04:04:11 +00:00
for (size_t i = 0; i < num_shards; ++i)
2022-04-21 08:59:30 +00:00
{
2022-05-05 04:04:11 +00:00
result.emplace_back(from_block.cloneEmpty());
2022-04-21 08:59:30 +00:00
}
2022-05-05 04:04:11 +00:00
for (size_t i = 0; i < num_cols; ++i)
2022-04-21 08:59:30 +00:00
{
auto dispatched_columns = from_block.getByPosition(i).column->scatter(num_shards, selector);
2022-05-05 04:04:11 +00:00
assert(result.size() == dispatched_columns.size());
2022-04-21 08:59:30 +00:00
for (size_t block_index = 0; block_index < num_shards; ++block_index)
{
2022-05-05 04:04:11 +00:00
result[block_index].getByPosition(i).column = std::move(dispatched_columns[block_index]);
}
}
2022-05-05 04:04:11 +00:00
return result;
}
}
}