ClickHouse/src/Interpreters/ConcurrentHashJoin.h

Ignoring revisions in .git-blame-ignore-revs. Click here to bypass and see the normal blame view.

79 lines
2.6 KiB
C++
Raw Normal View History

#pragma once
#include <condition_variable>
#include <memory>
#include <optional>
2022-04-19 08:53:24 +00:00
#include <Interpreters/Context.h>
#include <Interpreters/ExpressionActions.h>
#include <Interpreters/HashJoin.h>
2022-04-19 08:53:24 +00:00
#include <Interpreters/IJoin.h>
#include <base/defines.h>
#include <base/types.h>
#include <Common/Stopwatch.h>
2022-04-20 11:47:16 +00:00
namespace DB
{
/**
2023-07-05 17:03:18 +00:00
* Can run addBlockToJoin() parallelly to speedup the join process. On test, it almose linear speedup by
* the degree of parallelism.
*
* The default HashJoin is not thread safe for inserting right table's rows and run it in a single thread. When
* the right table is large, the join process is too slow.
*
2023-07-05 17:03:18 +00:00
* We create multiple HashJoin instances here. In addBlockToJoin(), one input block is split into multiple blocks
* corresponding to the HashJoin instances by hashing every row on the join keys. And make a guarantee that every HashJoin
* instance is written by only one thread.
*
* When come to the left table matching, the blocks from left table are alse split into different HashJoin instances.
*
*/
class ConcurrentHashJoin : public IJoin
{
public:
2023-08-21 10:01:10 +00:00
explicit ConcurrentHashJoin(
ContextPtr context_,
std::shared_ptr<TableJoin> table_join_,
size_t slots_,
const Block & right_sample_block,
bool any_take_last_row_ = false);
~ConcurrentHashJoin() override = default;
2023-08-03 15:55:15 +00:00
std::string getName() const override { return "ConcurrentHashJoin"; }
const TableJoin & getTableJoin() const override { return *table_join; }
2023-07-05 17:03:18 +00:00
bool addBlockToJoin(const Block & block, bool check_limits) override;
void checkTypesOfKeys(const Block & block) const override;
void joinBlock(Block & block, std::shared_ptr<ExtraBlock> & not_processed) override;
void setTotals(const Block & block) override;
const Block & getTotals() const override;
size_t getTotalRowCount() const override;
size_t getTotalByteCount() const override;
bool alwaysReturnsEmptySet() const override;
bool supportParallelJoin() const override { return true; }
2022-10-18 11:43:01 +00:00
IBlocksStreamPtr
getNonJoinedBlocks(const Block & left_sample_block, const Block & result_sample_block, UInt64 max_block_size) const override;
private:
2022-04-21 04:14:36 +00:00
struct InternalHashJoin
{
std::mutex mutex;
std::unique_ptr<HashJoin> data;
};
2022-04-21 05:19:33 +00:00
ContextPtr context;
std::shared_ptr<TableJoin> table_join;
size_t slots;
2022-04-21 04:14:36 +00:00
std::vector<std::shared_ptr<InternalHashJoin>> hash_joins;
2022-06-14 14:13:01 +00:00
std::mutex totals_mutex;
Block totals;
2022-06-14 14:13:01 +00:00
IColumn::Selector selectDispatchBlock(const Strings & key_columns_names, const Block & from_block);
2022-05-05 04:04:11 +00:00
Blocks dispatchBlock(const Strings & key_columns_names, const Block & from_block);
};
}