OmniSciDB  a5dc49c757
 All Classes Namespaces Files Functions Variables Typedefs Enumerations Enumerator Friends Macros Groups Pages
HashJoin.h
Go to the documentation of this file.
1 /*
2  * Copyright 2022 HEAVY.AI, Inc.
3  *
4  * Licensed under the Apache License, Version 2.0 (the "License");
5  * you may not use this file except in compliance with the License.
6  * You may obtain a copy of the License at
7  *
8  * http://www.apache.org/licenses/LICENSE-2.0
9  *
10  * Unless required by applicable law or agreed to in writing, software
11  * distributed under the License is distributed on an "AS IS" BASIS,
12  * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
13  * See the License for the specific language governing permissions and
14  * limitations under the License.
15  */
16 
17 #pragma once
18 
19 #include <llvm/IR/Value.h>
20 #include <cstdint>
21 #include <set>
22 #include <string>
23 
24 #include "Analyzer/Analyzer.h"
33 #include "Shared/DbObjectKeys.h"
34 #include "StringOps/StringOpInfo.h"
35 
36 class CodeGenerator;
37 
38 class JoinHashTableTooBig : public std::runtime_error {
39  public:
40  JoinHashTableTooBig(size_t cur_hash_table_size, size_t threshold_size)
41  : std::runtime_error("The size of hash table is larger than a threshold (" +
42  ::toString(cur_hash_table_size) + " > " +
43  ::toString(threshold_size) + ")") {}
44 };
45 
46 class TooManyHashEntries : public std::runtime_error {
47  public:
49  : std::runtime_error("Hash tables with more than 4B entries not supported yet") {}
50 
51  TooManyHashEntries(const std::string& reason) : std::runtime_error(reason) {}
52 };
53 
54 class TableMustBeReplicated : public std::runtime_error {
55  public:
56  TableMustBeReplicated(const std::string& table_name)
57  : std::runtime_error("Hash join failed: Table '" + table_name +
58  "' must be replicated.") {}
59 };
60 
61 enum class InnerQualDecision { IGNORE = 0, UNKNOWN, LHS, RHS };
62 
63 #ifndef __CUDACC__
64 inline std::ostream& operator<<(std::ostream& os, InnerQualDecision const decision) {
65  constexpr char const* strings[]{"IGNORE", "UNKNOWN", "LHS", "RHS"};
66  return os << strings[static_cast<int>(decision)];
67 }
68 #endif
69 
70 class HashJoinFail : public std::runtime_error {
71  public:
72  HashJoinFail(const std::string& err_msg)
73  : std::runtime_error(err_msg), inner_qual_decision(InnerQualDecision::UNKNOWN) {}
74  HashJoinFail(const std::string& err_msg, InnerQualDecision qual_decision)
75  : std::runtime_error(err_msg), inner_qual_decision(qual_decision) {}
76 
78 };
79 
81  public:
82  NeedsOneToManyHash() : HashJoinFail("Needs one to many hash") {}
83 };
84 
86  public:
88  : HashJoinFail("Not enough memory for columns involved in join") {}
89 };
90 
92  public:
93  FailedToJoinOnVirtualColumn() : HashJoinFail("Cannot join on rowid") {}
94 };
95 
97  public:
98  TooBigHashTableForBoundingBoxIntersect(const size_t bbox_intersect_hash_table_max_bytes)
99  : HashJoinFail(
100  "Could not create hash table for bounding box intersection with less than "
101  "max allowed size of " +
102  std::to_string(bbox_intersect_hash_table_max_bytes) + " bytes") {}
103 };
104 
105 using InnerOuter = std::pair<const Analyzer::ColumnVar*, const Analyzer::Expr*>;
106 using InnerOuterStringOpInfos = std::pair<std::vector<StringOps_Namespace::StringOpInfo>,
107  std::vector<StringOps_Namespace::StringOpInfo>>;
108 
110  const std::vector<JoinColumn> join_columns;
111  const std::vector<JoinColumnTypeInfo> join_column_types;
112  const std::vector<std::shared_ptr<Chunk_NS::Chunk>> chunks_owner;
113  std::vector<JoinBucketInfo> join_buckets;
114  const std::vector<std::shared_ptr<void>> malloc_owner;
115 
116  void setBucketInfo(const std::vector<double>& bucket_sizes_for_dimension,
117  const std::vector<InnerOuter> inner_outer_pairs);
118 };
119 
121  llvm::Value* elements;
122  llvm::Value* count;
123  llvm::Value* slot;
124  llvm::Value* error_code;
125 };
126 
128  std::vector<const void*> sd_inner_proxy_per_key;
129  std::vector<void*> sd_outer_proxy_per_key;
130  std::vector<ChunkKey> cache_key_chunks; // used for the cache key
131 };
132 
133 class DeviceAllocator;
134 
135 class HashJoin {
136  public:
137  static constexpr size_t MAX_NUM_HASH_ENTRIES = size_t(1) << 31;
138  virtual std::string toString(const ExecutorDeviceType device_type,
139  const int device_id = 0,
140  bool raw = false) const = 0;
141 
142  virtual std::string toStringFlat64(const ExecutorDeviceType device_type,
143  const int device_id) const;
144 
145  virtual std::string toStringFlat32(const ExecutorDeviceType device_type,
146  const int device_id) const;
147 
148  virtual DecodedJoinHashBufferSet toSet(const ExecutorDeviceType device_type,
149  const int device_id) const = 0;
150 
151  virtual llvm::Value* codegenSlot(const CompilationOptions&, const size_t) = 0;
152 
154  const size_t) = 0;
155 
156  virtual shared::TableKey getInnerTableId() const noexcept = 0;
157 
158  virtual int getInnerTableRteIdx() const noexcept = 0;
159 
160  virtual HashType getHashType() const noexcept = 0;
161 
162  static size_t getMaximumNumHashEntriesCanHold(MemoryLevel memory_level,
163  const Executor* executor,
164  size_t rowid_size) noexcept;
165 
166  static std::string generateTooManyHashEntriesErrMsg(size_t num_entries,
167  size_t threshold,
168  MemoryLevel memory_level) {
169  std::ostringstream oss;
170  oss << "Hash tables with more than " << threshold
171  << " entries (# hash entries: " << num_entries << ") on "
172  << ::toString(memory_level) << " not supported yet";
173  return oss.str();
174  }
175 
176  static bool layoutRequiresAdditionalBuffers(HashType layout) noexcept {
177  return (layout == HashType::ManyToMany || layout == HashType::OneToMany);
178  }
179 
180  static std::string getHashTypeString(HashType ht) noexcept {
181  const char* HashTypeStrings[3] = {"OneToOne", "OneToMany", "ManyToMany"};
182  return HashTypeStrings[static_cast<int>(ht)];
183  };
184 
186  const std::vector<llvm::Value*>& hash_join_idx_args_in,
187  const bool is_sharded,
188  const bool col_is_nullable,
189  const bool is_bw_eq,
190  const int64_t sub_buff_size,
191  Executor* executor,
192  const bool is_bucketized = false);
193 
194  static llvm::Value* codegenHashTableLoad(const size_t table_idx, Executor* executor);
195 
196  virtual Data_Namespace::MemoryLevel getMemoryLevel() const noexcept = 0;
197 
198  virtual int getDeviceCount() const noexcept = 0;
199 
200  virtual size_t offsetBufferOff() const noexcept = 0;
201 
202  virtual size_t countBufferOff() const noexcept = 0;
203 
204  virtual size_t payloadBufferOff() const noexcept = 0;
205 
206  virtual std::string getHashJoinType() const = 0;
207 
208  virtual bool isBitwiseEq() const = 0;
209 
211  const Analyzer::ColumnVar* hash_col,
212  const std::vector<Fragmenter_Namespace::FragmentInfo>& fragment_info,
213  const Data_Namespace::MemoryLevel effective_memory_level,
214  const int device_id,
215  std::vector<std::shared_ptr<Chunk_NS::Chunk>>& chunks_owner,
216  DeviceAllocator* dev_buff_owner,
217  std::vector<std::shared_ptr<void>>& malloc_owner,
218  Executor* executor,
219  ColumnCacheMap* column_cache);
220 
222  static std::shared_ptr<HashJoin> getInstance(
223  const std::shared_ptr<Analyzer::BinOper> qual_bin_oper,
224  const std::vector<InputTableInfo>& query_infos,
225  const Data_Namespace::MemoryLevel memory_level,
226  const JoinType join_type,
227  const HashType preferred_hash_type,
228  const int device_count,
229  ColumnCacheMap& column_cache,
230  Executor* executor,
231  const HashTableBuildDagMap& hashtable_build_dag_map,
232  const RegisteredQueryHint& query_hint,
233  const TableIdToNodeMap& table_id_to_node_map);
234 
236  static std::shared_ptr<HashJoin> getSyntheticInstance(
237  std::string_view table1,
238  std::string_view column1,
239  const Catalog_Namespace::Catalog& catalog1,
240  std::string_view table2,
241  std::string_view column2,
242  const Catalog_Namespace::Catalog& catalog2,
243  const Data_Namespace::MemoryLevel memory_level,
244  const HashType preferred_hash_type,
245  const int device_count,
246  ColumnCacheMap& column_cache,
247  Executor* executor);
248 
250  static std::shared_ptr<HashJoin> getSyntheticInstance(
251  const std::shared_ptr<Analyzer::BinOper> qual_bin_oper,
252  const Data_Namespace::MemoryLevel memory_level,
253  const HashType preferred_hash_type,
254  const int device_count,
255  ColumnCacheMap& column_cache,
256  Executor* executor);
257 
258  static std::pair<std::string, std::shared_ptr<HashJoin>> getSyntheticInstance(
259  std::vector<std::shared_ptr<Analyzer::BinOper>>,
260  const Data_Namespace::MemoryLevel memory_level,
261  const HashType preferred_hash_type,
262  const int device_count,
263  ColumnCacheMap& column_cache,
264  Executor* executor);
265 
266  static shared::TableKey getInnerTableId(
267  const std::vector<InnerOuter>& inner_outer_pairs) {
268  CHECK(!inner_outer_pairs.empty());
269  const auto first_inner_col = inner_outer_pairs.front().first;
270  return first_inner_col->getTableKey();
271  }
272 
273  static bool canAccessHashTable(bool allow_hash_table_recycling,
274  bool invalid_cache_key,
275  JoinType join_type);
276 
277  static void checkHashJoinReplicationConstraint(const shared::TableKey& table_key,
278  const size_t shard_count,
279  const Executor* executor);
280 
281  // Swap the columns if needed and make the inner column the first component.
282  static std::pair<InnerOuter, InnerOuterStringOpInfos> normalizeColumnPair(
283  const Analyzer::Expr* lhs,
284  const Analyzer::Expr* rhs,
285  const TemporaryTables* temporary_tables,
286  const bool is_bbox_intersect = false);
287 
288  template <typename T>
289  static const T* getHashJoinColumn(const Analyzer::Expr* expr);
290 
291  // Normalize each expression tuple
292  static std::pair<std::vector<InnerOuter>, std::vector<InnerOuterStringOpInfos>>
293  normalizeColumnPairs(const Analyzer::BinOper* condition,
294  const TemporaryTables* temporary_tables);
295 
296  HashTable* getHashTableForDevice(const size_t device_id) const {
297  CHECK_LT(device_id, hash_tables_for_device_.size());
298  return hash_tables_for_device_[device_id].get();
299  }
300 
301  size_t getJoinHashBufferSize(const ExecutorDeviceType device_type) {
302  CHECK(device_type == ExecutorDeviceType::CPU);
303  return getJoinHashBufferSize(device_type, 0);
304  }
305 
306  size_t getJoinHashBufferSize(const ExecutorDeviceType device_type,
307  const int device_id) const {
308  auto hash_table = getHashTableForDevice(device_id);
309  if (!hash_table) {
310  return 0;
311  }
312  return hash_table->getHashTableBufferSize(device_type);
313  }
314 
315  int8_t* getJoinHashBuffer(const ExecutorDeviceType device_type,
316  const int device_id) const {
317  // TODO: just make device_id a size_t
318  CHECK_LT(size_t(device_id), hash_tables_for_device_.size());
319  if (!hash_tables_for_device_[device_id]) {
320  return nullptr;
321  }
322  CHECK(hash_tables_for_device_[device_id]);
323  auto hash_table = hash_tables_for_device_[device_id].get();
324 #ifdef HAVE_CUDA
325  if (device_type == ExecutorDeviceType::CPU) {
326  return hash_table->getCpuBuffer();
327  } else {
328  CHECK(hash_table);
329  const auto gpu_buff = hash_table->getGpuBuffer();
330  return gpu_buff;
331  }
332 #else
333  CHECK(device_type == ExecutorDeviceType::CPU);
334  return hash_table->getCpuBuffer();
335 #endif
336  }
337 
339  auto empty_hash_tables =
341  hash_tables_for_device_.swap(empty_hash_tables);
342  }
343 
344  static std::vector<int> collectFragmentIds(
345  const std::vector<Fragmenter_Namespace::FragmentInfo>& fragments);
346 
348  const std::vector<InnerOuter>& inner_outer_pairs,
349  const Executor* executor,
350  const std::vector<InnerOuterStringOpInfos>& inner_outer_string_op_infos_pairs = {});
351 
352  static std::vector<const StringDictionaryProxy::IdMap*>
354  const CompositeKeyInfo& composite_key_info,
355  const std::vector<InnerOuterStringOpInfos>& string_op_infos_for_keys,
356  const Executor* executor);
357 
358  static std::pair<const StringDictionaryProxy*, StringDictionaryProxy*>
359  getStrDictProxies(const InnerOuter& cols,
360  const Executor* executor,
361  const bool has_string_ops);
362 
364  const InnerOuter& cols,
365  const InnerOuterStringOpInfos& inner_outer_string_op_infos,
366  ExpressionRange& old_col_range,
367  const Executor* executor);
368 
369  protected:
370  static llvm::Value* codegenColOrStringOper(
371  const Analyzer::Expr* col_or_string_oper,
372  const std::vector<StringOps_Namespace::StringOpInfo>& string_op_infos,
373  CodeGenerator& code_generator,
374  const CompilationOptions& co);
375 
376  virtual size_t getComponentBufferSize() const noexcept = 0;
377 
378  std::vector<std::shared_ptr<HashTable>> hash_tables_for_device_;
379 };
380 
381 std::ostream& operator<<(std::ostream& os, const DecodedJoinHashBufferEntry& e);
382 
383 std::ostream& operator<<(std::ostream& os, const DecodedJoinHashBufferSet& s);
384 
385 std::ostream& operator<<(std::ostream& os,
386  const InnerOuterStringOpInfos& inner_outer_string_op_infos);
387 std::ostream& operator<<(
388  std::ostream& os,
389  const std::vector<InnerOuterStringOpInfos>& inner_outer_string_op_infos_pairs);
390 
391 std::string toString(const InnerOuterStringOpInfos& inner_outer_string_op_infos);
392 
393 std::string toString(
394  const std::vector<InnerOuterStringOpInfos>& inner_outer_string_op_infos_pairs);
395 
396 std::shared_ptr<Analyzer::ColumnVar> getSyntheticColumnVar(
397  std::string_view table,
398  std::string_view column,
399  int rte_idx,
400  const Catalog_Namespace::Catalog& catalog);
401 
402 size_t get_shard_count(const Analyzer::BinOper* join_condition, const Executor* executor);
403 
404 size_t get_shard_count(
405  std::pair<const Analyzer::ColumnVar*, const Analyzer::Expr*> equi_pair,
406  const Executor* executor);
static std::vector< int > collectFragmentIds(const std::vector< Fragmenter_Namespace::FragmentInfo > &fragments)
Definition: HashJoin.cpp:461
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).
Definition: HashJoin.cpp:690
Defines data structures for the semantic analysis phase of query processing.
virtual int getInnerTableRteIdx() const noexcept=0
virtual size_t payloadBufferOff() const noexcept=0
virtual std::string getHashJoinType() const =0
virtual HashJoinMatchingSet codegenMatchingSet(const CompilationOptions &, const size_t)=0
JoinType
Definition: sqldefs.h:238
static llvm::Value * codegenHashTableLoad(const size_t table_idx, Executor *executor)
Definition: HashJoin.cpp:259
std::pair< const Analyzer::ColumnVar *, const Analyzer::Expr * > InnerOuter
Definition: HashJoin.h:105
static bool canAccessHashTable(bool allow_hash_table_recycling, bool invalid_cache_key, JoinType join_type)
Definition: HashJoin.cpp:1049
virtual HashType getHashType() const noexcept=0
std::vector< ChunkKey > cache_key_chunks
Definition: HashJoin.h:130
std::vector< const void * > sd_inner_proxy_per_key
Definition: HashJoin.h:128
virtual int getDeviceCount() const noexcept=0
virtual std::string toStringFlat64(const ExecutorDeviceType device_type, const int device_id) const
Definition: HashJoin.cpp:118
static void checkHashJoinReplicationConstraint(const shared::TableKey &table_key, const size_t shard_count, const Executor *executor)
Definition: HashJoin.cpp:796
std::ostream & operator<<(std::ostream &os, const SessionInfo &session_info)
Definition: SessionInfo.cpp:57
void setBucketInfo(const std::vector< double > &bucket_sizes_for_dimension, const std::vector< InnerOuter > inner_outer_pairs)
Definition: HashJoin.cpp:37
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)
Definition: HashJoin.cpp:60
llvm::Value * elements
Definition: HashJoin.h:121
llvm::Value * count
Definition: HashJoin.h:122
virtual Data_Namespace::MemoryLevel getMemoryLevel() const noexcept=0
std::vector< std::shared_ptr< HashTable > > hash_tables_for_device_
Definition: HashJoin.h:378
static std::pair< const StringDictionaryProxy *, StringDictionaryProxy * > getStrDictProxies(const InnerOuter &cols, const Executor *executor, const bool has_string_ops)
Definition: HashJoin.cpp:394
Definition: HashTable.h:21
virtual llvm::Value * codegenSlot(const CompilationOptions &, const size_t)=0
TableMustBeReplicated(const std::string &table_name)
Definition: HashJoin.h:56
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)
Definition: HashJoin.cpp:564
void freeHashBufferMemory()
Definition: HashJoin.h:338
virtual size_t offsetBufferOff() const noexcept=0
ExecutorDeviceType
virtual std::string toStringFlat32(const ExecutorDeviceType device_type, const int device_id) const
Definition: HashJoin.cpp:123
std::string to_string(char const *&&v)
virtual size_t countBufferOff() const noexcept=0
std::unordered_map< int, const ResultSetPtr & > TemporaryTables
Definition: InputMetadata.h:31
const std::vector< JoinColumnTypeInfo > join_column_types
Definition: HashJoin.h:111
std::unordered_map< size_t, HashTableBuildDag > HashTableBuildDagMap
std::vector< void * > sd_outer_proxy_per_key
Definition: HashJoin.h:129
HashJoinFail(const std::string &err_msg, InnerQualDecision qual_decision)
Definition: HashJoin.h:74
static size_t getMaximumNumHashEntriesCanHold(MemoryLevel memory_level, const Executor *executor, size_t rowid_size) noexcept
Definition: HashJoin.cpp:1056
static std::string generateTooManyHashEntriesErrMsg(size_t num_entries, size_t threshold, MemoryLevel memory_level)
Definition: HashJoin.h:166
static constexpr size_t MAX_NUM_HASH_ENTRIES
Definition: HashJoin.h:137
int8_t * getJoinHashBuffer(const ExecutorDeviceType device_type, const int device_id) const
Definition: HashJoin.h:315
static std::vector< const StringDictionaryProxy::IdMap * > translateCompositeStrDictProxies(const CompositeKeyInfo &composite_key_info, const std::vector< InnerOuterStringOpInfos > &string_op_infos_for_keys, const Executor *executor)
Definition: HashJoin.cpp:528
JoinHashTableTooBig(size_t cur_hash_table_size, size_t threshold_size)
Definition: HashJoin.h:40
std::string toString(const Executor::ExtModuleKinds &kind)
Definition: Execute.h:1703
size_t getJoinHashBufferSize(const ExecutorDeviceType device_type, const int device_id) const
Definition: HashJoin.h:306
virtual size_t getComponentBufferSize() const noexcept=0
const std::vector< std::shared_ptr< Chunk_NS::Chunk > > chunks_owner
Definition: HashJoin.h:112
static const StringDictionaryProxy::IdMap * translateInnerToOuterStrDictProxies(const InnerOuter &cols, const InnerOuterStringOpInfos &inner_outer_string_op_infos, ExpressionRange &old_col_range, const Executor *executor)
Definition: HashJoin.cpp:423
HashTable * getHashTableForDevice(const size_t device_id) const
Definition: HashJoin.h:296
virtual shared::TableKey getInnerTableId() const noexcept=0
std::unordered_map< shared::TableKey, const RelAlgNode * > TableIdToNodeMap
#define CHECK_LT(x, y)
Definition: Logger.h:303
TooManyHashEntries(const std::string &reason)
Definition: HashJoin.h:51
static std::string getHashTypeString(HashType ht) noexcept
Definition: HashJoin.h:180
size_t getJoinHashBufferSize(const ExecutorDeviceType device_type)
Definition: HashJoin.h:301
std::set< DecodedJoinHashBufferEntry > DecodedJoinHashBufferSet
Definition: HashTable.h:72
std::unordered_map< shared::TableKey, std::unordered_map< int, std::shared_ptr< const ColumnarResults >>> ColumnCacheMap
HashJoinFail(const std::string &err_msg)
Definition: HashJoin.h:72
std::shared_ptr< Analyzer::ColumnVar > getSyntheticColumnVar(std::string_view table, std::string_view column, int rte_idx, const Catalog_Namespace::Catalog &catalog)
Definition: HashJoin.cpp:580
TooBigHashTableForBoundingBoxIntersect(const size_t bbox_intersect_hash_table_max_bytes)
Definition: HashJoin.h:98
#define CHECK(condition)
Definition: Logger.h:291
std::pair< std::vector< StringOps_Namespace::StringOpInfo >, std::vector< StringOps_Namespace::StringOpInfo >> InnerOuterStringOpInfos
Definition: HashJoin.h:107
llvm::Value * slot
Definition: HashJoin.h:123
static std::pair< InnerOuter, InnerOuterStringOpInfos > normalizeColumnPair(const Analyzer::Expr *lhs, const Analyzer::Expr *rhs, const TemporaryTables *temporary_tables, const bool is_bbox_intersect=false)
Definition: HashJoin.cpp:822
static const T * getHashJoinColumn(const Analyzer::Expr *expr)
Definition: HashJoin.cpp:813
static std::pair< std::vector< InnerOuter >, std::vector< InnerOuterStringOpInfos > > normalizeColumnPairs(const Analyzer::BinOper *condition, const TemporaryTables *temporary_tables)
Definition: HashJoin.cpp:1015
FileBuffer Chunk
A Chunk is the fundamental unit of execution in Map-D.
Definition: FileMgr.h:80
const std::vector< std::shared_ptr< void > > malloc_owner
Definition: HashJoin.h:114
InnerQualDecision
Definition: HashJoin.h:61
virtual DecodedJoinHashBufferSet toSet(const ExecutorDeviceType device_type, const int device_id) const =0
std::vector< JoinBucketInfo > join_buckets
Definition: HashJoin.h:113
size_t get_shard_count(const Analyzer::BinOper *join_condition, const Executor *executor)
Definition: HashJoin.cpp:1084
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&#39;s parse tree etc.
Definition: HashJoin.cpp:285
virtual std::string toString(const ExecutorDeviceType device_type, const int device_id=0, bool raw=false) const =0
HashType
Definition: HashTable.h:19
InnerQualDecision inner_qual_decision
Definition: HashJoin.h:77
const std::vector< JoinColumn > join_columns
Definition: HashJoin.h:110
static bool layoutRequiresAdditionalBuffers(HashType layout) noexcept
Definition: HashJoin.h:176
virtual bool isBitwiseEq() const =0
llvm::Value * error_code
Definition: HashJoin.h:124
static CompositeKeyInfo getCompositeKeyInfo(const std::vector< InnerOuter > &inner_outer_pairs, const Executor *executor, const std::vector< InnerOuterStringOpInfos > &inner_outer_string_op_infos_pairs={})
Definition: HashJoin.cpp:470