2019-10-15 16:31:49 +00:00
|
|
|
#include <limits>
|
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
#include <Columns/ColumnNullable.h>
|
2021-08-06 14:15:11 +00:00
|
|
|
#include <Columns/ColumnLowCardinality.h>
|
|
|
|
|
2019-09-13 16:17:37 +00:00
|
|
|
#include <Core/SortCursor.h>
|
2021-07-21 17:03:33 +00:00
|
|
|
#include <DataStreams/TemporaryFileStream.h>
|
|
|
|
#include <DataStreams/materializeBlock.h>
|
|
|
|
#include <DataTypes/DataTypeNullable.h>
|
|
|
|
#include <DataTypes/DataTypesNumber.h>
|
2019-09-09 19:43:37 +00:00
|
|
|
#include <Interpreters/MergeJoin.h>
|
2020-04-07 09:48:47 +00:00
|
|
|
#include <Interpreters/TableJoin.h>
|
2019-09-18 12:46:57 +00:00
|
|
|
#include <Interpreters/join_common.h>
|
2019-09-12 18:06:25 +00:00
|
|
|
#include <Interpreters/sortBlock.h>
|
2021-07-22 16:05:52 +00:00
|
|
|
#include <Processors/Sources/BlocksListSource.h>
|
2020-06-01 16:58:36 +00:00
|
|
|
#include <Processors/QueryPipeline.h>
|
|
|
|
#include <Processors/Transforms/MergeSortingTransform.h>
|
|
|
|
#include <Processors/Executors/PipelineExecutingBlockInputStream.h>
|
2019-09-09 19:43:37 +00:00
|
|
|
|
2020-10-10 16:31:10 +00:00
|
|
|
|
2019-09-09 19:43:37 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2019-09-10 14:51:28 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
2019-09-13 16:17:37 +00:00
|
|
|
extern const int NOT_IMPLEMENTED;
|
2019-09-23 19:36:47 +00:00
|
|
|
extern const int PARAMETER_OUT_OF_BOUND;
|
2021-07-21 17:03:33 +00:00
|
|
|
extern const int ILLEGAL_COLUMN;
|
2019-09-23 19:36:47 +00:00
|
|
|
extern const int LOGICAL_ERROR;
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
|
|
|
|
2019-09-19 17:09:59 +00:00
|
|
|
namespace
|
|
|
|
{
|
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
String deriveTempName(const String & name)
|
|
|
|
{
|
|
|
|
return "--" + name;
|
|
|
|
}
|
|
|
|
|
|
|
|
/*
|
|
|
|
* Convert column with conditions for left or right table to join to joining key.
|
|
|
|
* Input column type is UInt8 output is Nullable(UInt8).
|
|
|
|
* 0 converted to NULL and such rows won't be joined,
|
|
|
|
* 1 converted to 0 (any constant non-NULL value to join)
|
|
|
|
*/
|
|
|
|
ColumnWithTypeAndName condtitionColumnToJoinable(const Block & block, const String & src_column_name)
|
|
|
|
{
|
|
|
|
size_t res_size = block.rows();
|
|
|
|
auto data_col = ColumnUInt8::create(res_size, 0);
|
|
|
|
auto null_map = ColumnUInt8::create(res_size, 0);
|
|
|
|
|
|
|
|
if (!src_column_name.empty())
|
|
|
|
{
|
|
|
|
auto mask_col = JoinCommon::getColumnAsMask(block, src_column_name);
|
|
|
|
assert(mask_col);
|
|
|
|
const auto & mask_data = assert_cast<const ColumnUInt8 &>(*mask_col).getData();
|
|
|
|
|
|
|
|
for (size_t i = 0; i < res_size; ++i)
|
|
|
|
null_map->getData()[i] = !mask_data[i];
|
|
|
|
}
|
|
|
|
|
|
|
|
ColumnPtr res_col = ColumnNullable::create(std::move(data_col), std::move(null_map));
|
|
|
|
DataTypePtr res_col_type = std::make_shared<DataTypeNullable>(std::make_shared<DataTypeUInt8>());
|
|
|
|
String res_name = deriveTempName(src_column_name);
|
|
|
|
|
|
|
|
if (block.has(res_name))
|
|
|
|
throw Exception(ErrorCodes::ILLEGAL_COLUMN, "Conflicting column name '{}'", res_name);
|
|
|
|
|
|
|
|
return {res_col, res_col_type, res_name};
|
|
|
|
}
|
|
|
|
|
2021-05-15 11:26:59 +00:00
|
|
|
template <bool has_left_nulls, bool has_right_nulls>
|
2019-09-19 17:09:59 +00:00
|
|
|
int nullableCompareAt(const IColumn & left_column, const IColumn & right_column, size_t lhs_pos, size_t rhs_pos)
|
|
|
|
{
|
|
|
|
static constexpr int null_direction_hint = 1;
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
if constexpr (has_left_nulls && has_right_nulls)
|
2019-09-19 17:09:59 +00:00
|
|
|
{
|
2020-04-22 06:01:33 +00:00
|
|
|
const auto * left_nullable = checkAndGetColumn<ColumnNullable>(left_column);
|
|
|
|
const auto * right_nullable = checkAndGetColumn<ColumnNullable>(right_column);
|
2019-09-19 17:09:59 +00:00
|
|
|
|
|
|
|
if (left_nullable && right_nullable)
|
|
|
|
{
|
|
|
|
int res = left_column.compareAt(lhs_pos, rhs_pos, right_column, null_direction_hint);
|
|
|
|
if (res)
|
|
|
|
return res;
|
|
|
|
|
|
|
|
/// NULL != NULL case
|
|
|
|
if (left_column.isNullAt(lhs_pos))
|
|
|
|
return null_direction_hint;
|
2020-06-16 20:13:18 +00:00
|
|
|
|
|
|
|
return 0;
|
2019-09-19 17:09:59 +00:00
|
|
|
}
|
2020-06-16 20:13:18 +00:00
|
|
|
}
|
2019-09-19 17:09:59 +00:00
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
if constexpr (has_left_nulls)
|
|
|
|
{
|
|
|
|
if (const auto * left_nullable = checkAndGetColumn<ColumnNullable>(left_column))
|
2019-09-19 17:09:59 +00:00
|
|
|
{
|
|
|
|
if (left_column.isNullAt(lhs_pos))
|
|
|
|
return null_direction_hint;
|
|
|
|
return left_nullable->getNestedColumn().compareAt(lhs_pos, rhs_pos, right_column, null_direction_hint);
|
|
|
|
}
|
2020-06-16 20:13:18 +00:00
|
|
|
}
|
2019-09-19 17:09:59 +00:00
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
if constexpr (has_right_nulls)
|
|
|
|
{
|
|
|
|
if (const auto * right_nullable = checkAndGetColumn<ColumnNullable>(right_column))
|
2019-09-19 17:09:59 +00:00
|
|
|
{
|
|
|
|
if (right_column.isNullAt(rhs_pos))
|
|
|
|
return -null_direction_hint;
|
|
|
|
return left_column.compareAt(lhs_pos, rhs_pos, right_nullable->getNestedColumn(), null_direction_hint);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
return left_column.compareAt(lhs_pos, rhs_pos, right_column, null_direction_hint);
|
|
|
|
}
|
|
|
|
|
2021-04-02 14:51:02 +00:00
|
|
|
/// Get first and last row from sorted block
|
2019-09-23 19:36:47 +00:00
|
|
|
Block extractMinMax(const Block & block, const Block & keys)
|
|
|
|
{
|
|
|
|
if (block.rows() == 0)
|
|
|
|
throw Exception("Unexpected empty block", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
|
|
|
Block min_max = keys.cloneEmpty();
|
|
|
|
MutableColumns columns = min_max.mutateColumns();
|
|
|
|
|
|
|
|
for (size_t i = 0; i < columns.size(); ++i)
|
|
|
|
{
|
2021-04-02 13:55:42 +00:00
|
|
|
const auto & src_column = block.getByName(min_max.getByPosition(i).name);
|
2021-04-02 14:51:02 +00:00
|
|
|
|
|
|
|
columns[i]->insertFrom(*src_column.column, 0);
|
|
|
|
columns[i]->insertFrom(*src_column.column, block.rows() - 1);
|
2019-09-23 19:36:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
min_max.setColumns(std::move(columns));
|
|
|
|
return min_max;
|
|
|
|
}
|
|
|
|
|
2019-09-19 17:09:59 +00:00
|
|
|
}
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
|
|
|
|
class RowBitmaps
|
|
|
|
{
|
|
|
|
public:
|
|
|
|
struct Bitmap
|
|
|
|
{
|
|
|
|
using Container = std::vector<bool>;
|
|
|
|
|
|
|
|
std::mutex mutex;
|
|
|
|
Container bitmap;
|
|
|
|
|
|
|
|
size_t size() const { return bitmap.size(); }
|
|
|
|
bool empty() const { return bitmap.empty(); }
|
|
|
|
|
|
|
|
void applyOr(Container && addition) noexcept
|
|
|
|
{
|
|
|
|
std::lock_guard lock(mutex);
|
|
|
|
|
|
|
|
if (bitmap.empty())
|
|
|
|
{
|
|
|
|
bitmap.swap(addition);
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
|
|
|
|
/// TODO: simd bit or (need padding and tail in container)
|
|
|
|
for (size_t i = 0; i < bitmap.size(); ++i)
|
|
|
|
if (addition[i])
|
|
|
|
bitmap[i] = true;
|
|
|
|
}
|
|
|
|
};
|
|
|
|
|
|
|
|
using Container = Bitmap::Container;
|
|
|
|
|
|
|
|
explicit RowBitmaps(size_t size)
|
|
|
|
{
|
|
|
|
maps.reserve(size);
|
|
|
|
for (size_t i = 0; i < size; ++i)
|
|
|
|
maps.emplace_back(std::make_unique<Bitmap>());
|
|
|
|
}
|
|
|
|
|
|
|
|
bool used(size_t bitmap_number) const
|
|
|
|
{
|
|
|
|
return !maps[bitmap_number]->empty();
|
|
|
|
}
|
|
|
|
|
|
|
|
void applyOr(size_t bitmap_number, Container && addition) noexcept
|
|
|
|
{
|
|
|
|
maps[bitmap_number]->applyOr(std::move(addition));
|
|
|
|
}
|
|
|
|
|
|
|
|
IColumn::Filter getNotUsed(size_t bitmap_number) const
|
|
|
|
{
|
|
|
|
const Container & bitmap = maps[bitmap_number]->bitmap;
|
|
|
|
|
|
|
|
IColumn::Filter filter(bitmap.size());
|
|
|
|
for (size_t i = 0; i < bitmap.size(); ++i)
|
|
|
|
filter[i] = !bitmap[i];
|
|
|
|
return filter;
|
|
|
|
}
|
|
|
|
|
|
|
|
private:
|
|
|
|
std::vector<std::unique_ptr<Bitmap>> maps;
|
|
|
|
};
|
|
|
|
|
2019-09-13 16:17:37 +00:00
|
|
|
struct MergeJoinEqualRange
|
|
|
|
{
|
|
|
|
size_t left_start = 0;
|
|
|
|
size_t right_start = 0;
|
|
|
|
size_t left_length = 0;
|
|
|
|
size_t right_length = 0;
|
|
|
|
|
|
|
|
bool empty() const { return !left_length && !right_length; }
|
|
|
|
};
|
|
|
|
|
|
|
|
using Range = MergeJoinEqualRange;
|
|
|
|
|
|
|
|
|
|
|
|
class MergeJoinCursor
|
|
|
|
{
|
|
|
|
public:
|
|
|
|
MergeJoinCursor(const Block & block, const SortDescription & desc_)
|
2021-07-21 17:03:33 +00:00
|
|
|
: impl(block, desc_)
|
2020-12-09 15:07:58 +00:00
|
|
|
{
|
|
|
|
/// SortCursorImpl can work with permutation, but MergeJoinCursor can't.
|
|
|
|
if (impl.permutation)
|
|
|
|
throw Exception("Logical error: MergeJoinCursor doesn't support permutation", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
}
|
2019-09-13 16:17:37 +00:00
|
|
|
|
2020-12-04 16:25:30 +00:00
|
|
|
size_t position() const { return impl.getRow(); }
|
2019-09-16 19:31:22 +00:00
|
|
|
size_t end() const { return impl.rows; }
|
2020-12-04 16:25:30 +00:00
|
|
|
bool atEnd() const { return impl.getRow() >= impl.rows; }
|
2020-11-12 20:29:36 +00:00
|
|
|
void nextN(size_t num) { impl.getPosRef() += num; }
|
2019-09-13 16:17:37 +00:00
|
|
|
|
2021-05-15 11:26:59 +00:00
|
|
|
void setCompareNullability(const MergeJoinCursor & rhs)
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
2021-05-15 11:26:59 +00:00
|
|
|
has_left_nullable = false;
|
|
|
|
has_right_nullable = false;
|
2019-09-19 17:09:59 +00:00
|
|
|
|
2019-09-13 16:17:37 +00:00
|
|
|
for (size_t i = 0; i < impl.sort_columns_size; ++i)
|
|
|
|
{
|
2020-06-16 20:13:18 +00:00
|
|
|
has_left_nullable = has_left_nullable || isColumnNullable(*impl.sort_columns[i]);
|
|
|
|
has_right_nullable = has_right_nullable || isColumnNullable(*rhs.impl.sort_columns[i]);
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2019-09-19 17:09:59 +00:00
|
|
|
Range getNextEqualRange(MergeJoinCursor & rhs)
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
2020-06-16 20:13:18 +00:00
|
|
|
if (has_left_nullable && has_right_nullable)
|
2021-05-15 11:26:59 +00:00
|
|
|
return getNextEqualRangeImpl<true, true>(rhs);
|
2020-06-16 20:13:18 +00:00
|
|
|
else if (has_left_nullable)
|
2021-05-15 11:26:59 +00:00
|
|
|
return getNextEqualRangeImpl<true, false>(rhs);
|
2020-06-16 20:13:18 +00:00
|
|
|
else if (has_right_nullable)
|
2021-05-15 11:26:59 +00:00
|
|
|
return getNextEqualRangeImpl<false, true>(rhs);
|
|
|
|
return getNextEqualRangeImpl<false, false>(rhs);
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
int intersect(const Block & min_max, const Names & key_names)
|
2019-09-23 19:36:47 +00:00
|
|
|
{
|
|
|
|
if (end() == 0 || min_max.rows() != 2)
|
|
|
|
throw Exception("Unexpected block size", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
|
|
|
size_t last_position = end() - 1;
|
|
|
|
int first_vs_max = 0;
|
|
|
|
int last_vs_min = 0;
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
for (size_t i = 0; i < impl.sort_columns_size; ++i)
|
2019-09-23 19:36:47 +00:00
|
|
|
{
|
2020-04-22 06:01:33 +00:00
|
|
|
const auto & left_column = *impl.sort_columns[i];
|
|
|
|
const auto & right_column = *min_max.getByName(key_names[i]).column; /// cannot get by position cause of possible duplicates
|
2019-09-23 19:36:47 +00:00
|
|
|
|
|
|
|
if (!first_vs_max)
|
2021-05-15 11:26:59 +00:00
|
|
|
first_vs_max = nullableCompareAt<true, true>(left_column, right_column, position(), 1);
|
2019-09-23 19:36:47 +00:00
|
|
|
|
|
|
|
if (!last_vs_min)
|
2021-05-15 11:26:59 +00:00
|
|
|
last_vs_min = nullableCompareAt<true, true>(left_column, right_column, last_position, 0);
|
2019-09-23 19:36:47 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if (first_vs_max > 0)
|
|
|
|
return 1;
|
|
|
|
if (last_vs_min < 0)
|
|
|
|
return -1;
|
|
|
|
return 0;
|
|
|
|
}
|
|
|
|
|
2019-09-19 17:09:59 +00:00
|
|
|
private:
|
|
|
|
SortCursorImpl impl;
|
2020-06-16 20:13:18 +00:00
|
|
|
bool has_left_nullable = false;
|
|
|
|
bool has_right_nullable = false;
|
2019-09-13 16:17:37 +00:00
|
|
|
|
2021-05-15 11:26:59 +00:00
|
|
|
template <bool left_nulls, bool right_nulls>
|
2019-09-19 17:09:59 +00:00
|
|
|
Range getNextEqualRangeImpl(MergeJoinCursor & rhs)
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
|
|
|
while (!atEnd() && !rhs.atEnd())
|
|
|
|
{
|
2021-05-15 11:26:59 +00:00
|
|
|
int cmp = compareAtCursor<left_nulls, right_nulls>(rhs);
|
2019-09-13 16:17:37 +00:00
|
|
|
if (cmp < 0)
|
|
|
|
impl.next();
|
2020-06-16 20:13:18 +00:00
|
|
|
else if (cmp > 0)
|
2019-09-13 16:17:37 +00:00
|
|
|
rhs.impl.next();
|
2020-06-16 20:13:18 +00:00
|
|
|
else if (!cmp)
|
2020-12-04 16:25:30 +00:00
|
|
|
return Range{impl.getRow(), rhs.impl.getRow(), getEqualLength(), rhs.getEqualLength()};
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
|
|
|
|
2020-12-04 16:25:30 +00:00
|
|
|
return Range{impl.getRow(), rhs.impl.getRow(), 0, 0};
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
|
|
|
|
2021-05-15 11:26:59 +00:00
|
|
|
template <bool left_nulls, bool right_nulls>
|
2020-06-16 20:13:18 +00:00
|
|
|
int ALWAYS_INLINE compareAtCursor(const MergeJoinCursor & rhs) const
|
2019-09-19 17:09:59 +00:00
|
|
|
{
|
|
|
|
for (size_t i = 0; i < impl.sort_columns_size; ++i)
|
|
|
|
{
|
2020-04-22 06:01:33 +00:00
|
|
|
const auto * left_column = impl.sort_columns[i];
|
|
|
|
const auto * right_column = rhs.impl.sort_columns[i];
|
2019-09-19 17:09:59 +00:00
|
|
|
|
2021-05-15 11:26:59 +00:00
|
|
|
int res = nullableCompareAt<left_nulls, right_nulls>(*left_column, *right_column, impl.getRow(), rhs.impl.getRow());
|
2019-09-19 17:09:59 +00:00
|
|
|
if (res)
|
2020-06-16 20:13:18 +00:00
|
|
|
return res;
|
2019-09-19 17:09:59 +00:00
|
|
|
}
|
2020-06-16 20:13:18 +00:00
|
|
|
return 0;
|
2019-09-19 17:09:59 +00:00
|
|
|
}
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
/// Expects !atEnd()
|
2019-09-19 17:09:59 +00:00
|
|
|
size_t getEqualLength()
|
|
|
|
{
|
2020-12-04 16:25:30 +00:00
|
|
|
size_t pos = impl.getRow() + 1;
|
2020-06-16 20:13:18 +00:00
|
|
|
for (; pos < impl.rows; ++pos)
|
|
|
|
if (!samePrev(pos))
|
|
|
|
break;
|
2020-12-04 16:25:30 +00:00
|
|
|
return pos - impl.getRow();
|
2019-09-19 17:09:59 +00:00
|
|
|
}
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
/// Expects lhs_pos > 0
|
|
|
|
bool ALWAYS_INLINE samePrev(size_t lhs_pos) const
|
2019-09-19 17:09:59 +00:00
|
|
|
{
|
|
|
|
for (size_t i = 0; i < impl.sort_columns_size; ++i)
|
2020-06-16 20:13:18 +00:00
|
|
|
if (impl.sort_columns[i]->compareAt(lhs_pos - 1, lhs_pos, *(impl.sort_columns[i]), 1) != 0)
|
2019-09-19 17:09:59 +00:00
|
|
|
return false;
|
|
|
|
return true;
|
|
|
|
}
|
2019-09-13 16:17:37 +00:00
|
|
|
};
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
|
2019-09-16 19:31:22 +00:00
|
|
|
namespace
|
|
|
|
{
|
|
|
|
|
2019-09-17 16:55:11 +00:00
|
|
|
MutableColumns makeMutableColumns(const Block & block, size_t rows_to_reserve = 0)
|
2019-09-16 19:31:22 +00:00
|
|
|
{
|
|
|
|
MutableColumns columns;
|
|
|
|
columns.reserve(block.columns());
|
|
|
|
|
|
|
|
for (const auto & src_column : block)
|
2019-09-17 16:55:11 +00:00
|
|
|
{
|
2019-09-16 19:31:22 +00:00
|
|
|
columns.push_back(src_column.column->cloneEmpty());
|
2019-09-17 16:55:11 +00:00
|
|
|
columns.back()->reserve(rows_to_reserve);
|
|
|
|
}
|
2019-09-16 19:31:22 +00:00
|
|
|
return columns;
|
|
|
|
}
|
|
|
|
|
|
|
|
void makeSortAndMerge(const Names & keys, SortDescription & sort, SortDescription & merge)
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
|
|
|
NameSet unique_keys;
|
2021-07-21 17:03:33 +00:00
|
|
|
for (const auto & sd: merge)
|
|
|
|
unique_keys.insert(sd.column_name);
|
|
|
|
|
2020-04-22 06:01:33 +00:00
|
|
|
for (const auto & key_name : keys)
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
2021-07-21 17:03:33 +00:00
|
|
|
merge.emplace_back(key_name);
|
2019-09-13 16:17:37 +00:00
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
if (!unique_keys.contains(key_name))
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
|
|
|
unique_keys.insert(key_name);
|
2021-07-21 17:03:33 +00:00
|
|
|
sort.emplace_back(key_name);
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
|
|
|
}
|
2019-09-10 14:51:28 +00:00
|
|
|
}
|
|
|
|
|
2019-09-16 19:31:22 +00:00
|
|
|
void copyLeftRange(const Block & block, MutableColumns & columns, size_t start, size_t rows_to_add)
|
|
|
|
{
|
|
|
|
for (size_t i = 0; i < block.columns(); ++i)
|
|
|
|
{
|
2019-09-17 16:55:11 +00:00
|
|
|
const auto & src_column = block.getByPosition(i).column;
|
|
|
|
columns[i]->insertRangeFrom(*src_column, start, rows_to_add);
|
2019-09-16 19:31:22 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
void copyRightRange(const Block & right_block, const Block & right_columns_to_add, MutableColumns & columns,
|
|
|
|
size_t row_position, size_t rows_to_add)
|
|
|
|
{
|
|
|
|
for (size_t i = 0; i < right_columns_to_add.columns(); ++i)
|
|
|
|
{
|
2019-09-17 16:55:11 +00:00
|
|
|
const auto & src_column = right_block.getByName(right_columns_to_add.getByPosition(i).name).column;
|
2019-09-16 19:31:22 +00:00
|
|
|
auto & dst_column = columns[i];
|
2019-09-17 16:55:11 +00:00
|
|
|
auto * dst_nullable = typeid_cast<ColumnNullable *>(dst_column.get());
|
2019-09-16 19:31:22 +00:00
|
|
|
|
2019-09-17 16:55:11 +00:00
|
|
|
if (dst_nullable && !isColumnNullable(*src_column))
|
2019-09-18 18:44:44 +00:00
|
|
|
dst_nullable->insertManyFromNotNullable(*src_column, row_position, rows_to_add);
|
2019-09-17 16:55:11 +00:00
|
|
|
else
|
2019-09-18 18:44:44 +00:00
|
|
|
dst_column->insertManyFrom(*src_column, row_position, rows_to_add);
|
2019-09-16 19:31:22 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
void joinEqualsAnyLeft(const Block & right_block, const Block & right_columns_to_add, MutableColumns & right_columns, const Range & range)
|
|
|
|
{
|
|
|
|
copyRightRange(right_block, right_columns_to_add, right_columns, range.right_start, range.left_length);
|
|
|
|
}
|
|
|
|
|
2020-01-13 18:00:32 +00:00
|
|
|
template <bool is_all>
|
2020-01-15 20:33:29 +00:00
|
|
|
bool joinEquals(const Block & left_block, const Block & right_block, const Block & right_columns_to_add,
|
|
|
|
MutableColumns & left_columns, MutableColumns & right_columns, Range & range, size_t max_rows [[maybe_unused]])
|
2019-09-16 19:31:22 +00:00
|
|
|
{
|
2020-01-15 20:33:29 +00:00
|
|
|
bool one_more = true;
|
|
|
|
|
2020-01-13 18:00:32 +00:00
|
|
|
if constexpr (is_all)
|
2020-01-15 20:33:29 +00:00
|
|
|
{
|
|
|
|
size_t range_rows = range.left_length * range.right_length;
|
|
|
|
if (range_rows > max_rows)
|
2020-01-15 21:33:21 +00:00
|
|
|
{
|
2020-01-23 15:47:49 +00:00
|
|
|
/// We need progress. So we join at least one right row.
|
2020-01-15 21:33:21 +00:00
|
|
|
range.right_length = max_rows / range.left_length;
|
|
|
|
if (!range.right_length)
|
|
|
|
range.right_length = 1;
|
2020-01-15 20:33:29 +00:00
|
|
|
one_more = false;
|
2020-01-15 21:33:21 +00:00
|
|
|
}
|
2019-09-16 19:31:22 +00:00
|
|
|
|
2020-01-15 20:33:29 +00:00
|
|
|
size_t left_rows_to_add = range.left_length;
|
|
|
|
size_t row_position = range.right_start;
|
|
|
|
for (size_t right_row = 0; right_row < range.right_length; ++right_row, ++row_position)
|
|
|
|
{
|
|
|
|
copyLeftRange(left_block, left_columns, range.left_start, left_rows_to_add);
|
|
|
|
copyRightRange(right_block, right_columns_to_add, right_columns, row_position, left_rows_to_add);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
else
|
2019-09-16 19:31:22 +00:00
|
|
|
{
|
2020-01-15 20:33:29 +00:00
|
|
|
size_t left_rows_to_add = range.left_length;
|
2019-09-16 19:31:22 +00:00
|
|
|
copyLeftRange(left_block, left_columns, range.left_start, left_rows_to_add);
|
2020-01-15 20:33:29 +00:00
|
|
|
copyRightRange(right_block, right_columns_to_add, right_columns, range.right_start, left_rows_to_add);
|
2019-09-16 19:31:22 +00:00
|
|
|
}
|
2020-01-15 20:33:29 +00:00
|
|
|
|
|
|
|
return one_more;
|
2019-09-16 19:31:22 +00:00
|
|
|
}
|
|
|
|
|
2020-01-13 18:00:32 +00:00
|
|
|
template <bool copy_left>
|
2021-01-21 13:46:46 +00:00
|
|
|
void joinInequalsLeft(const Block & left_block, MutableColumns & left_columns,
|
|
|
|
const Block & right_block, MutableColumns & right_columns,
|
2020-01-13 18:00:32 +00:00
|
|
|
size_t start, size_t end)
|
2019-09-16 19:31:22 +00:00
|
|
|
{
|
|
|
|
if (end <= start)
|
|
|
|
return;
|
|
|
|
|
|
|
|
size_t rows_to_add = end - start;
|
2020-01-13 18:00:32 +00:00
|
|
|
if constexpr (copy_left)
|
2019-09-16 19:31:22 +00:00
|
|
|
copyLeftRange(left_block, left_columns, start, rows_to_add);
|
2020-01-13 18:00:32 +00:00
|
|
|
|
2021-01-21 13:46:46 +00:00
|
|
|
for (size_t i = 0; i < right_columns.size(); ++i)
|
|
|
|
{
|
2021-01-21 14:18:57 +00:00
|
|
|
JoinCommon::addDefaultValues(*right_columns[i], right_block.getByPosition(i).type, rows_to_add);
|
2021-01-21 13:46:46 +00:00
|
|
|
}
|
2019-09-16 19:31:22 +00:00
|
|
|
}
|
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2020-04-07 09:48:47 +00:00
|
|
|
MergeJoin::MergeJoin(std::shared_ptr<TableJoin> table_join_, const Block & right_sample_block_)
|
2019-09-09 19:43:37 +00:00
|
|
|
: table_join(table_join_)
|
2019-10-15 16:31:49 +00:00
|
|
|
, size_limits(table_join->sizeLimits())
|
|
|
|
, right_sample_block(right_sample_block_)
|
2019-09-18 12:24:35 +00:00
|
|
|
, nullable_right_side(table_join->forceNullableRight())
|
2020-07-10 18:10:06 +00:00
|
|
|
, nullable_left_side(table_join->forceNullableLeft())
|
2020-01-31 14:29:49 +00:00
|
|
|
, is_any_join(table_join->strictness() == ASTTableJoin::Strictness::Any)
|
2020-01-13 18:00:32 +00:00
|
|
|
, is_all_join(table_join->strictness() == ASTTableJoin::Strictness::All)
|
2020-01-31 14:29:49 +00:00
|
|
|
, is_semi_join(table_join->strictness() == ASTTableJoin::Strictness::Semi)
|
2019-09-16 19:31:22 +00:00
|
|
|
, is_inner(isInner(table_join->kind()))
|
|
|
|
, is_left(isLeft(table_join->kind()))
|
2020-07-10 18:10:06 +00:00
|
|
|
, is_right(isRight(table_join->kind()))
|
|
|
|
, is_full(isFull(table_join->kind()))
|
2020-01-17 11:45:43 +00:00
|
|
|
, max_joined_block_rows(table_join->maxJoinedBlockRows())
|
2019-10-15 16:31:49 +00:00
|
|
|
, max_rows_in_right_block(table_join->maxRowsInRightBlock())
|
2020-04-28 13:55:50 +00:00
|
|
|
, max_files_to_merge(table_join->maxFilesToMerge())
|
2019-09-09 19:43:37 +00:00
|
|
|
{
|
2020-02-21 13:16:59 +00:00
|
|
|
switch (table_join->strictness())
|
|
|
|
{
|
|
|
|
case ASTTableJoin::Strictness::All:
|
2020-07-10 18:10:06 +00:00
|
|
|
break;
|
|
|
|
case ASTTableJoin::Strictness::Any:
|
2020-02-21 13:16:59 +00:00
|
|
|
case ASTTableJoin::Strictness::Semi:
|
2020-07-10 18:10:06 +00:00
|
|
|
if (!is_left && !is_inner)
|
|
|
|
throw Exception("Not supported. MergeJoin supports SEMI and ANY variants only for LEFT and INNER JOINs.",
|
|
|
|
ErrorCodes::NOT_IMPLEMENTED);
|
2020-02-21 13:16:59 +00:00
|
|
|
break;
|
|
|
|
default:
|
2020-07-10 18:10:06 +00:00
|
|
|
throw Exception("Not supported. MergeJoin supports ALL, ANY and SEMI JOINs variants.", ErrorCodes::NOT_IMPLEMENTED);
|
2020-02-21 13:16:59 +00:00
|
|
|
}
|
2019-09-13 16:17:37 +00:00
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
if (!max_rows_in_right_block)
|
|
|
|
throw Exception("partial_merge_join_rows_in_right_blocks cannot be zero", ErrorCodes::PARAMETER_OUT_OF_BOUND);
|
|
|
|
|
2020-04-28 13:55:50 +00:00
|
|
|
if (max_files_to_merge < 2)
|
|
|
|
throw Exception("max_files_to_merge cannot be less than 2", ErrorCodes::PARAMETER_OUT_OF_BOUND);
|
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
if (!size_limits.hasLimits())
|
|
|
|
{
|
|
|
|
size_limits.max_bytes = table_join->defaultMaxBytes();
|
|
|
|
if (!size_limits.max_bytes)
|
|
|
|
throw Exception("No limit for MergeJoin (max_rows_in_join, max_bytes_in_join or default_max_bytes_in_join have to be set)",
|
|
|
|
ErrorCodes::PARAMETER_OUT_OF_BOUND);
|
|
|
|
}
|
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
std::tie(mask_column_name_left, mask_column_name_right) = table_join->joinConditionColumnNames();
|
|
|
|
|
|
|
|
/// Add auxiliary joining keys to join only rows where conditions from JOIN ON sections holds
|
|
|
|
/// Input boolean column converted to nullable and only rows with non NULLS value will be joined
|
|
|
|
if (!mask_column_name_left.empty() || !mask_column_name_right.empty())
|
|
|
|
{
|
|
|
|
JoinCommon::checkTypesOfMasks({}, "", right_sample_block, mask_column_name_right);
|
|
|
|
|
|
|
|
key_names_left.push_back(deriveTempName(mask_column_name_left));
|
|
|
|
key_names_right.push_back(deriveTempName(mask_column_name_right));
|
|
|
|
}
|
|
|
|
|
|
|
|
key_names_left.insert(key_names_left.end(), table_join->keyNamesLeft().begin(), table_join->keyNamesLeft().end());
|
|
|
|
key_names_right.insert(key_names_right.end(), table_join->keyNamesRight().begin(), table_join->keyNamesRight().end());
|
|
|
|
|
|
|
|
addConditionJoinColumn(right_sample_block, JoinTableSide::Right);
|
|
|
|
JoinCommon::splitAdditionalColumns(key_names_right, right_sample_block, right_table_keys, right_columns_to_add);
|
|
|
|
|
|
|
|
for (const auto & right_key : key_names_right)
|
2021-05-17 11:18:03 +00:00
|
|
|
{
|
|
|
|
if (right_sample_block.getByName(right_key).type->lowCardinality())
|
|
|
|
lowcard_right_keys.push_back(right_key);
|
|
|
|
}
|
2021-05-15 11:26:59 +00:00
|
|
|
JoinCommon::removeLowCardinalityInplace(right_table_keys);
|
2021-07-21 17:03:33 +00:00
|
|
|
JoinCommon::removeLowCardinalityInplace(right_sample_block, key_names_right);
|
2019-09-12 14:09:05 +00:00
|
|
|
|
2019-09-16 12:37:46 +00:00
|
|
|
const NameSet required_right_keys = table_join->requiredRightKeys();
|
2019-09-12 14:09:05 +00:00
|
|
|
for (const auto & column : right_table_keys)
|
|
|
|
if (required_right_keys.count(column.name))
|
|
|
|
right_columns_to_add.insert(ColumnWithTypeAndName{nullptr, column.type, column.name});
|
|
|
|
|
|
|
|
JoinCommon::createMissedColumns(right_columns_to_add);
|
|
|
|
|
|
|
|
if (nullable_right_side)
|
|
|
|
JoinCommon::convertColumnsToNullable(right_columns_to_add);
|
2019-09-12 18:06:25 +00:00
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
makeSortAndMerge(key_names_left, left_sort_description, left_merge_description);
|
|
|
|
makeSortAndMerge(key_names_right, right_sort_description, right_merge_description);
|
2020-06-16 20:13:18 +00:00
|
|
|
|
|
|
|
/// Temporary disable 'partial_merge_join_left_table_buffer_bytes' without 'partial_merge_join_optimizations'
|
|
|
|
if (table_join->enablePartialMergeJoinOptimizations())
|
|
|
|
if (size_t max_bytes = table_join->maxBytesInLeftBuffer())
|
|
|
|
left_blocks_buffer = std::make_shared<SortedBlocksBuffer>(left_sort_description, max_bytes);
|
2019-09-12 18:06:25 +00:00
|
|
|
}
|
|
|
|
|
2021-04-02 18:07:11 +00:00
|
|
|
/// Has to be called even if totals are empty
|
2019-09-12 18:06:25 +00:00
|
|
|
void MergeJoin::setTotals(const Block & totals_block)
|
|
|
|
{
|
|
|
|
totals = totals_block;
|
|
|
|
mergeRightBlocks();
|
2020-07-10 18:10:06 +00:00
|
|
|
|
|
|
|
if (is_right || is_full)
|
|
|
|
used_rows_bitmap = std::make_shared<RowBitmaps>(getRightBlocksCount());
|
2019-09-09 19:43:37 +00:00
|
|
|
}
|
|
|
|
|
2019-09-13 16:17:37 +00:00
|
|
|
void MergeJoin::mergeRightBlocks()
|
|
|
|
{
|
2019-10-15 16:31:49 +00:00
|
|
|
if (is_in_memory)
|
|
|
|
mergeInMemoryRightBlocks();
|
|
|
|
else
|
|
|
|
mergeFlushedRightBlocks();
|
|
|
|
}
|
|
|
|
|
|
|
|
void MergeJoin::mergeInMemoryRightBlocks()
|
|
|
|
{
|
|
|
|
std::unique_lock lock(rwlock);
|
|
|
|
|
2019-09-16 19:31:22 +00:00
|
|
|
if (right_blocks.empty())
|
|
|
|
return;
|
|
|
|
|
2021-07-22 16:05:52 +00:00
|
|
|
Pipe source(std::make_shared<BlocksListSource>(std::move(right_blocks.blocks)));
|
2020-04-28 13:55:50 +00:00
|
|
|
right_blocks.clear();
|
2019-09-23 19:36:47 +00:00
|
|
|
|
2020-06-01 16:58:36 +00:00
|
|
|
QueryPipeline pipeline;
|
2020-06-02 05:16:45 +00:00
|
|
|
pipeline.init(std::move(source));
|
2020-06-01 16:58:36 +00:00
|
|
|
|
2020-08-08 01:01:47 +00:00
|
|
|
/// TODO: there should be no split keys by blocks for RIGHT|FULL JOIN
|
2021-07-21 17:03:33 +00:00
|
|
|
pipeline.addTransform(std::make_shared<MergeSortingTransform>(
|
|
|
|
pipeline.getHeader(), right_sort_description, max_rows_in_right_block, 0, 0, 0, 0, nullptr, 0));
|
2020-06-01 16:58:36 +00:00
|
|
|
|
|
|
|
auto sorted_input = PipelineExecutingBlockInputStream(std::move(pipeline));
|
2019-09-13 16:17:37 +00:00
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
while (Block block = sorted_input.read())
|
|
|
|
{
|
|
|
|
if (!block.rows())
|
|
|
|
continue;
|
|
|
|
|
|
|
|
if (skip_not_intersected)
|
|
|
|
min_max_right_blocks.emplace_back(extractMinMax(block, right_table_keys));
|
2020-04-28 13:55:50 +00:00
|
|
|
right_blocks.countBlockSize(block);
|
2019-10-15 16:31:49 +00:00
|
|
|
loaded_right_blocks.emplace_back(std::make_shared<Block>(std::move(block)));
|
|
|
|
}
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
void MergeJoin::mergeFlushedRightBlocks()
|
2019-09-09 19:43:37 +00:00
|
|
|
{
|
2019-10-15 16:31:49 +00:00
|
|
|
std::unique_lock lock(rwlock);
|
2019-09-17 18:53:52 +00:00
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
auto callback = [&](const Block & block)
|
|
|
|
{
|
|
|
|
if (skip_not_intersected)
|
|
|
|
min_max_right_blocks.emplace_back(extractMinMax(block, right_table_keys));
|
2020-04-28 13:55:50 +00:00
|
|
|
right_blocks.countBlockSize(block);
|
2019-10-15 16:31:49 +00:00
|
|
|
};
|
|
|
|
|
2020-04-28 13:55:50 +00:00
|
|
|
flushed_right_blocks = disk_writer->finishMerge(callback);
|
|
|
|
disk_writer.reset();
|
2019-10-15 16:31:49 +00:00
|
|
|
|
2020-01-11 09:50:41 +00:00
|
|
|
/// Get memory limit or approximate it from row limit and bytes per row factor
|
2019-10-15 16:31:49 +00:00
|
|
|
UInt64 memory_limit = size_limits.max_bytes;
|
|
|
|
UInt64 rows_limit = size_limits.max_rows;
|
|
|
|
if (!memory_limit && rows_limit)
|
2020-04-28 13:55:50 +00:00
|
|
|
memory_limit = right_blocks.bytes * rows_limit / right_blocks.row_count;
|
2019-10-15 16:31:49 +00:00
|
|
|
|
|
|
|
cached_right_blocks = std::make_unique<Cache>(memory_limit);
|
|
|
|
}
|
|
|
|
|
|
|
|
bool MergeJoin::saveRightBlock(Block && block)
|
|
|
|
{
|
2020-04-28 13:55:50 +00:00
|
|
|
if (is_in_memory)
|
|
|
|
{
|
|
|
|
std::unique_lock lock(rwlock);
|
2019-09-10 14:51:28 +00:00
|
|
|
|
2020-04-28 13:55:50 +00:00
|
|
|
if (!is_in_memory)
|
|
|
|
{
|
|
|
|
disk_writer->insert(std::move(block));
|
|
|
|
return true;
|
|
|
|
}
|
2019-10-15 16:31:49 +00:00
|
|
|
|
2020-04-28 13:55:50 +00:00
|
|
|
right_blocks.insert(std::move(block));
|
|
|
|
|
|
|
|
bool has_memory = size_limits.softCheck(right_blocks.row_count, right_blocks.bytes);
|
|
|
|
if (!has_memory)
|
|
|
|
{
|
2020-06-16 20:13:18 +00:00
|
|
|
initRightTableWriter();
|
2020-04-28 13:55:50 +00:00
|
|
|
is_in_memory = false;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
else
|
|
|
|
disk_writer->insert(std::move(block));
|
2019-10-15 16:31:49 +00:00
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
Block MergeJoin::modifyRightBlock(const Block & src_block) const
|
2019-10-15 16:31:49 +00:00
|
|
|
{
|
|
|
|
Block block = materializeBlock(src_block);
|
2021-05-15 11:26:59 +00:00
|
|
|
JoinCommon::removeLowCardinalityInplace(block, table_join->keyNamesRight());
|
2020-07-10 18:10:06 +00:00
|
|
|
return block;
|
|
|
|
}
|
|
|
|
|
|
|
|
bool MergeJoin::addJoinedBlock(const Block & src_block, bool)
|
|
|
|
{
|
|
|
|
Block block = modifyRightBlock(src_block);
|
2019-09-10 14:51:28 +00:00
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
addConditionJoinColumn(block, JoinTableSide::Right);
|
2019-10-15 16:31:49 +00:00
|
|
|
sortBlock(block, right_sort_description);
|
|
|
|
return saveRightBlock(std::move(block));
|
2019-09-09 19:43:37 +00:00
|
|
|
}
|
|
|
|
|
2020-01-15 20:33:29 +00:00
|
|
|
void MergeJoin::joinBlock(Block & block, ExtraBlockPtr & not_processed)
|
2019-09-09 19:43:37 +00:00
|
|
|
{
|
2021-05-17 11:18:03 +00:00
|
|
|
Names lowcard_keys = lowcard_right_keys;
|
2020-06-16 20:13:18 +00:00
|
|
|
if (block)
|
|
|
|
{
|
2021-07-21 17:03:33 +00:00
|
|
|
JoinCommon::checkTypesOfMasks(block, mask_column_name_left, right_sample_block, mask_column_name_right);
|
|
|
|
|
|
|
|
/// Add auxiliary column, will be removed after joining
|
|
|
|
addConditionJoinColumn(block, JoinTableSide::Left);
|
|
|
|
|
|
|
|
JoinCommon::checkTypesOfKeys(block, key_names_left, right_table_keys, key_names_right);
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
materializeBlockInplace(block);
|
2021-05-17 11:18:03 +00:00
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
for (const auto & column_name : key_names_left)
|
2021-05-17 11:18:03 +00:00
|
|
|
{
|
|
|
|
if (block.getByName(column_name).type->lowCardinality())
|
|
|
|
lowcard_keys.push_back(column_name);
|
|
|
|
}
|
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
JoinCommon::removeLowCardinalityInplace(block, key_names_left, false);
|
2020-06-16 20:13:18 +00:00
|
|
|
|
|
|
|
sortBlock(block, left_sort_description);
|
2020-07-10 18:10:06 +00:00
|
|
|
|
|
|
|
if (nullable_left_side)
|
|
|
|
JoinCommon::convertColumnsToNullable(block);
|
2020-06-16 20:13:18 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if (!not_processed && left_blocks_buffer)
|
|
|
|
{
|
|
|
|
if (!block || block.rows())
|
|
|
|
block = left_blocks_buffer->exchange(std::move(block));
|
|
|
|
if (!block)
|
|
|
|
return;
|
|
|
|
}
|
2019-09-17 18:53:52 +00:00
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
if (is_in_memory)
|
2020-01-13 18:00:32 +00:00
|
|
|
{
|
|
|
|
if (is_all_join)
|
2020-01-14 19:02:22 +00:00
|
|
|
joinSortedBlock<true, true>(block, not_processed);
|
2020-01-13 18:00:32 +00:00
|
|
|
else
|
2020-01-14 19:02:22 +00:00
|
|
|
joinSortedBlock<true, false>(block, not_processed);
|
2020-01-13 18:00:32 +00:00
|
|
|
}
|
2019-10-15 16:31:49 +00:00
|
|
|
else
|
2020-01-13 18:00:32 +00:00
|
|
|
{
|
|
|
|
if (is_all_join)
|
2020-01-14 19:02:22 +00:00
|
|
|
joinSortedBlock<false, true>(block, not_processed);
|
2020-01-13 18:00:32 +00:00
|
|
|
else
|
2020-01-14 19:02:22 +00:00
|
|
|
joinSortedBlock<false, false>(block, not_processed);
|
2020-01-13 18:00:32 +00:00
|
|
|
}
|
2020-06-16 20:13:18 +00:00
|
|
|
|
|
|
|
/// Back thread even with no data. We have some unfinished data in buffer.
|
|
|
|
if (!not_processed && left_blocks_buffer)
|
|
|
|
not_processed = std::make_shared<NotProcessed>(NotProcessed{{}, 0, 0, 0});
|
2020-09-22 10:16:22 +00:00
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
if (needConditionJoinColumn())
|
|
|
|
block.erase(deriveTempName(mask_column_name_left));
|
|
|
|
|
2021-08-06 14:15:11 +00:00
|
|
|
JoinCommon::restoreLowCardinalityInplace(block, lowcard_keys);
|
2019-10-15 16:31:49 +00:00
|
|
|
}
|
2019-09-12 18:06:25 +00:00
|
|
|
|
2020-01-13 18:00:32 +00:00
|
|
|
template <bool in_memory, bool is_all>
|
2020-01-15 20:33:29 +00:00
|
|
|
void MergeJoin::joinSortedBlock(Block & block, ExtraBlockPtr & not_processed)
|
2019-10-15 16:31:49 +00:00
|
|
|
{
|
2019-09-17 16:55:11 +00:00
|
|
|
size_t rows_to_reserve = is_left ? block.rows() : 0;
|
|
|
|
MutableColumns left_columns = makeMutableColumns(block, (is_all ? rows_to_reserve : 0));
|
|
|
|
MutableColumns right_columns = makeMutableColumns(right_columns_to_add, rows_to_reserve);
|
2019-09-16 19:31:22 +00:00
|
|
|
MergeJoinCursor left_cursor(block, left_merge_description);
|
2019-09-17 18:53:52 +00:00
|
|
|
size_t left_key_tail = 0;
|
2020-01-15 21:33:21 +00:00
|
|
|
size_t skip_right = 0;
|
2019-10-15 16:31:49 +00:00
|
|
|
size_t right_blocks_count = rightBlocksCount<in_memory>();
|
2019-09-13 16:17:37 +00:00
|
|
|
|
2020-01-15 20:33:29 +00:00
|
|
|
size_t starting_right_block = 0;
|
|
|
|
if (not_processed)
|
|
|
|
{
|
|
|
|
auto & continuation = static_cast<NotProcessed &>(*not_processed);
|
2020-01-15 21:33:21 +00:00
|
|
|
left_cursor.nextN(continuation.left_position);
|
|
|
|
skip_right = continuation.right_position;
|
|
|
|
starting_right_block = continuation.right_block;
|
2020-01-15 20:33:29 +00:00
|
|
|
not_processed.reset();
|
|
|
|
}
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
bool with_left_inequals = (is_left && !is_semi_join) || is_full;
|
2020-01-31 14:29:49 +00:00
|
|
|
if (with_left_inequals)
|
2019-09-16 19:31:22 +00:00
|
|
|
{
|
2020-01-15 20:33:29 +00:00
|
|
|
for (size_t i = starting_right_block; i < right_blocks_count; ++i)
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
|
|
|
if (left_cursor.atEnd())
|
|
|
|
break;
|
2019-09-23 19:36:47 +00:00
|
|
|
|
|
|
|
if (skip_not_intersected)
|
|
|
|
{
|
2021-07-21 17:03:33 +00:00
|
|
|
int intersection = left_cursor.intersect(min_max_right_blocks[i], key_names_right);
|
2019-09-23 19:36:47 +00:00
|
|
|
if (intersection < 0)
|
|
|
|
break; /// (left) ... (right)
|
|
|
|
if (intersection > 0)
|
|
|
|
continue; /// (right) ... (left)
|
|
|
|
}
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
/// Use skip_right as ref. It would be updated in join.
|
|
|
|
RightBlockInfo right_block(loadRightBlock<in_memory>(i), i, skip_right, used_rows_bitmap.get());
|
2019-10-15 16:31:49 +00:00
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
if (!leftJoin<is_all>(left_cursor, block, right_block, left_columns, right_columns, left_key_tail))
|
2020-01-14 19:02:22 +00:00
|
|
|
{
|
2020-01-15 21:33:21 +00:00
|
|
|
not_processed = extraBlock<is_all>(block, std::move(left_columns), std::move(right_columns),
|
|
|
|
left_cursor.position(), skip_right, i);
|
2020-01-15 20:33:29 +00:00
|
|
|
return;
|
2020-01-14 19:02:22 +00:00
|
|
|
}
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
2019-09-11 18:03:21 +00:00
|
|
|
|
2019-09-17 18:53:52 +00:00
|
|
|
left_cursor.nextN(left_key_tail);
|
2021-01-21 13:46:46 +00:00
|
|
|
joinInequalsLeft<is_all>(block, left_columns, right_columns_to_add, right_columns, left_cursor.position(), left_cursor.end());
|
2019-09-16 19:31:22 +00:00
|
|
|
|
|
|
|
changeLeftColumns(block, std::move(left_columns));
|
|
|
|
addRightColumns(block, std::move(right_columns));
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
2020-01-31 14:29:49 +00:00
|
|
|
else /// no inequals
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
2020-01-15 20:33:29 +00:00
|
|
|
for (size_t i = starting_right_block; i < right_blocks_count; ++i)
|
2019-09-13 17:23:32 +00:00
|
|
|
{
|
|
|
|
if (left_cursor.atEnd())
|
|
|
|
break;
|
2019-09-23 19:36:47 +00:00
|
|
|
|
|
|
|
if (skip_not_intersected)
|
|
|
|
{
|
2021-07-21 17:03:33 +00:00
|
|
|
int intersection = left_cursor.intersect(min_max_right_blocks[i], key_names_right);
|
2019-09-23 19:36:47 +00:00
|
|
|
if (intersection < 0)
|
|
|
|
break; /// (left) ... (right)
|
|
|
|
if (intersection > 0)
|
|
|
|
continue; /// (right) ... (left)
|
|
|
|
}
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
/// Use skip_right as ref. It would be updated in join.
|
|
|
|
RightBlockInfo right_block(loadRightBlock<in_memory>(i), i, skip_right, used_rows_bitmap.get());
|
2019-10-15 16:31:49 +00:00
|
|
|
|
2020-01-31 14:29:49 +00:00
|
|
|
if constexpr (is_all)
|
2020-01-14 19:02:22 +00:00
|
|
|
{
|
2020-07-10 18:10:06 +00:00
|
|
|
if (!allInnerJoin(left_cursor, block, right_block, left_columns, right_columns, left_key_tail))
|
2020-01-31 14:29:49 +00:00
|
|
|
{
|
|
|
|
not_processed = extraBlock<is_all>(block, std::move(left_columns), std::move(right_columns),
|
|
|
|
left_cursor.position(), skip_right, i);
|
|
|
|
return;
|
|
|
|
}
|
2020-01-14 19:02:22 +00:00
|
|
|
}
|
2020-01-31 14:29:49 +00:00
|
|
|
else
|
2020-07-10 18:10:06 +00:00
|
|
|
semiLeftJoin(left_cursor, block, right_block, left_columns, right_columns);
|
2019-09-13 17:23:32 +00:00
|
|
|
}
|
|
|
|
|
2019-09-17 18:53:52 +00:00
|
|
|
left_cursor.nextN(left_key_tail);
|
2019-09-16 19:31:22 +00:00
|
|
|
changeLeftColumns(block, std::move(left_columns));
|
|
|
|
addRightColumns(block, std::move(right_columns));
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
2019-09-10 14:51:28 +00:00
|
|
|
}
|
|
|
|
|
2020-01-15 20:33:29 +00:00
|
|
|
static size_t maxRangeRows(size_t current_rows, size_t max_rows)
|
|
|
|
{
|
|
|
|
if (!max_rows)
|
|
|
|
return std::numeric_limits<size_t>::max();
|
|
|
|
if (current_rows >= max_rows)
|
|
|
|
return 0;
|
|
|
|
return max_rows - current_rows;
|
|
|
|
}
|
|
|
|
|
2020-01-13 18:00:32 +00:00
|
|
|
template <bool is_all>
|
2020-07-10 18:10:06 +00:00
|
|
|
bool MergeJoin::leftJoin(MergeJoinCursor & left_cursor, const Block & left_block, RightBlockInfo & right_block_info,
|
|
|
|
MutableColumns & left_columns, MutableColumns & right_columns, size_t & left_key_tail)
|
2019-09-10 14:51:28 +00:00
|
|
|
{
|
2020-07-10 18:10:06 +00:00
|
|
|
const Block & right_block = *right_block_info.block;
|
2019-09-13 16:17:37 +00:00
|
|
|
MergeJoinCursor right_cursor(right_block, right_merge_description);
|
2021-05-15 11:26:59 +00:00
|
|
|
left_cursor.setCompareNullability(right_cursor);
|
2019-09-13 16:17:37 +00:00
|
|
|
|
2020-01-15 21:33:21 +00:00
|
|
|
/// Set right cursor position in first continuation right block
|
|
|
|
if constexpr (is_all)
|
|
|
|
{
|
2020-07-10 18:10:06 +00:00
|
|
|
right_cursor.nextN(right_block_info.skip);
|
|
|
|
right_block_info.skip = 0;
|
2020-01-15 21:33:21 +00:00
|
|
|
}
|
|
|
|
|
2019-09-13 16:17:37 +00:00
|
|
|
while (!left_cursor.atEnd() && !right_cursor.atEnd())
|
|
|
|
{
|
2019-09-24 18:21:57 +00:00
|
|
|
/// Not zero left_key_tail means there were equality for the last left key in previous leftJoin() call.
|
|
|
|
/// Do not join it twice: join only if it's equal with a first right key of current leftJoin() call and skip otherwise.
|
|
|
|
size_t left_unequal_position = left_cursor.position() + left_key_tail;
|
|
|
|
left_key_tail = 0;
|
|
|
|
|
2019-09-13 16:17:37 +00:00
|
|
|
Range range = left_cursor.getNextEqualRange(right_cursor);
|
|
|
|
|
2021-01-21 13:46:46 +00:00
|
|
|
joinInequalsLeft<is_all>(left_block, left_columns, right_columns_to_add, right_columns, left_unequal_position, range.left_start);
|
2019-09-13 16:17:37 +00:00
|
|
|
|
|
|
|
if (range.empty())
|
|
|
|
break;
|
|
|
|
|
2020-01-13 18:00:32 +00:00
|
|
|
if constexpr (is_all)
|
2020-01-15 20:33:29 +00:00
|
|
|
{
|
2020-07-10 18:10:06 +00:00
|
|
|
right_block_info.setUsed(range.right_start, range.right_length);
|
|
|
|
|
2020-01-23 15:47:49 +00:00
|
|
|
size_t max_rows = maxRangeRows(left_columns[0]->size(), max_joined_block_rows);
|
2020-01-15 20:33:29 +00:00
|
|
|
|
|
|
|
if (!joinEquals<true>(left_block, right_block, right_columns_to_add, left_columns, right_columns, range, max_rows))
|
2020-01-15 21:33:21 +00:00
|
|
|
{
|
|
|
|
right_cursor.nextN(range.right_length);
|
2020-07-10 18:10:06 +00:00
|
|
|
right_block_info.skip = right_cursor.position();
|
2020-01-15 20:33:29 +00:00
|
|
|
return false;
|
2020-01-15 21:33:21 +00:00
|
|
|
}
|
2020-01-15 20:33:29 +00:00
|
|
|
}
|
2019-09-16 19:31:22 +00:00
|
|
|
else
|
|
|
|
joinEqualsAnyLeft(right_block, right_columns_to_add, right_columns, range);
|
|
|
|
|
2019-09-13 16:17:37 +00:00
|
|
|
right_cursor.nextN(range.right_length);
|
|
|
|
|
2019-09-16 19:31:22 +00:00
|
|
|
/// Do not run over last left keys for ALL JOIN (cause of possible duplicates in next right block)
|
2020-01-13 18:00:32 +00:00
|
|
|
if constexpr (is_all)
|
2019-09-17 18:53:52 +00:00
|
|
|
{
|
2020-01-13 18:00:32 +00:00
|
|
|
if (right_cursor.atEnd())
|
|
|
|
{
|
|
|
|
left_key_tail = range.left_length;
|
|
|
|
break;
|
|
|
|
}
|
2019-09-17 18:53:52 +00:00
|
|
|
}
|
2019-09-13 16:17:37 +00:00
|
|
|
left_cursor.nextN(range.left_length);
|
|
|
|
}
|
2020-01-15 20:33:29 +00:00
|
|
|
|
|
|
|
return true;
|
2019-09-10 14:51:28 +00:00
|
|
|
}
|
2019-09-09 19:43:37 +00:00
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
bool MergeJoin::allInnerJoin(MergeJoinCursor & left_cursor, const Block & left_block, RightBlockInfo & right_block_info,
|
2021-07-21 17:03:33 +00:00
|
|
|
MutableColumns & left_columns, MutableColumns & right_columns, size_t & left_key_tail)
|
2019-09-13 17:23:32 +00:00
|
|
|
{
|
2020-07-10 18:10:06 +00:00
|
|
|
const Block & right_block = *right_block_info.block;
|
2019-09-13 17:23:32 +00:00
|
|
|
MergeJoinCursor right_cursor(right_block, right_merge_description);
|
2021-05-15 11:26:59 +00:00
|
|
|
left_cursor.setCompareNullability(right_cursor);
|
2019-09-13 17:23:32 +00:00
|
|
|
|
2020-01-15 21:33:21 +00:00
|
|
|
/// Set right cursor position in first continuation right block
|
2020-07-10 18:10:06 +00:00
|
|
|
right_cursor.nextN(right_block_info.skip);
|
|
|
|
right_block_info.skip = 0;
|
2020-01-15 21:33:21 +00:00
|
|
|
|
2019-09-13 17:23:32 +00:00
|
|
|
while (!left_cursor.atEnd() && !right_cursor.atEnd())
|
|
|
|
{
|
|
|
|
Range range = left_cursor.getNextEqualRange(right_cursor);
|
|
|
|
if (range.empty())
|
|
|
|
break;
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
right_block_info.setUsed(range.right_start, range.right_length);
|
|
|
|
|
2020-01-31 14:29:49 +00:00
|
|
|
size_t max_rows = maxRangeRows(left_columns[0]->size(), max_joined_block_rows);
|
2020-01-15 20:33:29 +00:00
|
|
|
|
2020-01-31 14:29:49 +00:00
|
|
|
if (!joinEquals<true>(left_block, right_block, right_columns_to_add, left_columns, right_columns, range, max_rows))
|
|
|
|
{
|
|
|
|
right_cursor.nextN(range.right_length);
|
2020-07-10 18:10:06 +00:00
|
|
|
right_block_info.skip = right_cursor.position();
|
2020-01-31 14:29:49 +00:00
|
|
|
return false;
|
2020-01-15 20:33:29 +00:00
|
|
|
}
|
|
|
|
|
2019-09-13 17:23:32 +00:00
|
|
|
right_cursor.nextN(range.right_length);
|
|
|
|
|
2019-09-16 19:31:22 +00:00
|
|
|
/// Do not run over last left keys for ALL JOIN (cause of possible duplicates in next right block)
|
2020-01-31 14:29:49 +00:00
|
|
|
if (right_cursor.atEnd())
|
2019-09-17 18:53:52 +00:00
|
|
|
{
|
2020-01-31 14:29:49 +00:00
|
|
|
left_key_tail = range.left_length;
|
|
|
|
break;
|
2019-09-17 18:53:52 +00:00
|
|
|
}
|
2019-09-13 17:23:32 +00:00
|
|
|
left_cursor.nextN(range.left_length);
|
|
|
|
}
|
2020-01-15 20:33:29 +00:00
|
|
|
|
|
|
|
return true;
|
2019-09-13 17:23:32 +00:00
|
|
|
}
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
bool MergeJoin::semiLeftJoin(MergeJoinCursor & left_cursor, const Block & left_block, const RightBlockInfo & right_block_info,
|
2020-01-31 14:29:49 +00:00
|
|
|
MutableColumns & left_columns, MutableColumns & right_columns)
|
|
|
|
{
|
2020-07-10 18:10:06 +00:00
|
|
|
const Block & right_block = *right_block_info.block;
|
2020-01-31 14:29:49 +00:00
|
|
|
MergeJoinCursor right_cursor(right_block, right_merge_description);
|
2021-05-15 11:26:59 +00:00
|
|
|
left_cursor.setCompareNullability(right_cursor);
|
2020-01-31 14:29:49 +00:00
|
|
|
|
|
|
|
while (!left_cursor.atEnd() && !right_cursor.atEnd())
|
|
|
|
{
|
|
|
|
Range range = left_cursor.getNextEqualRange(right_cursor);
|
|
|
|
if (range.empty())
|
|
|
|
break;
|
|
|
|
|
|
|
|
joinEquals<false>(left_block, right_block, right_columns_to_add, left_columns, right_columns, range, 0);
|
|
|
|
|
|
|
|
right_cursor.nextN(range.right_length);
|
|
|
|
left_cursor.nextN(range.left_length);
|
|
|
|
}
|
|
|
|
|
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
2020-04-22 06:01:33 +00:00
|
|
|
void MergeJoin::changeLeftColumns(Block & block, MutableColumns && columns) const
|
2019-09-12 18:06:25 +00:00
|
|
|
{
|
2020-01-31 14:29:49 +00:00
|
|
|
if (is_left && is_any_join)
|
2019-09-16 19:31:22 +00:00
|
|
|
return;
|
|
|
|
block.setColumns(std::move(columns));
|
2019-09-13 16:17:37 +00:00
|
|
|
}
|
2019-09-12 18:06:25 +00:00
|
|
|
|
2019-09-16 19:31:22 +00:00
|
|
|
void MergeJoin::addRightColumns(Block & block, MutableColumns && right_columns)
|
2019-09-13 16:17:37 +00:00
|
|
|
{
|
|
|
|
for (size_t i = 0; i < right_columns_to_add.columns(); ++i)
|
|
|
|
{
|
|
|
|
const auto & column = right_columns_to_add.getByPosition(i);
|
|
|
|
block.insert(ColumnWithTypeAndName{std::move(right_columns[i]), column.type, column.name});
|
|
|
|
}
|
|
|
|
}
|
2019-09-12 18:06:25 +00:00
|
|
|
|
2020-01-14 19:02:22 +00:00
|
|
|
/// Split block into processed (result) and not processed. Not processed block would be joined next time.
|
2020-01-15 20:33:29 +00:00
|
|
|
template <bool is_all>
|
|
|
|
ExtraBlockPtr MergeJoin::extraBlock(Block & processed, MutableColumns && left_columns, MutableColumns && right_columns,
|
2020-01-15 21:33:21 +00:00
|
|
|
size_t left_position [[maybe_unused]], size_t right_position [[maybe_unused]],
|
|
|
|
size_t right_block_number [[maybe_unused]])
|
2020-01-14 19:02:22 +00:00
|
|
|
{
|
2020-01-15 20:33:29 +00:00
|
|
|
ExtraBlockPtr not_processed;
|
2020-01-14 19:02:22 +00:00
|
|
|
|
2020-01-15 20:33:29 +00:00
|
|
|
if constexpr (is_all)
|
|
|
|
{
|
2020-01-15 21:33:21 +00:00
|
|
|
not_processed = std::make_shared<NotProcessed>(
|
|
|
|
NotProcessed{{processed.cloneEmpty()}, left_position, right_position, right_block_number});
|
2020-01-15 20:33:29 +00:00
|
|
|
not_processed->block.swap(processed);
|
2020-01-14 19:02:22 +00:00
|
|
|
|
2020-01-15 20:33:29 +00:00
|
|
|
changeLeftColumns(processed, std::move(left_columns));
|
|
|
|
addRightColumns(processed, std::move(right_columns));
|
2020-01-14 19:02:22 +00:00
|
|
|
}
|
|
|
|
|
2020-01-15 20:33:29 +00:00
|
|
|
return not_processed;
|
2020-01-14 19:02:22 +00:00
|
|
|
}
|
|
|
|
|
2019-10-15 16:31:49 +00:00
|
|
|
template <bool in_memory>
|
2020-07-10 18:10:06 +00:00
|
|
|
size_t MergeJoin::rightBlocksCount() const
|
2019-10-15 16:31:49 +00:00
|
|
|
{
|
|
|
|
if constexpr (!in_memory)
|
|
|
|
return flushed_right_blocks.size();
|
|
|
|
else
|
|
|
|
return loaded_right_blocks.size();
|
|
|
|
}
|
|
|
|
|
|
|
|
template <bool in_memory>
|
2020-07-10 18:10:06 +00:00
|
|
|
std::shared_ptr<Block> MergeJoin::loadRightBlock(size_t pos) const
|
2019-10-15 16:31:49 +00:00
|
|
|
{
|
|
|
|
if constexpr (!in_memory)
|
|
|
|
{
|
|
|
|
auto load_func = [&]() -> std::shared_ptr<Block>
|
|
|
|
{
|
2021-01-26 12:37:42 +00:00
|
|
|
TemporaryFileStream input(flushed_right_blocks[pos]->path(), materializeBlock(right_sample_block));
|
2019-10-15 16:31:49 +00:00
|
|
|
return std::make_shared<Block>(input.block_in->read());
|
|
|
|
};
|
|
|
|
|
|
|
|
return cached_right_blocks->getOrSet(pos, load_func).first;
|
|
|
|
}
|
|
|
|
else
|
|
|
|
return loaded_right_blocks[pos];
|
|
|
|
}
|
|
|
|
|
2020-06-16 20:13:18 +00:00
|
|
|
void MergeJoin::initRightTableWriter()
|
|
|
|
{
|
|
|
|
disk_writer = std::make_unique<SortedBlocksWriter>(size_limits, table_join->getTemporaryVolume(),
|
|
|
|
right_sample_block, right_sort_description, max_rows_in_right_block, max_files_to_merge,
|
|
|
|
table_join->temporaryFilesCodec());
|
|
|
|
disk_writer->addBlocks(right_blocks);
|
|
|
|
right_blocks.clear();
|
|
|
|
}
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
/// Stream from not joined earlier rows of the right table.
|
2021-08-09 14:30:37 +00:00
|
|
|
class NotJoinedMerge final : public NotJoinedInputStream::RightColumnsFiller
|
2020-07-10 18:10:06 +00:00
|
|
|
{
|
|
|
|
public:
|
2021-08-09 14:30:37 +00:00
|
|
|
NotJoinedMerge(const MergeJoin & parent_, UInt64 max_block_size_)
|
|
|
|
: parent(parent_), max_block_size(max_block_size_)
|
2020-07-10 18:10:06 +00:00
|
|
|
{}
|
|
|
|
|
2021-08-09 14:30:37 +00:00
|
|
|
Block getEmptyBlock() override { return parent.modifyRightBlock(parent.right_sample_block).cloneEmpty(); }
|
2020-07-10 18:10:06 +00:00
|
|
|
|
2021-08-06 14:15:11 +00:00
|
|
|
size_t fillColumns(MutableColumns & columns_right) override
|
2020-07-10 18:10:06 +00:00
|
|
|
{
|
|
|
|
const RowBitmaps & bitmaps = *parent.used_rows_bitmap;
|
|
|
|
size_t rows_added = 0;
|
|
|
|
|
|
|
|
size_t blocks_count = parent.getRightBlocksCount();
|
|
|
|
for (; block_number < blocks_count; ++block_number)
|
|
|
|
{
|
|
|
|
auto right_block = parent.getRightBlock(block_number);
|
|
|
|
|
|
|
|
if (bitmaps.used(block_number))
|
|
|
|
{
|
|
|
|
IColumn::Filter not_used = bitmaps.getNotUsed(block_number);
|
|
|
|
|
|
|
|
for (const auto & row : not_used)
|
|
|
|
if (row)
|
|
|
|
++rows_added;
|
|
|
|
|
|
|
|
for (size_t col = 0; col < columns_right.size(); ++col)
|
|
|
|
{
|
|
|
|
/// TODO: IColumn::filteredInsertRangeFrom() ?
|
|
|
|
ColumnPtr portion = right_block->getByPosition(col).column->filter(not_used, 1);
|
|
|
|
columns_right[col]->insertRangeFrom(*portion, 0, portion->size());
|
|
|
|
}
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
rows_added += right_block->rows();
|
|
|
|
for (size_t col = 0; col < columns_right.size(); ++col)
|
|
|
|
{
|
|
|
|
const IColumn & column = *right_block->getByPosition(col).column;
|
|
|
|
columns_right[col]->insertRangeFrom(column, 0, column.size());
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (rows_added >= max_block_size)
|
2021-07-15 15:22:46 +00:00
|
|
|
{
|
|
|
|
++block_number;
|
2020-07-10 18:10:06 +00:00
|
|
|
break;
|
2021-07-15 15:22:46 +00:00
|
|
|
}
|
2020-07-10 18:10:06 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
return rows_added;
|
|
|
|
}
|
2021-08-06 14:15:11 +00:00
|
|
|
|
|
|
|
private:
|
|
|
|
const MergeJoin & parent;
|
|
|
|
size_t max_block_size;
|
|
|
|
size_t block_number = 0;
|
2020-07-10 18:10:06 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
|
|
|
|
BlockInputStreamPtr MergeJoin::createStreamWithNonJoinedRows(const Block & result_sample_block, UInt64 max_block_size) const
|
|
|
|
{
|
|
|
|
if (table_join->strictness() == ASTTableJoin::Strictness::All && (is_right || is_full))
|
2021-08-09 14:30:37 +00:00
|
|
|
{
|
|
|
|
size_t left_columns_count = result_sample_block.columns() - right_columns_to_add.columns();
|
|
|
|
auto non_joined = std::make_unique<NotJoinedMerge>(*this, max_block_size);
|
|
|
|
return std::make_shared<NotJoinedInputStream>(std::move(non_joined), result_sample_block, left_columns_count, table_join->leftToRightKeyRemap());
|
|
|
|
}
|
2020-07-10 18:10:06 +00:00
|
|
|
return {};
|
|
|
|
}
|
|
|
|
|
2021-07-21 17:03:33 +00:00
|
|
|
bool MergeJoin::needConditionJoinColumn() const
|
|
|
|
{
|
|
|
|
return !mask_column_name_left.empty() || !mask_column_name_right.empty();
|
|
|
|
}
|
|
|
|
|
|
|
|
void MergeJoin::addConditionJoinColumn(Block & block, JoinTableSide block_side) const
|
|
|
|
{
|
|
|
|
if (needConditionJoinColumn())
|
|
|
|
{
|
|
|
|
if (block_side == JoinTableSide::Left)
|
|
|
|
block.insert(condtitionColumnToJoinable(block, mask_column_name_left));
|
|
|
|
else
|
|
|
|
block.insert(condtitionColumnToJoinable(block, mask_column_name_right));
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-07-10 18:10:06 +00:00
|
|
|
|
|
|
|
MergeJoin::RightBlockInfo::RightBlockInfo(std::shared_ptr<Block> block_, size_t block_number_, size_t & skip_, RowBitmaps * bitmaps_)
|
|
|
|
: block(block_)
|
|
|
|
, block_number(block_number_)
|
|
|
|
, skip(skip_)
|
|
|
|
, bitmaps(bitmaps_)
|
|
|
|
{}
|
|
|
|
|
|
|
|
MergeJoin::RightBlockInfo::~RightBlockInfo()
|
|
|
|
{
|
|
|
|
if (used_bitmap)
|
|
|
|
bitmaps->applyOr(block_number, std::move(*used_bitmap));
|
|
|
|
}
|
|
|
|
|
|
|
|
void MergeJoin::RightBlockInfo::setUsed(size_t start, size_t length)
|
|
|
|
{
|
|
|
|
if (bitmaps)
|
|
|
|
{
|
|
|
|
if (!used_bitmap)
|
|
|
|
used_bitmap = std::make_unique<std::vector<bool>>(block->rows(), false);
|
|
|
|
|
|
|
|
for (size_t i = 0; i < length; ++i)
|
|
|
|
(*used_bitmap)[start + i] = true;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2019-09-09 19:43:37 +00:00
|
|
|
}
|