OmniSciDB
a5dc49c757
|
#include <HashJoin.h>
Public Member Functions | |
virtual std::string | toString (const ExecutorDeviceType device_type, const int device_id=0, bool raw=false) const =0 |
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 |
virtual DecodedJoinHashBufferSet | toSet (const ExecutorDeviceType device_type, const int device_id) const =0 |
virtual llvm::Value * | codegenSlot (const CompilationOptions &, const size_t)=0 |
virtual HashJoinMatchingSet | codegenMatchingSet (const CompilationOptions &, const size_t)=0 |
virtual shared::TableKey | getInnerTableId () const noexcept=0 |
virtual int | getInnerTableRteIdx () const noexcept=0 |
virtual HashType | getHashType () const noexcept=0 |
virtual Data_Namespace::MemoryLevel | getMemoryLevel () const noexcept=0 |
virtual int | getDeviceCount () const noexcept=0 |
virtual size_t | offsetBufferOff () const noexcept=0 |
virtual size_t | countBufferOff () const noexcept=0 |
virtual size_t | payloadBufferOff () const noexcept=0 |
virtual std::string | getHashJoinType () const =0 |
virtual bool | isBitwiseEq () const =0 |
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 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) |
Static Public Attributes | |
static constexpr size_t | MAX_NUM_HASH_ENTRIES = size_t(1) << 31 |
Protected Member Functions | |
virtual size_t | getComponentBufferSize () const noexcept=0 |
Static Protected Member Functions | |
static llvm::Value * | codegenColOrStringOper (const Analyzer::Expr *col_or_string_oper, const std::vector< StringOps_Namespace::StringOpInfo > &string_op_infos, CodeGenerator &code_generator, const CompilationOptions &co) |
Protected Attributes | |
std::vector< std::shared_ptr < HashTable > > | hash_tables_for_device_ |
Definition at line 135 of file HashJoin.h.
|
static |
Definition at line 1049 of file HashJoin.cpp.
References g_enable_data_recycler, g_use_hashtable_cache, and INVALID.
Referenced by PerfectJoinHashTable::reify(), and BaselineJoinHashTable::reifyWithLayout().
|
static |
Definition at line 796 of file HashJoin.cpp.
References CHECK, g_cluster, Catalog_Namespace::get_metadata_for_table(), shared::TableKey::table_id, and table_is_replicated().
Referenced by RangeJoinHashTable::getInstance(), BaselineJoinHashTable::reify(), and PerfectJoinHashTable::reify().
|
staticprotected |
Definition at line 564 of file HashJoin.cpp.
References CHECK, CodeGenerator::codegen(), and CodeGenerator::codegenPseudoStringOper().
Referenced by BaselineJoinHashTable::codegenKey(), PerfectJoinHashTable::codegenMatchingSet(), and PerfectJoinHashTable::codegenSlot().
|
static |
Definition at line 259 of file HashJoin.cpp.
References AUTOMATIC_IR_METADATA, CHECK, CHECK_LT, and get_arg_by_name().
Referenced by PerfectJoinHashTable::codegenHashTableLoad(), BaselineJoinHashTable::codegenMatchingSet(), BoundingBoxIntersectJoinHashTable::codegenMatchingSet(), RangeJoinHashTable::codegenMatchingSetWithOffset(), and BaselineJoinHashTable::hashPtr().
|
pure virtual |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
Referenced by PerfectJoinHashTable::codegenMatchingSet(), BaselineJoinHashTable::codegenMatchingSet(), BoundingBoxIntersectJoinHashTable::codegenMatchingSet(), and RangeJoinHashTable::codegenMatchingSetWithOffset().
|
static |
Definition at line 208 of file HashJoin.cpp.
References AUTOMATIC_IR_METADATA, and CHECK.
|
pure virtual |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
static |
Definition at line 461 of file HashJoin.cpp.
References gpu_enabled::sort(), and shared::transform().
Referenced by BoundingBoxIntersectJoinHashTable::generateCacheKey(), HashtableRecycler::getHashtableAccessPathInfo(), RangeJoinHashTable::reifyWithLayout(), and BoundingBoxIntersectJoinHashTable::reifyWithLayout().
|
pure virtualnoexcept |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
JoinColumn HashJoin::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 | ||
) |
fetchJoinColumn() calls ColumnFetcher::makeJoinColumn(), then copies the JoinColumn's col_chunks_buff memory onto the GPU if required by the effective_memory_level parameter. The dev_buff_owner parameter will manage the GPU memory.
Definition at line 60 of file HashJoin.cpp.
References Allocator::alloc(), CHECK, JoinColumn::col_chunks_buff, JoinColumn::col_chunks_buff_sz, DeviceAllocator::copyToDevice(), Data_Namespace::GPU_LEVEL, and ColumnFetcher::makeJoinColumn().
Referenced by BoundingBoxIntersectJoinHashTable::fetchColumnsForDevice(), PerfectJoinHashTable::fetchColumnsForDevice(), and BaselineJoinHashTable::fetchColumnsForDevice().
|
inline |
Definition at line 338 of file HashJoin.h.
References hash_tables_for_device_.
Referenced by BaselineJoinHashTable::reify(), and PerfectJoinHashTable::reify().
|
inlinestatic |
Definition at line 166 of file HashJoin.h.
References toString().
Referenced by PerfectJoinHashTable::getInstance().
|
protectedpure virtualnoexcept |
Implemented in PerfectJoinHashTable, BoundingBoxIntersectJoinHashTable, and BaselineJoinHashTable.
|
static |
Definition at line 470 of file HashJoin.cpp.
References CHECK, CHECK_EQ, and kENCODING_DICT.
Referenced by RangeJoinHashTable::approximateTupleCount(), BaselineJoinHashTable::initHashTableForDevice(), RangeJoinHashTable::initHashTableOnCpu(), BoundingBoxIntersectJoinHashTable::reify(), and BaselineJoinHashTable::reifyWithLayout().
|
pure virtualnoexcept |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
static |
|
pure virtual |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
inline |
Definition at line 296 of file HashJoin.h.
References CHECK_LT, and hash_tables_for_device_.
Referenced by BoundingBoxIntersectJoinHashTable::codegenManyKey(), BaselineJoinHashTable::codegenMatchingSet(), BaselineJoinHashTable::codegenSlot(), BaselineJoinHashTable::getComponentBufferSize(), BoundingBoxIntersectJoinHashTable::getEmittedKeysCount(), BoundingBoxIntersectJoinHashTable::getEntryCount(), BaselineJoinHashTable::getHashType(), BoundingBoxIntersectJoinHashTable::getHashType(), getJoinHashBufferSize(), BaselineJoinHashTable::getKeyBufferSize(), BaselineJoinHashTable::toSet(), and BoundingBoxIntersectJoinHashTable::toSet().
|
pure virtualnoexcept |
Implemented in BoundingBoxIntersectJoinHashTable, RangeJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
inlinestaticnoexcept |
Definition at line 180 of file HashJoin.h.
Referenced by BaselineJoinHashTable::getInstance(), PerfectJoinHashTable::getInstance(), BoundingBoxIntersectJoinHashTable::getInstance(), RangeJoinHashTable::reifyWithLayout(), BoundingBoxIntersectJoinHashTable::reifyWithLayout(), HashingSchemeRecycler::toString(), PerfectJoinHashTable::toString(), BaselineJoinHashTable::toString(), and BoundingBoxIntersectJoinHashTable::toString().
|
pure virtualnoexcept |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
Referenced by BoundingBoxIntersectJoinHashTable::getInnerTableId(), RangeJoinHashTable::getInstance(), BoundingBoxIntersectJoinHashTable::getInstance(), RangeJoinHashTable::reifyWithLayout(), and BoundingBoxIntersectJoinHashTable::reifyWithLayout().
|
inlinestatic |
|
pure virtualnoexcept |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
static |
Make hash table from an in-flight SQL query's parse tree etc.
Definition at line 285 of file HashJoin.cpp.
References CHECK, CHECK_EQ, CHECK_GT, coalesce_singleton_equi_join(), CPU, DEBUG_TIMER, RegisteredQueryHint::force_baseline_hash_join, g_enable_bbox_intersect_hashjoin, BaselineJoinHashTable::getInstance(), PerfectJoinHashTable::getInstance(), BoundingBoxIntersectJoinHashTable::getInstance(), GPU, Data_Namespace::GPU_LEVEL, logger::INFO, LOG, VLOG, and VLOGGING.
Referenced by Executor::buildHashTableForQualifier(), and getSyntheticInstance().
|
inline |
Definition at line 315 of file HashJoin.h.
References CHECK, CHECK_LT, CPU, and hash_tables_for_device_.
Referenced by PerfectJoinHashTable::toSet(), BaselineJoinHashTable::toSet(), BoundingBoxIntersectJoinHashTable::toSet(), PerfectJoinHashTable::toString(), BaselineJoinHashTable::toString(), BoundingBoxIntersectJoinHashTable::toString(), and anonymous_namespace{HashJoin.cpp}::toStringFlat().
|
inline |
Definition at line 301 of file HashJoin.h.
Referenced by PerfectJoinHashTable::toSet(), PerfectJoinHashTable::toString(), and anonymous_namespace{HashJoin.cpp}::toStringFlat().
|
inline |
Definition at line 306 of file HashJoin.h.
References getHashTableForDevice().
|
staticnoexcept |
Definition at line 1056 of file HashJoin.cpp.
References CHECK_GT, Data_Namespace::CPU_LEVEL, and MAX_NUM_HASH_ENTRIES.
Referenced by PerfectJoinHashTable::getInstance().
|
pure virtualnoexcept |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
static |
Definition at line 394 of file HashJoin.cpp.
References CHECK, CHECK_EQ, and Analyzer::Expr::get_type_info().
Referenced by translateInnerToOuterStrDictProxies().
|
static |
Make hash table from named tables and columns (such as for testing).
Definition at line 690 of file HashJoin.cpp.
References RegisteredQueryHint::defaults(), getInstance(), getSyntheticColumnVar(), getSyntheticInputTableInfo(), INNER, kBOOLEAN, kEQ, kONE, setupSyntheticCaching(), and ScalarExprVisitor< T >::visit().
|
static |
Make hash table from named tables and columns (such as for testing).
Definition at line 728 of file HashJoin.cpp.
References RegisteredQueryHint::defaults(), getInstance(), getSyntheticInputTableInfo(), INNER, setupSyntheticCaching(), and ScalarExprVisitor< T >::visit().
|
static |
Definition at line 755 of file HashJoin.cpp.
References RegisteredQueryHint::defaults(), getInstance(), getSyntheticInputTableInfo(), INNER, setupSyntheticCaching(), and ScalarExprVisitor< T >::visit().
|
pure virtual |
Implemented in BoundingBoxIntersectJoinHashTable, PerfectJoinHashTable, and BaselineJoinHashTable.
|
inlinestaticnoexcept |
Definition at line 176 of file HashJoin.h.
References ManyToMany, and OneToMany.
Referenced by BaselineHashTableEntryInfo::computeNumAdditionalSlotsForOneToManyJoin(), BaselineJoinHashTable::countBufferOff(), BoundingBoxIntersectJoinHashTable::countBufferOff(), BaselineJoinHashTable::getKeyBufferSize(), BoundingBoxIntersectJoinHashTable::getKeyBufferSize(), RangeJoinHashTable::initHashTableOnCpu(), BoundingBoxIntersectJoinHashTable::initHashTableOnCpu(), BaselineJoinHashTableBuilder::initHashTableOnCpu(), BaselineJoinHashTableBuilder::initHashTableOnGpu(), BaselineJoinHashTable::payloadBufferOff(), BoundingBoxIntersectJoinHashTable::payloadBufferOff(), RangeJoinHashTable::reifyForDevice(), BoundingBoxIntersectJoinHashTable::reifyForDevice(), and BoundingBoxIntersectJoinHashTable::reifyWithLayout().
|
static |
Definition at line 822 of file HashJoin.cpp.
References CHECK, CHECK_GT, get_column_descriptor_maybe(), get_column_type(), SQLTypeInfo::get_precision(), SQLTypeInfo::get_scale(), SQLTypeInfo::get_type(), Analyzer::Expr::get_type_info(), SQLTypeInfo::get_type_name(), is_constructed_point(), SQLTypeInfo::is_decimal(), SQLTypeInfo::is_integer(), SQLTypeInfo::is_string(), SQLTypeInfo::is_time(), kCAST, kENCODING_DICT, kPOINT, LHS, RHS, gpu_enabled::swap(), UNKNOWN, and ScalarExprVisitor< T >::visit().
Referenced by anonymous_namespace{PerfectJoinHashTable.cpp}::get_cols(), anonymous_namespace{HashJoin.cpp}::get_cols(), normalizeColumnPairs(), and QueryPlanDagExtractor::normalizeColumnsPair().
|
static |
Definition at line 1015 of file HashJoin.cpp.
References CHECK, CHECK_EQ, Analyzer::BinOper::get_left_operand(), Analyzer::BinOper::get_right_operand(), Analyzer::BinOper::is_bbox_intersect_oper(), normalizeColumnPair(), and run_benchmark_import::result.
Referenced by anonymous_namespace{FromTableReordering.cpp}::get_join_qual_cost(), BaselineJoinHashTable::getInstance(), BoundingBoxIntersectJoinHashTable::getInstance(), BaselineJoinHashTable::reifyWithLayout(), and Executor::skipFragmentPair().
|
pure virtualnoexcept |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
pure virtualnoexcept |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
pure virtual |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
|
pure virtual |
Implemented in BoundingBoxIntersectJoinHashTable, BaselineJoinHashTable, and PerfectJoinHashTable.
Referenced by generateTooManyHashEntriesErrMsg().
|
virtual |
Definition at line 123 of file HashJoin.cpp.
|
virtual |
Definition at line 118 of file HashJoin.cpp.
|
static |
Definition at line 528 of file HashJoin.cpp.
References CHECK, CHECK_EQ, CHECK_NE, CompositeKeyInfo::sd_inner_proxy_per_key, and CompositeKeyInfo::sd_outer_proxy_per_key.
Referenced by BaselineJoinHashTable::reifyWithLayout().
|
static |
Definition at line 423 of file HashJoin.cpp.
References CHECK, ExpressionRange::getIntMax(), ExpressionRange::getIntMin(), getStrDictProxies(), ExpressionRange::hasNulls(), ExpressionRange::makeIntRange(), and StringDictionaryProxy::TranslationMap< T >::size().
Referenced by PerfectJoinHashTable::reify().
|
protected |
Definition at line 378 of file HashJoin.h.
Referenced by BaselineJoinHashTable::BaselineJoinHashTable(), BoundingBoxIntersectJoinHashTable::BoundingBoxIntersectJoinHashTable(), BaselineJoinHashTable::copyCpuHashTableToGpu(), PerfectJoinHashTable::copyCpuHashTableToGpu(), freeHashBufferMemory(), BoundingBoxIntersectJoinHashTable::getComponentBufferSize(), PerfectJoinHashTable::getComponentBufferSize(), PerfectJoinHashTable::getHashTableForDevice(), getHashTableForDevice(), getJoinHashBuffer(), PerfectJoinHashTable::initHashTableForDevice(), BaselineJoinHashTable::initHashTableForDevice(), PerfectJoinHashTable::reify(), RangeJoinHashTable::reifyForDevice(), BoundingBoxIntersectJoinHashTable::reifyForDevice(), RangeJoinHashTable::reifyWithLayout(), BaselineJoinHashTable::reifyWithLayout(), BaselineJoinHashTable::toString(), and BoundingBoxIntersectJoinHashTable::toString().
|
static |
Definition at line 137 of file HashJoin.h.
Referenced by RangeJoinHashTable::getInstance(), BoundingBoxIntersectJoinHashTable::getInstance(), getMaximumNumHashEntriesCanHold(), PerfectJoinHashTable::reify(), and BaselineJoinHashTable::reifyWithLayout().