2014-06-12 02:31:30 +00:00
|
|
|
|
#pragma once
|
|
|
|
|
|
2017-07-28 17:34:02 +00:00
|
|
|
|
#include <shared_mutex>
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
|
#include <Parsers/ASTTablesInSelectQuery.h>
|
2014-06-18 18:31:35 +00:00
|
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
|
#include <Interpreters/AggregationCommon.h>
|
|
|
|
|
#include <Interpreters/SettingsCommon.h>
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
|
#include <Common/Arena.h>
|
|
|
|
|
#include <Common/HashTable/HashMap.h>
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
|
#include <Columns/ColumnString.h>
|
|
|
|
|
#include <Columns/ColumnFixedString.h>
|
2017-03-28 06:51:22 +00:00
|
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
|
#include <DataStreams/IBlockInputStream.h>
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
|
{
|
|
|
|
|
|
2017-03-28 06:51:22 +00:00
|
|
|
|
/// Helpers to obtain keys (to use in a hash table or similar data structure) for various equi-JOINs.
|
|
|
|
|
|
|
|
|
|
/// UInt8/16/32/64 or another types with same number of bits.
|
|
|
|
|
template <typename FieldType>
|
|
|
|
|
struct JoinKeyGetterOneNumber
|
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
|
using Key = FieldType;
|
|
|
|
|
|
|
|
|
|
const FieldType * vec;
|
|
|
|
|
|
|
|
|
|
/** Created before processing of each block.
|
|
|
|
|
* Initialize some members, used in another methods, called in inner loops.
|
|
|
|
|
*/
|
2017-12-13 01:27:53 +00:00
|
|
|
|
JoinKeyGetterOneNumber(const ColumnRawPtrs & key_columns)
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
vec = &static_cast<const ColumnVector<FieldType> *>(key_columns[0])->getData()[0];
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
Key getKey(
|
2017-12-13 01:27:53 +00:00
|
|
|
|
const ColumnRawPtrs & /*key_columns*/,
|
2017-12-01 19:34:51 +00:00
|
|
|
|
size_t /*keys_size*/, /// number of key columns.
|
|
|
|
|
size_t i, /// row number to get key from.
|
|
|
|
|
const Sizes & /*key_sizes*/) const /// If keys are of fixed size - their sizes. Not used for methods with variable-length keys.
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
return unionCastToUInt64(vec[i]);
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/// Place additional data into memory pool, if needed, when new key was inserted into hash table.
|
2017-12-01 19:34:51 +00:00
|
|
|
|
static void onNewKey(Key & /*key*/, Arena & /*pool*/) {}
|
2017-03-28 06:51:22 +00:00
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
/// For single String key.
|
|
|
|
|
struct JoinKeyGetterString
|
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
|
using Key = StringRef;
|
|
|
|
|
|
2017-12-15 21:32:25 +00:00
|
|
|
|
const ColumnString::Offsets * offsets;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
const ColumnString::Chars_t * chars;
|
|
|
|
|
|
2017-12-13 01:27:53 +00:00
|
|
|
|
JoinKeyGetterString(const ColumnRawPtrs & key_columns)
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
const IColumn & column = *key_columns[0];
|
|
|
|
|
const ColumnString & column_string = static_cast<const ColumnString &>(column);
|
|
|
|
|
offsets = &column_string.getOffsets();
|
|
|
|
|
chars = &column_string.getChars();
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
Key getKey(
|
2017-12-13 01:27:53 +00:00
|
|
|
|
const ColumnRawPtrs &,
|
2017-12-01 19:34:51 +00:00
|
|
|
|
size_t,
|
2017-04-01 07:20:54 +00:00
|
|
|
|
size_t i,
|
2017-12-01 19:34:51 +00:00
|
|
|
|
const Sizes &) const
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
return StringRef(
|
|
|
|
|
&(*chars)[i == 0 ? 0 : (*offsets)[i - 1]],
|
|
|
|
|
(i == 0 ? (*offsets)[i] : ((*offsets)[i] - (*offsets)[i - 1])) - 1);
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
static void onNewKey(Key & key, Arena & pool)
|
|
|
|
|
{
|
|
|
|
|
key.data = pool.insert(key.data, key.size);
|
|
|
|
|
}
|
2017-03-28 06:51:22 +00:00
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
/// For single FixedString key.
|
|
|
|
|
struct JoinKeyGetterFixedString
|
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
|
using Key = StringRef;
|
|
|
|
|
|
|
|
|
|
size_t n;
|
|
|
|
|
const ColumnFixedString::Chars_t * chars;
|
|
|
|
|
|
2017-12-13 01:27:53 +00:00
|
|
|
|
JoinKeyGetterFixedString(const ColumnRawPtrs & key_columns)
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
const IColumn & column = *key_columns[0];
|
|
|
|
|
const ColumnFixedString & column_string = static_cast<const ColumnFixedString &>(column);
|
|
|
|
|
n = column_string.getN();
|
|
|
|
|
chars = &column_string.getChars();
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
Key getKey(
|
2017-12-13 01:27:53 +00:00
|
|
|
|
const ColumnRawPtrs &,
|
2017-12-01 19:34:51 +00:00
|
|
|
|
size_t,
|
2017-04-01 07:20:54 +00:00
|
|
|
|
size_t i,
|
2017-12-01 19:34:51 +00:00
|
|
|
|
const Sizes &) const
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
return StringRef(&(*chars)[i * n], n);
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
static void onNewKey(Key & key, Arena & pool)
|
|
|
|
|
{
|
|
|
|
|
key.data = pool.insert(key.data, key.size);
|
|
|
|
|
}
|
2017-03-28 06:51:22 +00:00
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
/// For keys of fixed size, that could be packed in sizeof TKey width.
|
|
|
|
|
template <typename TKey>
|
|
|
|
|
struct JoinKeyGetterFixed
|
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
|
using Key = TKey;
|
|
|
|
|
|
2017-12-13 01:27:53 +00:00
|
|
|
|
JoinKeyGetterFixed(const ColumnRawPtrs &)
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
Key getKey(
|
2017-12-13 01:27:53 +00:00
|
|
|
|
const ColumnRawPtrs & key_columns,
|
2017-04-01 07:20:54 +00:00
|
|
|
|
size_t keys_size,
|
|
|
|
|
size_t i,
|
|
|
|
|
const Sizes & key_sizes) const
|
|
|
|
|
{
|
|
|
|
|
return packFixed<Key>(i, keys_size, key_columns, key_sizes);
|
|
|
|
|
}
|
|
|
|
|
|
2017-12-01 19:34:51 +00:00
|
|
|
|
static void onNewKey(Key &, Arena &) {}
|
2017-03-28 06:51:22 +00:00
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
/// Generic method, use crypto hash function.
|
|
|
|
|
struct JoinKeyGetterHashed
|
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
|
using Key = UInt128;
|
|
|
|
|
|
2017-12-13 01:27:53 +00:00
|
|
|
|
JoinKeyGetterHashed(const ColumnRawPtrs &)
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
Key getKey(
|
2017-12-13 01:27:53 +00:00
|
|
|
|
const ColumnRawPtrs & key_columns,
|
2017-04-01 07:20:54 +00:00
|
|
|
|
size_t keys_size,
|
|
|
|
|
size_t i,
|
2017-12-01 19:34:51 +00:00
|
|
|
|
const Sizes &) const
|
2017-04-01 07:20:54 +00:00
|
|
|
|
{
|
|
|
|
|
return hash128(i, keys_size, key_columns);
|
|
|
|
|
}
|
|
|
|
|
|
2017-12-01 19:34:51 +00:00
|
|
|
|
static void onNewKey(Key &, Arena &) {}
|
2017-03-28 06:51:22 +00:00
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
2017-01-21 04:24:28 +00:00
|
|
|
|
struct Limits;
|
|
|
|
|
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-03-29 11:12:47 +00:00
|
|
|
|
/** Data structure for implementation of JOIN.
|
|
|
|
|
* It is just a hash table: keys -> rows of joined ("right") table.
|
|
|
|
|
* Additionally, CROSS JOIN is supported: instead of hash table, it use just set of blocks without keys.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* JOIN-s could be of nine types: ANY/ALL × LEFT/INNER/RIGHT/FULL, and also CROSS.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* If ANY is specified - then select only one row from the "right" table, (first encountered row), even if there was more matching rows.
|
|
|
|
|
* If ALL is specified - usual JOIN, when rows are multiplied by number of matching rows from the "right" table.
|
|
|
|
|
* ANY is more efficient.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* If INNER is specified - leave only rows that have matching rows from "right" table.
|
|
|
|
|
* If LEFT is specified - in case when there is no matching row in "right" table, fill it with default values instead.
|
|
|
|
|
* If RIGHT is specified - first process as INNER, but track what rows from the right table was joined,
|
|
|
|
|
* and at the end, add rows from right table that was not joined and substitute default values for columns of left table.
|
|
|
|
|
* If FULL is specified - first process as LEFT, but track what rows from the right table was joined,
|
|
|
|
|
* and at the end, add rows from right table that was not joined and substitute default values for columns of left table.
|
2015-04-17 08:46:06 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* Thus, LEFT and RIGHT JOINs are not symmetric in terms of implementation.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* All JOINs (except CROSS) are done by equality condition on keys (equijoin).
|
|
|
|
|
* Non-equality and other conditions are not supported.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* Implementation:
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* 1. Build hash table in memory from "right" table.
|
|
|
|
|
* This hash table is in form of keys -> row in case of ANY or keys -> [rows...] in case of ALL.
|
|
|
|
|
* This is done in insertFromBlock method.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* 2. Process "left" table and join corresponding rows from "right" table by lookups in the map.
|
|
|
|
|
* This is done in joinBlock methods.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* In case of ANY LEFT JOIN - form new columns with found values or default values.
|
|
|
|
|
* This is the most simple. Number of rows in left table does not change.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* In case of ANY INNER JOIN - form new columns with found values,
|
|
|
|
|
* and also build a filter - in what rows nothing was found.
|
|
|
|
|
* Then filter columns of "left" table.
|
2014-06-19 19:00:58 +00:00
|
|
|
|
*
|
2017-03-29 11:12:47 +00:00
|
|
|
|
* In case of ALL ... JOIN - form new columns with all found rows,
|
|
|
|
|
* and also fill 'offsets' array, describing how many times we need to replicate values of "left" table.
|
|
|
|
|
* Then replicate columns of "left" table.
|
2017-03-30 14:09:24 +00:00
|
|
|
|
*
|
|
|
|
|
* How Nullable keys are processed:
|
|
|
|
|
*
|
|
|
|
|
* NULLs never join to anything, even to each other.
|
|
|
|
|
* During building of map, we just skip keys with NULL value of any component.
|
|
|
|
|
* During joining, we simply treat rows with any NULLs in key as non joined.
|
2017-04-04 06:52:39 +00:00
|
|
|
|
*
|
|
|
|
|
* Default values for outer joins (LEFT, RIGHT, FULL):
|
|
|
|
|
*
|
|
|
|
|
* Behaviour is controlled by 'join_use_nulls' settings.
|
|
|
|
|
* If it is false, we substitute (global) default value for the data type, for non-joined rows
|
|
|
|
|
* (zero, empty string, etc. and NULL for Nullable data types).
|
|
|
|
|
* If it is true, we always generate Nullable column and substitute NULLs for non-joined rows,
|
|
|
|
|
* as in standard SQL.
|
2014-06-12 02:31:30 +00:00
|
|
|
|
*/
|
|
|
|
|
class Join
|
|
|
|
|
{
|
|
|
|
|
public:
|
2017-04-04 06:52:39 +00:00
|
|
|
|
Join(const Names & key_names_left_, const Names & key_names_right_, bool use_nulls_,
|
2017-04-01 07:20:54 +00:00
|
|
|
|
const Limits & limits, ASTTableJoin::Kind kind_, ASTTableJoin::Strictness strictness_);
|
|
|
|
|
|
|
|
|
|
bool empty() { return type == Type::EMPTY; }
|
|
|
|
|
|
|
|
|
|
/** Set information about structure of right hand of JOIN (joined data).
|
|
|
|
|
* You must call this method before subsequent calls to insertFromBlock.
|
|
|
|
|
*/
|
|
|
|
|
void setSampleBlock(const Block & block);
|
|
|
|
|
|
|
|
|
|
/** Add block of data from right hand of JOIN to the map.
|
|
|
|
|
* Returns false, if some limit was exceeded and you should not insert more data.
|
|
|
|
|
*/
|
|
|
|
|
bool insertFromBlock(const Block & block);
|
|
|
|
|
|
|
|
|
|
/** Join data from the map (that was previously built by calls to insertFromBlock) to the block with data from "left" table.
|
|
|
|
|
* Could be called from different threads in parallel.
|
|
|
|
|
*/
|
|
|
|
|
void joinBlock(Block & block) const;
|
|
|
|
|
|
|
|
|
|
/** Keep "totals" (separate part of dataset, see WITH TOTALS) to use later.
|
|
|
|
|
*/
|
|
|
|
|
void setTotals(const Block & block) { totals = block; }
|
|
|
|
|
bool hasTotals() const { return totals; };
|
|
|
|
|
|
|
|
|
|
void joinTotals(Block & block) const;
|
|
|
|
|
|
|
|
|
|
/** For RIGHT and FULL JOINs.
|
|
|
|
|
* A stream that will contain default values from left table, joined with rows from right table, that was not joined before.
|
|
|
|
|
* Use only after all calls to joinBlock was done.
|
2017-04-04 06:52:39 +00:00
|
|
|
|
* left_sample_block is passed without account of 'use_nulls' setting (columns will be converted to Nullable inside).
|
2017-04-01 07:20:54 +00:00
|
|
|
|
*/
|
2018-02-21 08:16:01 +00:00
|
|
|
|
BlockInputStreamPtr createStreamWithNonJoinedRows(const Block & left_sample_block, size_t max_block_size) const;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
|
|
/// Number of keys in all built JOIN maps.
|
|
|
|
|
size_t getTotalRowCount() const;
|
|
|
|
|
/// Sum size in bytes of all buffers, used for JOIN maps and for all memory pools.
|
|
|
|
|
size_t getTotalByteCount() const;
|
|
|
|
|
|
|
|
|
|
ASTTableJoin::Kind getKind() const { return kind; }
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
/// Reference to the row in block.
|
|
|
|
|
struct RowRef
|
|
|
|
|
{
|
|
|
|
|
const Block * block;
|
|
|
|
|
size_t row_num;
|
|
|
|
|
|
|
|
|
|
RowRef() {}
|
|
|
|
|
RowRef(const Block * block_, size_t row_num_) : block(block_), row_num(row_num_) {}
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
/// Single linked list of references to rows. Used for ALL JOINs (non-unique JOINs)
|
|
|
|
|
struct RowRefList : RowRef
|
|
|
|
|
{
|
|
|
|
|
RowRefList * next = nullptr;
|
|
|
|
|
|
|
|
|
|
RowRefList() {}
|
|
|
|
|
RowRefList(const Block * block_, size_t row_num_) : RowRef(block_, row_num_) {}
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
/** Depending on template parameter, adds or doesn't add a flag, that element was used (row was joined).
|
|
|
|
|
* For implementation of RIGHT and FULL JOINs.
|
|
|
|
|
* NOTE: It is possible to store the flag in one bit of pointer to block or row_num. It seems not reasonable, because memory saving is minimal.
|
|
|
|
|
*/
|
|
|
|
|
template <bool enable, typename Base>
|
|
|
|
|
struct WithUsedFlag;
|
|
|
|
|
|
|
|
|
|
template <typename Base>
|
|
|
|
|
struct WithUsedFlag<true, Base> : Base
|
|
|
|
|
{
|
|
|
|
|
mutable std::atomic<bool> used {};
|
|
|
|
|
using Base::Base;
|
|
|
|
|
using Base_t = Base;
|
|
|
|
|
void setUsed() const { used.store(true, std::memory_order_relaxed); } /// Could be set simultaneously from different threads.
|
|
|
|
|
bool getUsed() const { return used; }
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
template <typename Base>
|
|
|
|
|
struct WithUsedFlag<false, Base> : Base
|
|
|
|
|
{
|
|
|
|
|
using Base::Base;
|
|
|
|
|
using Base_t = Base;
|
|
|
|
|
void setUsed() const {}
|
|
|
|
|
bool getUsed() const { return true; }
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
/// Different types of keys for maps.
|
|
|
|
|
#define APPLY_FOR_JOIN_VARIANTS(M) \
|
2017-11-12 00:46:22 +00:00
|
|
|
|
M(key8) \
|
|
|
|
|
M(key16) \
|
|
|
|
|
M(key32) \
|
|
|
|
|
M(key64) \
|
|
|
|
|
M(key_string) \
|
|
|
|
|
M(key_fixed_string) \
|
|
|
|
|
M(keys128) \
|
|
|
|
|
M(keys256) \
|
2017-04-01 07:20:54 +00:00
|
|
|
|
M(hashed)
|
|
|
|
|
|
|
|
|
|
enum class Type
|
|
|
|
|
{
|
|
|
|
|
EMPTY,
|
|
|
|
|
CROSS,
|
|
|
|
|
#define M(NAME) NAME,
|
|
|
|
|
APPLY_FOR_JOIN_VARIANTS(M)
|
|
|
|
|
#undef M
|
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
|
|
|
|
|
/** Different data structures, that are used to perform JOIN.
|
|
|
|
|
*/
|
|
|
|
|
template <typename Mapped>
|
|
|
|
|
struct MapsTemplate
|
|
|
|
|
{
|
2017-11-12 00:46:22 +00:00
|
|
|
|
std::unique_ptr<HashMap<UInt8, Mapped, TrivialHash, HashTableFixedGrower<8>>> key8;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
std::unique_ptr<HashMap<UInt16, Mapped, TrivialHash, HashTableFixedGrower<16>>> key16;
|
|
|
|
|
std::unique_ptr<HashMap<UInt32, Mapped, HashCRC32<UInt32>>> key32;
|
|
|
|
|
std::unique_ptr<HashMap<UInt64, Mapped, HashCRC32<UInt64>>> key64;
|
2017-11-12 00:46:22 +00:00
|
|
|
|
std::unique_ptr<HashMapWithSavedHash<StringRef, Mapped>> key_string;
|
|
|
|
|
std::unique_ptr<HashMapWithSavedHash<StringRef, Mapped>> key_fixed_string;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
std::unique_ptr<HashMap<UInt128, Mapped, UInt128HashCRC32>> keys128;
|
|
|
|
|
std::unique_ptr<HashMap<UInt256, Mapped, UInt256HashCRC32>> keys256;
|
2017-11-12 00:46:22 +00:00
|
|
|
|
std::unique_ptr<HashMap<UInt128, Mapped, UInt128TrivialHash>> hashed;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
};
|
|
|
|
|
|
|
|
|
|
using MapsAny = MapsTemplate<WithUsedFlag<false, RowRef>>;
|
|
|
|
|
using MapsAll = MapsTemplate<WithUsedFlag<false, RowRefList>>;
|
|
|
|
|
using MapsAnyFull = MapsTemplate<WithUsedFlag<true, RowRef>>;
|
|
|
|
|
using MapsAllFull = MapsTemplate<WithUsedFlag<true, RowRefList>>;
|
2014-06-18 20:08:31 +00:00
|
|
|
|
|
2014-06-18 19:14:29 +00:00
|
|
|
|
private:
|
2017-04-01 07:20:54 +00:00
|
|
|
|
friend class NonJoinedBlockInputStream;
|
2015-04-17 08:46:06 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
ASTTableJoin::Kind kind;
|
|
|
|
|
ASTTableJoin::Strictness strictness;
|
2014-06-18 19:14:29 +00:00
|
|
|
|
|
2017-04-27 17:48:30 +00:00
|
|
|
|
/// Names of key columns (columns for equi-JOIN) in "left" table (in the order they appear in USING clause).
|
2017-04-01 07:20:54 +00:00
|
|
|
|
const Names key_names_left;
|
2017-04-27 17:48:30 +00:00
|
|
|
|
/// Names of key columns (columns for equi-JOIN) in "right" table (in the order they appear in USING clause).
|
2017-04-01 07:20:54 +00:00
|
|
|
|
const Names key_names_right;
|
2014-06-18 19:14:29 +00:00
|
|
|
|
|
2017-04-04 06:52:39 +00:00
|
|
|
|
/// Substitute NULLs for non-JOINed rows.
|
|
|
|
|
bool use_nulls;
|
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
/** Blocks of "right" table.
|
|
|
|
|
*/
|
|
|
|
|
BlocksList blocks;
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
MapsAny maps_any; /// For ANY LEFT|INNER JOIN
|
|
|
|
|
MapsAll maps_all; /// For ALL LEFT|INNER JOIN
|
|
|
|
|
MapsAnyFull maps_any_full; /// For ANY RIGHT|FULL JOIN
|
|
|
|
|
MapsAllFull maps_all_full; /// For ALL RIGHT|FULL JOIN
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
/// Additional data - strings for string keys and continuation elements of single-linked lists of references to rows.
|
|
|
|
|
Arena pool;
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2016-03-07 05:05:42 +00:00
|
|
|
|
private:
|
2017-04-01 07:20:54 +00:00
|
|
|
|
Type type = Type::EMPTY;
|
2015-03-02 01:10:58 +00:00
|
|
|
|
|
2017-12-13 01:27:53 +00:00
|
|
|
|
static Type chooseMethod(const ColumnRawPtrs & key_columns, Sizes & key_sizes);
|
2014-07-06 19:48:39 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
Sizes key_sizes;
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-04-27 17:48:30 +00:00
|
|
|
|
/// Block with columns from the right-side table except key columns.
|
2017-04-01 07:20:54 +00:00
|
|
|
|
Block sample_block_with_columns_to_add;
|
2017-04-27 17:48:30 +00:00
|
|
|
|
/// Block with key columns in the same order they appear in the right-side table.
|
2017-04-01 07:20:54 +00:00
|
|
|
|
Block sample_block_with_keys;
|
2015-05-26 00:37:48 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
Poco::Logger * log;
|
2014-07-06 19:48:39 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
/// Limits for maximum map size.
|
|
|
|
|
size_t max_rows;
|
|
|
|
|
size_t max_bytes;
|
|
|
|
|
OverflowMode overflow_mode;
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
Block totals;
|
2015-04-16 09:55:24 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
/** Protect state for concurrent use in insertFromBlock and joinBlock.
|
|
|
|
|
* Note that these methods could be called simultaneously only while use of StorageJoin,
|
|
|
|
|
* and StorageJoin only calls these two methods.
|
|
|
|
|
* That's why another methods are not guarded.
|
|
|
|
|
*/
|
2017-07-28 17:34:02 +00:00
|
|
|
|
mutable std::shared_mutex rwlock;
|
2015-01-27 21:24:24 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
void init(Type type_);
|
2014-06-18 20:08:31 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
bool checkSizeLimits() const;
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
/// Throw an exception if blocks have different types of key columns.
|
|
|
|
|
void checkTypesOfKeys(const Block & block_left, const Block & block_right) const;
|
2017-03-28 06:51:22 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
template <ASTTableJoin::Kind KIND, ASTTableJoin::Strictness STRICTNESS, typename Maps>
|
|
|
|
|
void joinBlockImpl(Block & block, const Maps & maps) const;
|
2014-06-18 18:31:35 +00:00
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
void joinBlockImplCross(Block & block) const;
|
2014-06-12 02:31:30 +00:00
|
|
|
|
};
|
|
|
|
|
|
2016-05-28 10:35:44 +00:00
|
|
|
|
using JoinPtr = std::shared_ptr<Join>;
|
|
|
|
|
using Joins = std::vector<JoinPtr>;
|
2014-06-12 02:31:30 +00:00
|
|
|
|
|
|
|
|
|
|
|
|
|
|
}
|