OmniSciDB
a5dc49c757
|
#include <RangeJoinHashTable.h>
Public Member Functions | |
RangeJoinHashTable (const std::shared_ptr< Analyzer::BinOper > condition, const JoinType join_type, const Analyzer::RangeOper *range_expr, std::shared_ptr< Analyzer::ColumnVar > inner_col_expr, const std::vector< InputTableInfo > &query_infos, const Data_Namespace::MemoryLevel memory_level, ColumnCacheMap &column_cache, Executor *executor, const std::vector< InnerOuter > &inner_outer_pairs, const int device_count, const RegisteredQueryHint &query_hints, const HashTableBuildDagMap &hashtable_build_dag_map, const TableIdToNodeMap &table_id_to_node_map) | |
~RangeJoinHashTable () override=default | |
llvm::Value * | codegenKey (const CompilationOptions &co, llvm::Value *offset) |
HashJoinMatchingSet | codegenMatchingSetWithOffset (const CompilationOptions &, const size_t, llvm::Value *) |
Public Member Functions inherited from BoundingBoxIntersectJoinHashTable | |
BoundingBoxIntersectJoinHashTable (const std::shared_ptr< Analyzer::BinOper > condition, const JoinType join_type, const std::vector< InputTableInfo > &query_infos, const Data_Namespace::MemoryLevel memory_level, ColumnCacheMap &column_cache, Executor *executor, const std::vector< InnerOuter > &inner_outer_pairs, const int device_count, const RegisteredQueryHint &query_hints, const HashTableBuildDagMap &hashtable_build_dag_map, const TableIdToNodeMap &table_id_to_node_map) | |
virtual | ~BoundingBoxIntersectJoinHashTable () |
Public Member Functions inherited from HashJoin | |
virtual std::string | toStringFlat64 (const ExecutorDeviceType device_type, const int device_id) const |
virtual std::string | toStringFlat32 (const ExecutorDeviceType device_type, const int device_id) const |
JoinColumn | fetchJoinColumn (const Analyzer::ColumnVar *hash_col, const std::vector< Fragmenter_Namespace::FragmentInfo > &fragment_info, const Data_Namespace::MemoryLevel effective_memory_level, const int device_id, std::vector< std::shared_ptr< Chunk_NS::Chunk >> &chunks_owner, DeviceAllocator *dev_buff_owner, std::vector< std::shared_ptr< void >> &malloc_owner, Executor *executor, ColumnCacheMap *column_cache) |
HashTable * | getHashTableForDevice (const size_t device_id) const |
size_t | getJoinHashBufferSize (const ExecutorDeviceType device_type) |
size_t | getJoinHashBufferSize (const ExecutorDeviceType device_type, const int device_id) const |
int8_t * | getJoinHashBuffer (const ExecutorDeviceType device_type, const int device_id) const |
void | freeHashBufferMemory () |
Static Public Member Functions | |
static std::shared_ptr < RangeJoinHashTable > | getInstance (const std::shared_ptr< Analyzer::BinOper > condition, const Analyzer::RangeOper *range_expr, const std::vector< InputTableInfo > &query_infos, const Data_Namespace::MemoryLevel memory_level, const JoinType join_type, const int device_count, ColumnCacheMap &column_cache, Executor *executor, const HashTableBuildDagMap &hashtable_build_dag_map, const RegisteredQueryHint &query_hints, const TableIdToNodeMap &table_id_to_node_map) |
Static Public Member Functions inherited from BoundingBoxIntersectJoinHashTable | |
static std::shared_ptr < BoundingBoxIntersectJoinHashTable > | getInstance (const std::shared_ptr< Analyzer::BinOper > condition, const std::vector< InputTableInfo > &query_infos, const Data_Namespace::MemoryLevel memory_level, const JoinType join_type, const int device_count, ColumnCacheMap &column_cache, Executor *executor, const HashTableBuildDagMap &hashtable_build_dag_map, const RegisteredQueryHint &query_hint, const TableIdToNodeMap &table_id_to_node_map) |
Make hash table from an in-flight SQL query's parse tree etc. More... | |
static void | invalidateCache () |
static void | markCachedItemAsDirty (size_t table_key) |
static HashtableRecycler * | getHashTableCache () |
static BoundingBoxIntersectTuningParamRecycler * | getBoundingBoxIntersectTuningParamCache () |
Static Public Member Functions inherited from HashJoin | |
static size_t | getMaximumNumHashEntriesCanHold (MemoryLevel memory_level, const Executor *executor, size_t rowid_size) noexcept |
static std::string | generateTooManyHashEntriesErrMsg (size_t num_entries, size_t threshold, MemoryLevel memory_level) |
static bool | layoutRequiresAdditionalBuffers (HashType layout) noexcept |
static std::string | getHashTypeString (HashType ht) noexcept |
static HashJoinMatchingSet | codegenMatchingSet (const std::vector< llvm::Value * > &hash_join_idx_args_in, const bool is_sharded, const bool col_is_nullable, const bool is_bw_eq, const int64_t sub_buff_size, Executor *executor, const bool is_bucketized=false) |
static llvm::Value * | codegenHashTableLoad (const size_t table_idx, Executor *executor) |
static std::shared_ptr< HashJoin > | getInstance (const std::shared_ptr< Analyzer::BinOper > qual_bin_oper, const std::vector< InputTableInfo > &query_infos, const Data_Namespace::MemoryLevel memory_level, const JoinType join_type, const HashType preferred_hash_type, const int device_count, ColumnCacheMap &column_cache, Executor *executor, const HashTableBuildDagMap &hashtable_build_dag_map, const RegisteredQueryHint &query_hint, const TableIdToNodeMap &table_id_to_node_map) |
Make hash table from an in-flight SQL query's parse tree etc. More... | |
static std::shared_ptr< HashJoin > | getSyntheticInstance (std::string_view table1, std::string_view column1, const Catalog_Namespace::Catalog &catalog1, std::string_view table2, std::string_view column2, const Catalog_Namespace::Catalog &catalog2, const Data_Namespace::MemoryLevel memory_level, const HashType preferred_hash_type, const int device_count, ColumnCacheMap &column_cache, Executor *executor) |
Make hash table from named tables and columns (such as for testing). More... | |
static std::shared_ptr< HashJoin > | getSyntheticInstance (const std::shared_ptr< Analyzer::BinOper > qual_bin_oper, const Data_Namespace::MemoryLevel memory_level, const HashType preferred_hash_type, const int device_count, ColumnCacheMap &column_cache, Executor *executor) |
Make hash table from named tables and columns (such as for testing). More... | |
static std::pair< std::string, std::shared_ptr< HashJoin > > | getSyntheticInstance (std::vector< std::shared_ptr< Analyzer::BinOper >>, const Data_Namespace::MemoryLevel memory_level, const HashType preferred_hash_type, const int device_count, ColumnCacheMap &column_cache, Executor *executor) |
static shared::TableKey | getInnerTableId (const std::vector< InnerOuter > &inner_outer_pairs) |
static bool | canAccessHashTable (bool allow_hash_table_recycling, bool invalid_cache_key, JoinType join_type) |
static void | checkHashJoinReplicationConstraint (const shared::TableKey &table_key, const size_t shard_count, const Executor *executor) |
static std::pair< InnerOuter, InnerOuterStringOpInfos > | normalizeColumnPair (const Analyzer::Expr *lhs, const Analyzer::Expr *rhs, const TemporaryTables *temporary_tables, const bool is_bbox_intersect=false) |
template<typename T > | |
static const T * | getHashJoinColumn (const Analyzer::Expr *expr) |
static std::pair< std::vector < InnerOuter >, std::vector < InnerOuterStringOpInfos > > | normalizeColumnPairs (const Analyzer::BinOper *condition, const TemporaryTables *temporary_tables) |
static std::vector< int > | collectFragmentIds (const std::vector< Fragmenter_Namespace::FragmentInfo > &fragments) |
static CompositeKeyInfo | getCompositeKeyInfo (const std::vector< InnerOuter > &inner_outer_pairs, const Executor *executor, const std::vector< InnerOuterStringOpInfos > &inner_outer_string_op_infos_pairs={}) |
static std::vector< const StringDictionaryProxy::IdMap * > | translateCompositeStrDictProxies (const CompositeKeyInfo &composite_key_info, const std::vector< InnerOuterStringOpInfos > &string_op_infos_for_keys, const Executor *executor) |
static std::pair< const StringDictionaryProxy *, StringDictionaryProxy * > | getStrDictProxies (const InnerOuter &cols, const Executor *executor, const bool has_string_ops) |
static const StringDictionaryProxy::IdMap * | translateInnerToOuterStrDictProxies (const InnerOuter &cols, const InnerOuterStringOpInfos &inner_outer_string_op_infos, ExpressionRange &old_col_range, const Executor *executor) |
Protected Member Functions | |
void | reifyWithLayout (const HashType layout) override |
void | reifyForDevice (const ColumnsForDevice &columns_for_device, const HashType layout, const size_t entry_count, const size_t emitted_keys_count, const int device_id, const logger::ThreadLocalIds parent_thread_local_ids) |
std::shared_ptr < BaselineHashTable > | initHashTableOnCpu (const std::vector< JoinColumn > &join_columns, const std::vector< JoinColumnTypeInfo > &join_column_types, const std::vector< JoinBucketInfo > &join_bucket_info, const BaselineHashTableEntryInfo hash_table_entry_info) |
HashType | getHashType () const noexceptoverride |
std::pair< size_t, size_t > | approximateTupleCount (const std::vector< double > &inverse_bucket_sizes_for_dimension, std::vector< ColumnsForDevice > &columns_per_device, const size_t chosen_max_hashtable_size, const double chosen_bucket_threshold) override |
std::pair< size_t, size_t > | computeRangeHashTableCounts (const size_t shard_count, std::vector< ColumnsForDevice > &columns_per_device) |
Protected Member Functions inherited from BoundingBoxIntersectJoinHashTable | |
void | reify (const HashType preferred_layout) |
virtual void | reifyImpl (std::vector< ColumnsForDevice > &columns_per_device, const Fragmenter_Namespace::TableInfo &query_info, const HashType layout, const size_t shard_count, const size_t entry_count, const size_t emitted_keys_count, const bool skip_hashtable_caching, const size_t chosen_max_hashtable_size, const double chosen_bucket_threshold) |
void | reifyForDevice (const ColumnsForDevice &columns_for_device, const HashType layout, const size_t entry_count, const size_t emitted_keys_count, const bool skip_hashtable_caching, const int device_id, const logger::ThreadLocalIds parent_thread_local_ids) |
size_t | calculateHashTableSize (size_t number_of_dimensions, size_t emitted_keys_count, size_t entry_count) const |
ColumnsForDevice | fetchColumnsForDevice (const std::vector< Fragmenter_Namespace::FragmentInfo > &fragments, const int device_id, DeviceAllocator *dev_buff_owner) |
virtual std::pair< size_t, size_t > | computeHashTableCounts (const size_t shard_count, const std::vector< double > &inverse_bucket_sizes_for_dimension, std::vector< ColumnsForDevice > &columns_per_device, const size_t chosen_max_hashtable_size, const double chosen_bucket_threshold) |
void | setInverseBucketSizeInfo (const std::vector< double > &inverse_bucket_sizes, std::vector< ColumnsForDevice > &columns_per_device, const size_t device_count) |
size_t | getKeyComponentWidth () const |
size_t | getKeyComponentCount () const |
Data_Namespace::MemoryLevel | getMemoryLevel () const noexceptoverride |
int | getDeviceCount () const noexceptoverride |
std::shared_ptr < BaselineHashTable > | initHashTableOnCpu (const std::vector< JoinColumn > &join_columns, const std::vector< JoinColumnTypeInfo > &join_column_types, const std::vector< JoinBucketInfo > &join_bucket_info, const BaselineHashTableEntryInfo hash_table_entry_info, const bool skip_hashtable_caching) |
HashJoinMatchingSet | codegenMatchingSet (const CompilationOptions &, const size_t) override |
std::string | toString (const ExecutorDeviceType device_type, const int device_id=0, bool raw=false) const override |
DecodedJoinHashBufferSet | toSet (const ExecutorDeviceType device_type, const int device_id) const override |
llvm::Value * | codegenSlot (const CompilationOptions &, const size_t) override |
const RegisteredQueryHint & | getRegisteredQueryHint () |
size_t | getEntryCount () const |
size_t | getEmittedKeysCount () const |
size_t | getComponentBufferSize () const noexceptoverride |
size_t | shardCount () const |
Data_Namespace::MemoryLevel | getEffectiveMemoryLevel (const std::vector< InnerOuter > &inner_outer_pairs) const |
shared::TableKey | getInnerTableId () const noexceptoverride |
int | getInnerTableRteIdx () const noexceptoverride |
size_t | getKeyBufferSize () const noexcept |
size_t | offsetBufferOff () const noexceptoverride |
size_t | countBufferOff () const noexceptoverride |
size_t | payloadBufferOff () const noexceptoverride |
std::string | getHashJoinType () const final |
bool | isBitwiseEq () const override |
std::shared_ptr< HashTable > | initHashTableOnCpuFromCache (QueryPlanHash key, CacheItemType item_type, DeviceIdentifier device_identifier) |
std::optional< std::pair < size_t, size_t > > | getApproximateTupleCountFromCache (QueryPlanHash key, CacheItemType item_type, DeviceIdentifier device_identifier) |
void | putHashTableOnCpuToCache (QueryPlanHash key, CacheItemType item_type, std::shared_ptr< HashTable > hashtable_ptr, DeviceIdentifier device_identifier, size_t hashtable_building_time) |
llvm::Value * | codegenKey (const CompilationOptions &) |
std::vector< llvm::Value * > | codegenManyKey (const CompilationOptions &) |
std::optional < BoundingBoxIntersectMetaInfo > | getBoundingBoxIntersectMetaInfo () |
QueryPlanHash | getAlternativeCacheKey (AlternativeCacheKeyForBoundingBoxIntersection &info) |
void | generateCacheKey (const size_t max_hashtable_size, const double bucket_threshold, const std::vector< double > &bucket_sizes, std::vector< std::vector< Fragmenter_Namespace::FragmentInfo >> &fragments_per_device, int device_count) |
QueryPlanHash | getCacheKey (int device_id) const |
const std::vector< InnerOuter > & | getInnerOuterPairs () const |
void | setBoundingBoxIntersectionMetaInfo (size_t max_table_size_bytes, double bucket_threshold, std::vector< double > &bucket_sizes) |
Private Member Functions | |
bool | isInnerColCompressed () const |
bool | isProbeCompressed () const |
Private Attributes | |
const Analyzer::RangeOper * | range_expr_ |
std::shared_ptr < Analyzer::ColumnVar > | inner_col_expr_ |
Data_Namespace::MemoryLevel | effective_memory_level_ |
const double | bucket_threshold_ {std::numeric_limits<double>::max()} |
const size_t | max_hashtable_size_ {std::numeric_limits<size_t>::max()} |
Definition at line 21 of file RangeJoinHashTable.h.
|
inline |
Definition at line 23 of file RangeJoinHashTable.h.
|
overridedefault |
|
overrideprotectedvirtual |
Reimplemented from BoundingBoxIntersectJoinHashTable.
Definition at line 518 of file RangeJoinHashTable.cpp.
References approximate_distinct_tuples_on_device_range(), approximate_distinct_tuples_range(), threading_serial::async(), Bitmap, CHECK, CHECK_EQ, CHECK_GT, CPU, Data_Namespace::CPU_LEVEL, cpu_threads(), BoundingBoxIntersectJoinHashTable::device_count_, effective_memory_level_, BoundingBoxIntersectJoinHashTable::executor_, HashJoin::getCompositeKeyInfo(), getQueryEngineCudaStreamForDevice(), GPU, Data_Namespace::GPU_LEVEL, hll_size(), hll_unify(), CountDistinctDescriptor::impl_type_, BoundingBoxIntersectJoinHashTable::inner_outer_pairs_, isInnerColCompressed(), transfer_flat_object_to_gpu(), transfer_vector_of_flat_objects_to_gpu(), and UNREACHABLE.
Referenced by computeRangeHashTableCounts().
llvm::Value * RangeJoinHashTable::codegenKey | ( | const CompilationOptions & | co, |
llvm::Value * | offset | ||
) |
Definition at line 699 of file RangeJoinHashTable.cpp.
References CodeGenerator::castArrayPointer(), CHECK, CHECK_EQ, CodeGenerator::codegen(), BoundingBoxIntersectJoinHashTable::executor_, func_resolve, get_int_type(), Catalog_Namespace::get_metadata_for_column(), BoundingBoxIntersectJoinHashTable::getKeyComponentCount(), BoundingBoxIntersectJoinHashTable::getKeyComponentWidth(), BoundingBoxIntersectJoinHashTable::inner_outer_pairs_, BoundingBoxIntersectJoinHashTable::inverse_bucket_sizes_for_dimension_, isProbeCompressed(), kPOINT, kTINYINT, LL_BUILDER, LL_CONTEXT, LL_FP, LL_INT, and CodeGenerator::posArg().
Referenced by codegenMatchingSetWithOffset().
HashJoinMatchingSet RangeJoinHashTable::codegenMatchingSetWithOffset | ( | const CompilationOptions & | co, |
const size_t | index, | ||
llvm::Value * | range_offset | ||
) |
Definition at line 829 of file RangeJoinHashTable.cpp.
References CHECK, HashJoin::codegenHashTableLoad(), codegenKey(), HashJoin::codegenMatchingSet(), BoundingBoxIntersectJoinHashTable::executor_, get_int_type(), BoundingBoxIntersectJoinHashTable::getComponentBufferSize(), BoundingBoxIntersectJoinHashTable::getEntryCount(), getHashType(), BoundingBoxIntersectJoinHashTable::getKeyComponentCount(), BoundingBoxIntersectJoinHashTable::getKeyComponentWidth(), LL_BUILDER, LL_CONTEXT, LL_INT, BoundingBoxIntersectJoinHashTable::offsetBufferOff(), OneToMany, and to_string().
|
protected |
Definition at line 502 of file RangeJoinHashTable.cpp.
References approximateTupleCount(), bucket_threshold_, CHECK, BoundingBoxIntersectJoinHashTable::device_count_, get_entries_per_device(), BoundingBoxIntersectJoinHashTable::inverse_bucket_sizes_for_dimension_, max_hashtable_size_, and BoundingBoxIntersectJoinHashTable::memory_level_.
Referenced by reifyWithLayout().
|
inlineoverrideprotectedvirtualnoexcept |
Reimplemented from BoundingBoxIntersectJoinHashTable.
Definition at line 90 of file RangeJoinHashTable.h.
References OneToMany.
Referenced by codegenMatchingSetWithOffset().
|
static |
First, let's take a concrete example of a query that is rewritten as a range join. Notice in the first code block, that the condition operator is a bounding box intersection operator. The LHS is a column, and the RHS is the range operator. In order to have the hash table build and probe work properly, we need to ensure that the approriate runtime functions are selected. The following breakdown is provided to help document how the appropriate runtime funditon is selected.
SELECT count(*) FROM t1, t2 where ST_Distance(t1.p1_comp32, t2.p1) <= 6.3;
((BoundingBoxIntersect) (ColumnVar table: (t1) column: (p1_comp32) GEOMETRY(POINT, 4326) ENCODING COMPRESSED(32)) (RangeOper) (ColumnVar table: (t2) column: (p1) GEOMETRY(POINT, 4326) ENCODING NONE), (Const 6.330000))
[(ColumnVar table: 5 (t2) column: 1 rte: 1 GEOMETRY(POINT, 4326) ENCODING NONE), (Const 6.330000)]
Same example as above, annotated:
SELECT count(*) FROM t1, t2 where ST_Distance( t1.p1_comp32, << Bounding Box Intersection Condition LHS t2.p1 << RangeOper LHS ) <= 6.3; << RangeOper RHS
In this case, we select the uncompressed runtime functions when building the hash table over t2.p1. When performing the probe, we must select the compressed runtime functions.
Definition at line 71 of file RangeJoinHashTable.cpp.
References CHECK, HashJoin::checkHashJoinReplicationConstraint(), logger::FATAL, get_inner_query_info(), Analyzer::RangeOper::get_left_operand(), Catalog_Namespace::get_metadata_for_column(), HashJoin::getInnerTableId(), Fragmenter_Namespace::TableInfo::getNumTuplesUpperBound(), BaselineJoinHashTable::getShardCountForCondition(), Data_Namespace::GPU_LEVEL, InputTableInfo::info, LOG, HashJoin::MAX_NUM_HASH_ENTRIES, OneToMany, and Analyzer::RangeOper::toString().
Referenced by BoundingBoxIntersectJoinHashTable::getInstance().
|
protected |
Definition at line 448 of file RangeJoinHashTable.cpp.
References BBOX_INTERSECT_HT, CHECK, DataRecyclerUtil::CPU_DEVICE_IDENTIFIER, DEBUG_TIMER, BoundingBoxIntersectJoinHashTable::executor_, HashJoin::getCompositeKeyInfo(), HashTableEntryInfo::getHashTableLayout(), BoundingBoxIntersectJoinHashTable::hashtable_cache_key_, BoundingBoxIntersectJoinHashTable::inner_outer_pairs_, isInnerColCompressed(), BoundingBoxIntersectJoinHashTable::join_type_, HashJoin::layoutRequiresAdditionalBuffers(), BoundingBoxIntersectJoinHashTable::putHashTableOnCpuToCache(), BoundingBoxIntersectJoinHashTable::query_hints_, and to_string().
Referenced by reifyForDevice().
|
inlineprivate |
Definition at line 112 of file RangeJoinHashTable.h.
References SQLTypeInfo::get_compression(), Analyzer::RangeOper::get_left_operand(), Analyzer::Expr::get_type_info(), kENCODING_GEOINT, and range_expr_.
Referenced by approximateTupleCount(), and initHashTableOnCpu().
|
inlineprivate |
Definition at line 117 of file RangeJoinHashTable.h.
References BoundingBoxIntersectJoinHashTable::getInnerOuterPairs(), and kENCODING_GEOINT.
Referenced by codegenKey().
|
protected |
Definition at line 346 of file RangeJoinHashTable.cpp.
References CHECK, CHECK_EQ, CHECK_LT, Data_Namespace::CPU_LEVEL, DEBUG_TIMER_NEW_THREAD, effective_memory_level_, BoundingBoxIntersectJoinHashTable::getKeyComponentCount(), BoundingBoxIntersectJoinHashTable::getKeyComponentWidth(), Data_Namespace::GPU_LEVEL, HashJoin::hash_tables_for_device_, initHashTableOnCpu(), ColumnsForDevice::join_buckets, ColumnsForDevice::join_column_types, ColumnsForDevice::join_columns, HashJoin::layoutRequiresAdditionalBuffers(), BoundingBoxIntersectJoinHashTable::memory_level_, logger::ThreadLocalIds::setNewThreadId(), logger::ThreadLocalIds::thread_id_, UNREACHABLE, and VLOG.
Referenced by reifyWithLayout().
|
overrideprotectedvirtual |
Reimplemented from BoundingBoxIntersectJoinHashTable.
Definition at line 155 of file RangeJoinHashTable.cpp.
References threading_serial::async(), BBOX_INTERSECT_HT, bucket_threshold_, CompositeKeyInfo::cache_key_chunks, BoundingBoxIntersectJoinHashTable::calculateHashTableSize(), CHECK, CHECK_EQ, CHECK_LT, HashJoin::collectFragmentIds(), BoundingBoxIntersectJoinHashTable::composite_key_info_, computeRangeHashTableCounts(), BoundingBoxIntersectJoinHashTable::condition_, DataRecyclerUtil::CPU_DEVICE_IDENTIFIER, BoundingBoxIntersectJoinHashTable::cpu_hash_table_buff_mutex_, Data_Namespace::CPU_LEVEL, DEBUG_TIMER, BoundingBoxIntersectJoinHashTable::device_count_, effective_memory_level_, BoundingBoxIntersectJoinHashTable::executor_, BoundingBoxIntersectJoinHashTable::fetchColumnsForDevice(), BoundingBoxIntersectJoinHashTable::generateCacheKey(), get_inner_query_info(), Analyzer::RangeOper::get_left_operand(), Analyzer::RangeOper::get_right_operand(), BoundingBoxIntersectJoinHashTable::getAlternativeCacheKey(), DataRecyclerUtil::getAlternativeTableKeys(), BoundingBoxIntersectJoinHashTable::getEffectiveMemoryLevel(), HashtableRecycler::getHashtableAccessPathInfo(), HashJoin::getHashTypeString(), HashJoin::getInnerTableId(), BoundingBoxIntersectJoinHashTable::getInnerTableId(), getQueryEngineCudaStreamForDevice(), Data_Namespace::GPU_LEVEL, BoundingBoxIntersectJoinHashTable::hash_table_cache_, HashJoin::hash_tables_for_device_, hash_value(), BoundingBoxIntersectJoinHashTable::hashtable_build_dag_map_, BoundingBoxIntersectJoinHashTable::hashtable_cache_key_, InputTableInfo::info, BoundingBoxIntersectJoinHashTable::initHashTableOnCpuFromCache(), BoundingBoxIntersectJoinHashTable::inner_outer_pairs_, BoundingBoxIntersectJoinHashTable::inverse_bucket_sizes_for_dimension_, HashtableRecycler::isInvalidHashTableCacheKey(), BoundingBoxIntersectJoinHashTable::join_type_, BoundingBoxIntersectJoinHashTable::layout_override_, ManyToMany, max_hashtable_size_, BoundingBoxIntersectJoinHashTable::memory_level_, OneToMany, only_shards_for_device(), BoundingBoxIntersectJoinHashTable::query_infos_, range_expr_, reifyForDevice(), BoundingBoxIntersectJoinHashTable::setBoundingBoxIntersectionMetaInfo(), BoundingBoxIntersectJoinHashTable::setInverseBucketSizeInfo(), BoundingBoxIntersectJoinHashTable::shardCount(), BoundingBoxIntersectJoinHashTable::table_keys_, logger::thread_local_ids(), UNREACHABLE, and VLOG.
|
private |
Definition at line 128 of file RangeJoinHashTable.h.
Referenced by computeRangeHashTableCounts(), and reifyWithLayout().
|
private |
Definition at line 127 of file RangeJoinHashTable.h.
Referenced by approximateTupleCount(), reifyForDevice(), and reifyWithLayout().
|
private |
Definition at line 126 of file RangeJoinHashTable.h.
|
private |
Definition at line 129 of file RangeJoinHashTable.h.
Referenced by computeRangeHashTableCounts(), and reifyWithLayout().
|
private |
Definition at line 125 of file RangeJoinHashTable.h.
Referenced by isInnerColCompressed(), and reifyWithLayout().