OmniSciDB  a5dc49c757
 All Classes Namespaces Files Functions Variables Typedefs Enumerations Enumerator Friends Macros Groups Pages
WindowContext.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 "Analyzer/Analyzer.h"
20 #include "DataMgr/Chunk/Chunk.h"
23 #include "Shared/sqltypes.h"
24 
25 #include <functional>
26 #include <unordered_map>
27 
28 // Returns true for value window functions, false otherwise.
30  switch (kind) {
41  return true;
42  default:
43  return false;
44  }
45 }
46 
48  switch (kind) {
54  return true;
55  default:
56  return false;
57  }
58 }
59 
60 // Returns true for aggregate window functions, false otherwise.
62  switch (kind) {
72  return true;
73  default:
74  return false;
75  }
76 }
77 
79  switch (kind) {
82  return true;
83  default:
84  return false;
85  }
86 }
87 
88 class Executor;
89 
91  std::vector<int64_t*> aggregate_tree_for_integer_type_;
92  std::vector<double*> aggregate_tree_for_double_type_;
93  std::vector<SumAndCountPair<int64_t>*> derived_aggregate_tree_for_integer_type_;
94  std::vector<SumAndCountPair<double>*> derived_aggregate_tree_for_double_type_;
96 
97  void resizeStorageForWindowFraming(size_t partition_count) {
98  aggregate_tree_for_integer_type_.resize(partition_count);
99  aggregate_tree_for_double_type_.resize(partition_count);
100  derived_aggregate_tree_for_integer_type_.resize(partition_count);
101  derived_aggregate_tree_for_double_type_.resize(partition_count);
102  }
103 };
104 
108  llvm::Value* current_row_pos_lv;
109  llvm::Value* current_col_value_lv;
111  llvm::Value* int64_t_zero_val_lv;
112  llvm::Value* int64_t_one_val_lv;
114  llvm::Value* order_key_buf_ptr_lv;
115  std::string order_type_col_name;
118  llvm::Value* nulls_first_lv;
119  llvm::Value* null_start_pos_lv;
120  llvm::Value* null_end_pos_lv;
121 };
122 
128 };
129 
130 // Per-window function context which encapsulates the logic for computing the various
131 // window function kinds and keeps ownership of buffers which contain the results. For
132 // rank functions, the code generated for the projection simply reads the values and
133 // writes them to the result set. For value and aggregate functions, only the iteration
134 // order is written to the buffer, the rest is handled by generating code in a similar
135 // way we do for non-window queries.
137  public:
138  // we currently only use a single GPU to process the window function because
139  // a query step having a window function expression only has a single fragment input
140  // (i.e., push the window function expression down to the child projection node)
141  // todo (yoonmin) : support window function execution with multi-fragmented input
142  // todo (yoonmin) : support heterogeneous execution (i.e., CPU + GPU)
143  static const int NUM_EXECUTION_DEVICES = 1;
144 
145  // non-partitioned version
147  const size_t elem_count,
148  const ExecutorDeviceType device_type,
149  std::shared_ptr<RowSetMemoryOwner> row_set_mem_owner);
150 
151  // partitioned version
153  const Analyzer::WindowFunction* window_func,
154  QueryPlanHash cache_key,
155  const std::shared_ptr<HashJoin>& partitions,
156  const size_t elem_count,
157  const ExecutorDeviceType device_type,
158  std::shared_ptr<RowSetMemoryOwner> row_set_mem_owner,
159  size_t aggregation_tree_fan_out = g_window_function_aggregation_tree_fanout);
160 
162 
164 
166 
167  // Adds the order column buffer to the context and keeps ownership of it.
168  void addOrderColumn(const int8_t* column,
169  const SQLTypeInfo& ti,
170  const std::vector<std::shared_ptr<Chunk_NS::Chunk>>& chunks_owner);
171 
173 
175  const int8_t* column,
176  const std::vector<std::shared_ptr<Chunk_NS::Chunk>>& chunks_owner);
177 
178  enum class WindowComparatorResult { LT, EQ, GT };
179  using Comparator =
180  std::function<WindowFunctionContext::WindowComparatorResult(const int64_t lhs,
181  const int64_t rhs)>;
182 
183  std::vector<Comparator> createComparator(size_t partition_idx);
184 
185  // Computes the window function result to be used during the actual projection query.
186  void compute(
187  std::unordered_map<QueryPlanHash, size_t>& sorted_partition_key_ref_count_map,
188  std::unordered_map<QueryPlanHash, std::shared_ptr<std::vector<int64_t>>>&
189  sorted_partition_cache,
190  std::unordered_map<QueryPlanHash, AggregateTreeForWindowFraming>&
191  aggregate_tree_map);
192 
193  // Returns a pointer to the window function associated with this context.
195 
196  // Returns a pointer to the output buffer of the window function result.
197  const int8_t* output() const;
198 
199  // Returns a pointer to the sorted row index buffer
200  const int64_t* sortedPartition() const;
201 
202  // Returns a pointer to the value field of the aggregation state.
203  const int64_t* aggregateState() const;
204 
205  // Returns a pointer to the count field of the aggregation state.
206  const int64_t* aggregateStateCount() const;
207 
208  // Returns a handle to the pending outputs for the aggregate window function.
209  int64_t aggregateStatePendingOutputs() const;
210 
211  const int64_t* partitionStartOffset() const;
212 
213  const int64_t* partitionNumCountBuf() const;
214 
215  const std::vector<const int8_t*>& getColumnBufferForWindowFunctionExpressions() const;
216 
217  const std::vector<const int8_t*>& getOrderKeyColumnBuffers() const;
218 
219  const std::vector<SQLTypeInfo>& getOrderKeyColumnBufferTypes() const;
220 
222 
224 
226 
228 
229  size_t* getAggregateTreeDepth() const;
230 
231  size_t getAggregateTreeFanout() const;
232 
233  int64_t* getNullValueStartPos() const;
234 
235  int64_t* getNullValueEndPos() const;
236 
237  // Returns a pointer to the partition start bitmap.
238  const int8_t* partitionStart() const;
239 
240  // Returns a pointer to the partition end bitmap.
241  const int8_t* partitionEnd() const;
242 
243  // Returns the element count in the columns used by the window function.
244  size_t elementCount() const;
245 
246  const int32_t* payload() const;
247 
248  const int32_t* offsets() const;
249 
250  const int32_t* counts() const;
251 
252  size_t partitionCount() const;
253 
254  const bool needsToBuildAggregateTree() const;
255 
256  private:
257  // State for a window aggregate. The count field is only used for average.
258  struct AggregateState {
259  int64_t val;
260  int64_t count;
261  std::vector<void*> outputs;
262  llvm::Value* row_number = nullptr;
263  };
264 
265  static Comparator makeComparator(const Analyzer::ColumnVar* col_var,
266  const int8_t* partition_values,
267  const int32_t* partition_indices,
268  const bool asc_ordering,
269  const bool nulls_first);
270 
271  void computePartitionBuffer(const size_t partition_idx,
272  int64_t* output_for_partition_buff,
273  const Analyzer::WindowFunction* window_func);
274 
275  void sortPartition(const size_t partition_idx,
276  int64_t* output_for_partition_buff,
277  bool should_parallelize);
278 
279  void computeNullRangeOfSortedPartition(const SQLTypeInfo& order_col_ti,
280  size_t partition_idx,
281  const int32_t* original_col_idx_buf,
282  const int64_t* ordered_col_idx_buf);
283 
285  size_t partition_idx,
286  size_t partition_size,
287  const int32_t* original_rowid_buf,
288  const int64_t* ordered_rowid_buf,
289  const SQLTypeInfo& input_col_ti);
290 
291  void fillPartitionStart();
292 
293  void fillPartitionEnd();
294 
295  void resizeStorageForWindowFraming(bool const for_reuse = false);
296 
298 
302  // Keeps ownership of order column.
303  std::vector<std::vector<std::shared_ptr<Chunk_NS::Chunk>>> order_columns_owner_;
304  // Order column buffers.
305  std::vector<const int8_t*> order_columns_;
306  std::vector<SQLTypeInfo> order_columns_ti_;
307  // Hash table which contains the partitions specified by the window.
308  std::shared_ptr<HashJoin> partitions_;
309  // The number of elements in the table.
310  size_t elem_count_;
311  // The output of the window function.
312  int8_t* output_;
313  std::shared_ptr<std::vector<int64_t>> sorted_partition_buf_;
314  // Keeps ownership of column referenced in window function expression.
315  std::vector<std::vector<std::shared_ptr<Chunk_NS::Chunk>>>
317  // Column buffers used for window function expression
318  std::vector<const int8_t*> window_func_expr_columns_;
319  // we need to build a segment tree depending on the input column type
320  std::vector<std::shared_ptr<void>> segment_trees_owned_;
327  // Markers for partition start used to reinitialize state for aggregate window
328  // functions.
330  // Markers for partition end used to reinitialize state for aggregate window
331  // functions.
332  int8_t* partition_end_;
333  // State for aggregate function over a window.
336  std::shared_ptr<RowSetMemoryOwner> row_set_mem_owner_;
337 
338  // For use when we do not have partitions_ hash table
339  const int32_t dummy_count_;
340  const int32_t dummy_offset_;
341  // dummy_payload_ is only initialized if there is no partitions_ hash table
342  // TODO(todd): There is no need for index buffer for non-partitioned
343  // window functions, as the row to index mapping is the identity function,
344  // so refactor makeComparator and ilk to allow for this
345  int32_t* dummy_payload_;
346 };
347 
348 // Keeps track of the multiple window functions in a window query.
350  public:
352  std::unique_ptr<WindowFunctionContext> window_function_context,
353  const size_t target_index);
354 
355  // Marks the window function at the given target index as active. This simplifies the
356  // code generation since it's now context sensitive. Each value window function can
357  // have its own iteration order, therefore fetching a column at a given position
358  // changes depending on which window function is active.
360  Executor* executor,
361  const size_t target_index) const;
362 
363  // Resets the active window function, which restores the regular (non-window) codegen
364  // behavior.
365  static void resetWindowFunctionContext(Executor* executor);
366 
367  // Gets the current active window function.
368  static WindowFunctionContext* getActiveWindowFunctionContext(Executor* executor);
369 
370  // Creates the context for a window function execution unit.
371  static WindowProjectNodeContext* create(Executor* executor);
372 
373  // Retrieves the context for the active window function execution unit.
374  static const WindowProjectNodeContext* get(Executor* executor);
375 
376  // Resets the active context.
377  static void reset(Executor* executor);
378 
379  private:
380  // A map from target index to the context associated with the window function at that
381  // target index.
382  std::unordered_map<size_t, std::unique_ptr<WindowFunctionContext>> window_contexts_;
383 };
384 
386 
size_t getAggregateTreeFanout() const
Defines data structures for the semantic analysis phase of query processing.
std::vector< SumAndCountPair< double > * > derived_aggregate_tree_for_double_type_
Definition: WindowContext.h:94
void addOrderColumn(const int8_t *column, const SQLTypeInfo &ti, const std::vector< std::shared_ptr< Chunk_NS::Chunk >> &chunks_owner)
llvm::Value * num_elem_current_partition_lv
std::shared_ptr< RowSetMemoryOwner > row_set_mem_owner_
int64_t * ordered_partition_null_start_pos_
llvm::Value * current_col_value_lv
const int32_t dummy_count_
std::vector< double * > aggregate_tree_for_double_type_
Definition: WindowContext.h:92
int64_t * getNullValueEndPos() const
bool window_function_conditional_aggregate(const SqlWindowFunctionKind kind)
Definition: WindowContext.h:78
llvm::Value * current_partition_start_offset_lv
const int32_t dummy_offset_
static Comparator makeComparator(const Analyzer::ColumnVar *col_var, const int8_t *partition_values, const int32_t *partition_indices, const bool asc_ordering, const bool nulls_first)
llvm::Value * current_row_pos_lv
llvm::Value * target_partition_rowid_ptr_lv
const int8_t * partitionStart() const
llvm::Value * frame_end_bound_expr_lv
llvm::Value * num_elem_current_partition_lv
llvm::Value * nulls_first_lv
const std::vector< SQLTypeInfo > & getOrderKeyColumnBufferTypes() const
void setSortedPartitionCacheKey(QueryPlanHash cache_key)
void computeNullRangeOfSortedPartition(const SQLTypeInfo &order_col_ti, size_t partition_idx, const int32_t *original_col_idx_buf, const int64_t *ordered_col_idx_buf)
static WindowProjectNodeContext * create(Executor *executor)
size_t elementCount() const
const int8_t * output() const
const Analyzer::WindowFunction * window_func_
const int32_t * counts() const
Constants for Builtin SQL Types supported by HEAVY.AI.
const int32_t * offsets() const
llvm::Value * target_partition_sorted_rowid_ptr_lv
llvm::Value * target_partition_rowid_ptr_lv
static WindowFunctionContext * getActiveWindowFunctionContext(Executor *executor)
ExecutorDeviceType
const bool needsToBuildAggregateTree() const
size_t * getAggregateTreeDepth() const
int64_t ** getAggregationTreesForIntegerTypeWindowExpr() const
const std::vector< const int8_t * > & getColumnBufferForWindowFunctionExpressions() const
int64_t * getNullValueStartPos() const
SumAndCountPair< double > ** getDerivedAggregationTreesForDoubleTypeWindowExpr() const
const int64_t * partitionStartOffset() const
std::vector< std::shared_ptr< void > > segment_trees_owned_
std::shared_ptr< std::vector< int64_t > > sorted_partition_buf_
size_t partitionCount() const
AggregateState aggregate_state_
llvm::Value * null_start_pos_lv
bool window_function_is_value(const SqlWindowFunctionKind kind)
Definition: WindowContext.h:29
size_t g_window_function_aggregation_tree_fanout
const int64_t * aggregateStateCount() const
std::vector< Comparator > createComparator(size_t partition_idx)
QueryPlanHash sorted_partition_cache_key_
const int8_t * partitionEnd() const
std::vector< std::vector< std::shared_ptr< Chunk_NS::Chunk > > > window_func_expr_columns_owner_
void addColumnBufferForWindowFunctionExpression(const int8_t *column, const std::vector< std::shared_ptr< Chunk_NS::Chunk >> &chunks_owner)
static void reset(Executor *executor)
const WindowFunctionContext * activateWindowFunctionContext(Executor *executor, const size_t target_index) const
QueryPlanHash partition_cache_key_
int64_t aggregateStatePendingOutputs() const
void buildAggregationTreeForPartition(SqlWindowFunctionKind agg_type, size_t partition_idx, size_t partition_size, const int32_t *original_rowid_buf, const int64_t *ordered_rowid_buf, const SQLTypeInfo &input_col_ti)
const int64_t * aggregateState() const
std::function< bool(const PermutationIdx, const PermutationIdx)> Comparator
Definition: ResultSet.h:155
WindowFunctionContext & operator=(const WindowFunctionContext &)=delete
SumAndCountPair< int64_t > ** getDerivedAggregationTreesForIntegerTypeWindowExpr() const
static const int NUM_EXECUTION_DEVICES
Executor(const ExecutorId id, Data_Namespace::DataMgr *data_mgr, const size_t block_size_x, const size_t grid_size_x, const size_t max_gpu_slab_size, const std::string &debug_dir, const std::string &debug_file)
Definition: Execute.cpp:276
llvm::Value * current_partition_start_offset_lv
bool window_function_is_aggregate(const SqlWindowFunctionKind kind)
Definition: WindowContext.h:61
void addWindowFunctionContext(std::unique_ptr< WindowFunctionContext > window_function_context, const size_t target_index)
llvm::Value * target_partition_sorted_rowid_ptr_lv
std::vector< const int8_t * > window_func_expr_columns_
AggregateTreeForWindowFraming aggregate_trees_
void sortPartition(const size_t partition_idx, int64_t *output_for_partition_buff, bool should_parallelize)
WindowFunctionContext(const Analyzer::WindowFunction *window_func, const size_t elem_count, const ExecutorDeviceType device_type, std::shared_ptr< RowSetMemoryOwner > row_set_mem_owner)
const std::vector< const int8_t * > & getOrderKeyColumnBuffers() const
std::shared_ptr< HashJoin > partitions_
size_t QueryPlanHash
llvm::Value * null_end_pos_lv
void resizeStorageForWindowFraming(size_t partition_count)
Definition: WindowContext.h:97
const int64_t * partitionNumCountBuf() const
bool window_function_is_value_with_frame(const SqlWindowFunctionKind kind)
Definition: WindowContext.h:47
bool window_function_requires_peer_handling(const Analyzer::WindowFunction *window_func)
SqlWindowFunctionKind
Definition: sqldefs.h:129
void resizeStorageForWindowFraming(bool const for_reuse=false)
std::unordered_map< size_t, std::unique_ptr< WindowFunctionContext > > window_contexts_
void computePartitionBuffer(const size_t partition_idx, int64_t *output_for_partition_buff, const Analyzer::WindowFunction *window_func)
llvm::Value * order_key_buf_ptr_lv
int64_t * partition_start_offset_
llvm::Value * int64_t_zero_val_lv
std::vector< const int8_t * > order_columns_
const int64_t * sortedPartition() const
const QueryPlanHash computeAggregateTreeCacheKey() const
static void resetWindowFunctionContext(Executor *executor)
std::vector< SQLTypeInfo > order_columns_ti_
std::vector< SumAndCountPair< int64_t > * > derived_aggregate_tree_for_integer_type_
Definition: WindowContext.h:93
const Analyzer::WindowFunction * getWindowFunction() const
const int32_t * payload() const
std::function< WindowFunctionContext::WindowComparatorResult(const int64_t lhs, const int64_t rhs)> Comparator
size_t * aggregate_trees_depth_
std::vector< std::vector< std::shared_ptr< Chunk_NS::Chunk > > > order_columns_owner_
llvm::Value * int64_t_one_val_lv
llvm::Value * frame_start_bound_expr_lv
std::vector< int64_t * > aggregate_tree_for_integer_type_
Definition: WindowContext.h:91
const ExecutorDeviceType device_type_
std::string order_type_col_name
void compute(std::unordered_map< QueryPlanHash, size_t > &sorted_partition_key_ref_count_map, std::unordered_map< QueryPlanHash, std::shared_ptr< std::vector< int64_t >>> &sorted_partition_cache, std::unordered_map< QueryPlanHash, AggregateTreeForWindowFraming > &aggregate_tree_map)
double ** getAggregationTreesForDoubleTypeWindowExpr() const
int64_t * ordered_partition_null_end_pos_