This is an automated email from the ASF dual-hosted git repository. yangzhg pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/master by this push: new 1cf57a985d [fix] Fix the query result error caused by the grouping sets statemen… (#11316) 1cf57a985d is described below commit 1cf57a985d4ab330565d8d90a0cb6dba011d84e2 Author: luozenglin <37725793+luozeng...@users.noreply.github.com> AuthorDate: Mon Aug 1 13:52:18 2022 +0800 [fix] Fix the query result error caused by the grouping sets statemen… (#11316) * [fix] Fix the query result error caused by the grouping sets statement grouping as an expression --- be/src/exec/repeat_node.cpp | 125 ++++++++++----------- be/src/exec/repeat_node.h | 10 +- be/src/vec/exec/vrepeat_node.cpp | 73 +++++------- be/src/vec/exec/vrepeat_node.h | 12 +- .../org/apache/doris/analysis/GroupingInfo.java | 118 ++++++++++++++++--- .../java/org/apache/doris/analysis/SelectStmt.java | 9 +- .../org/apache/doris/analysis/VirtualSlotRef.java | 10 ++ .../java/org/apache/doris/planner/RepeatNode.java | 94 +++++----------- .../org/apache/doris/planner/QueryPlanTest.java | 2 +- .../org/apache/doris/planner/RepeatNodeTest.java | 117 +++++++++---------- gensrc/thrift/PlanNodes.thrift | 1 + .../query/grouping_sets/test_grouping_sets.out | 37 ++++++ .../query/grouping_sets/test_grouping_sets.groovy | 22 ++++ 13 files changed, 363 insertions(+), 267 deletions(-) diff --git a/be/src/exec/repeat_node.cpp b/be/src/exec/repeat_node.cpp index 6f075cb00f..90ff8c9d43 100644 --- a/be/src/exec/repeat_node.cpp +++ b/be/src/exec/repeat_node.cpp @@ -34,7 +34,7 @@ RepeatNode::RepeatNode(ObjectPool* pool, const TPlanNode& tnode, const Descripto _repeat_id_list(tnode.repeat_node.repeat_id_list), _grouping_list(tnode.repeat_node.grouping_list), _output_tuple_id(tnode.repeat_node.output_tuple_id), - _tuple_desc(nullptr), + _output_tuple_desc(nullptr), _child_row_batch(nullptr), _child_eos(false), _repeat_id_idx(0), @@ -42,16 +42,30 @@ RepeatNode::RepeatNode(ObjectPool* pool, const TPlanNode& tnode, const Descripto RepeatNode::~RepeatNode() {} +Status RepeatNode::init(const TPlanNode& tnode, RuntimeState* state) { + RETURN_IF_ERROR(ExecNode::init(tnode, state)); + const RowDescriptor& row_desc = child(0)->row_desc(); + RETURN_IF_ERROR(Expr::create(tnode.repeat_node.exprs, row_desc, state, &_exprs)); + DCHECK(!_exprs.empty()); + return Status::OK(); +} + Status RepeatNode::prepare(RuntimeState* state) { SCOPED_TIMER(_runtime_profile->total_time_counter()); RETURN_IF_ERROR(ExecNode::prepare(state)); SCOPED_CONSUME_MEM_TRACKER(mem_tracker()); _runtime_state = state; - _tuple_desc = state->desc_tbl().get_tuple_descriptor(_output_tuple_id); - if (_tuple_desc == nullptr) { + _output_tuple_desc = state->desc_tbl().get_tuple_descriptor(_output_tuple_id); + if (_output_tuple_desc == nullptr) { return Status::InternalError("Failed to get tuple descriptor."); } + for (int i = 0; i < _exprs.size(); i++) { + ExprContext* context = _pool->add(new ExprContext(_exprs[i])); + RETURN_IF_ERROR(context->prepare(state, child(0)->row_desc())); + _expr_evals.push_back(context); + } + DCHECK_EQ(_exprs.size(), _expr_evals.size()); return Status::OK(); } @@ -59,6 +73,10 @@ Status RepeatNode::open(RuntimeState* state) { SCOPED_TIMER(_runtime_profile->total_time_counter()); RETURN_IF_ERROR(ExecNode::open(state)); SCOPED_CONSUME_MEM_TRACKER(mem_tracker()); + + for (int i = 0; i < _expr_evals.size(); i++) { + RETURN_IF_ERROR(_expr_evals[i]->open(state)); + } RETURN_IF_CANCELLED(state); RETURN_IF_ERROR(child(0)->open(state)); return Status::OK(); @@ -77,66 +95,14 @@ Status RepeatNode::get_repeated_batch(RowBatch* child_row_batch, int repeat_id_i // Fill all slots according to child MemPool* tuple_pool = row_batch->tuple_data_pool(); - const std::vector<TupleDescriptor*>& src_tuple_descs = - child_row_batch->row_desc().tuple_descriptors(); - const std::vector<TupleDescriptor*>& dst_tuple_descs = - row_batch->row_desc().tuple_descriptors(); - std::vector<Tuple*> dst_tuples(src_tuple_descs.size(), nullptr); - for (int i = 0; i < child_row_batch->num_rows(); ++i) { + Tuple* tuple = nullptr; + for (int row_index = 0; row_index < child_row_batch->num_rows(); ++row_index) { int row_idx = row_batch->add_row(); TupleRow* dst_row = row_batch->get_row(row_idx); - TupleRow* src_row = child_row_batch->get_row(i); - - auto src_it = src_tuple_descs.begin(); - auto dst_it = dst_tuple_descs.begin(); - for (int j = 0; src_it != src_tuple_descs.end() && dst_it != dst_tuple_descs.end(); - ++src_it, ++dst_it, ++j) { - Tuple* src_tuple = src_row->get_tuple(j); - if (src_tuple == nullptr) { - dst_row->set_tuple(j, nullptr); - continue; - } - - if (dst_tuples[j] == nullptr) { - int size = row_batch->capacity() * (*dst_it)->byte_size(); - void* tuple_buffer = tuple_pool->allocate(size); - if (tuple_buffer == nullptr) { - return Status::InternalError("Allocate memory for row batch failed."); - } - dst_tuples[j] = reinterpret_cast<Tuple*>(tuple_buffer); - } else { - char* new_tuple = reinterpret_cast<char*>(dst_tuples[j]); - new_tuple += (*dst_it)->byte_size(); - dst_tuples[j] = reinterpret_cast<Tuple*>(new_tuple); - } - dst_row->set_tuple(j, dst_tuples[j]); - memset(dst_tuples[j], 0, (*dst_it)->num_null_bytes()); - src_tuple->deep_copy(dst_tuples[j], **dst_it, tuple_pool); - for (int k = 0; k < (*src_it)->slots().size(); k++) { - SlotDescriptor* src_slot_desc = (*src_it)->slots()[k]; - SlotDescriptor* dst_slot_desc = (*dst_it)->slots()[k]; - DCHECK_EQ(src_slot_desc->type().type, dst_slot_desc->type().type); - DCHECK_EQ(src_slot_desc->col_name(), dst_slot_desc->col_name()); - // set null base on repeated list - if (_all_slot_ids.find(src_slot_desc->id()) != _all_slot_ids.end()) { - std::set<SlotId>& repeat_ids = _slot_id_set_list[repeat_id_idx]; - if (repeat_ids.find(src_slot_desc->id()) == repeat_ids.end()) { - dst_tuples[j]->set_null(dst_slot_desc->null_indicator_offset()); - continue; - } - } - } - } - row_batch->commit_last_row(); - } - Tuple* tuple = nullptr; - // Fill grouping ID to tuple - for (int i = 0; i < child_row_batch->num_rows(); ++i) { - int row_idx = i; - TupleRow* row = row_batch->get_row(row_idx); + TupleRow* src_row = child_row_batch->get_row(row_index); - if (tuple == nullptr) { - int size = row_batch->capacity() * _tuple_desc->byte_size(); + if (UNLIKELY(tuple == nullptr)) { + int size = row_batch->capacity() * _output_tuple_desc->byte_size(); void* tuple_buffer = tuple_pool->allocate(size); if (tuple_buffer == nullptr) { return Status::InternalError("Allocate memory for row batch failed."); @@ -144,21 +110,38 @@ Status RepeatNode::get_repeated_batch(RowBatch* child_row_batch, int repeat_id_i tuple = reinterpret_cast<Tuple*>(tuple_buffer); } else { char* new_tuple = reinterpret_cast<char*>(tuple); - new_tuple += _tuple_desc->byte_size(); + new_tuple += _output_tuple_desc->byte_size(); tuple = reinterpret_cast<Tuple*>(new_tuple); } + dst_row->set_tuple(0, tuple); + memset(tuple, 0, _output_tuple_desc->num_null_bytes()); + + int slot_index = 0; + for (; slot_index < _expr_evals.size(); ++slot_index) { + const SlotDescriptor* slot_desc = _output_tuple_desc->slots()[slot_index]; + // set null base on repeated list + if (_all_slot_ids.find(slot_desc->id()) != _all_slot_ids.end()) { + std::set<SlotId>& repeat_ids = _slot_id_set_list[repeat_id_idx]; + if (repeat_ids.find(slot_desc->id()) == repeat_ids.end()) { + tuple->set_null(slot_desc->null_indicator_offset()); + continue; + } + } - row->set_tuple(src_tuple_descs.size(), tuple); - memset(tuple, 0, _tuple_desc->num_null_bytes()); + void* val = _expr_evals[slot_index]->get_value(src_row); + tuple->set_not_null(slot_desc->null_indicator_offset()); + RawValue::write(val, tuple, slot_desc, tuple_pool); + } - for (size_t slot_idx = 0; slot_idx < _grouping_list.size(); ++slot_idx) { - int64_t val = _grouping_list[slot_idx][repeat_id_idx]; - DCHECK_LT(slot_idx, _tuple_desc->slots().size()) - << "TupleDescriptor: " << _tuple_desc->debug_string(); - const SlotDescriptor* slot_desc = _tuple_desc->slots()[slot_idx]; + DCHECK_EQ(slot_index + _grouping_list.size(), _output_tuple_desc->slots().size()); + for (int i = 0; slot_index < _output_tuple_desc->slots().size(); ++i, ++slot_index) { + const SlotDescriptor* slot_desc = _output_tuple_desc->slots()[slot_index]; tuple->set_not_null(slot_desc->null_indicator_offset()); + + int64_t val = _grouping_list[i][repeat_id_idx]; RawValue::write(&val, tuple, slot_desc, tuple_pool); } + row_batch->commit_last_row(); } return Status::OK(); @@ -207,6 +190,11 @@ Status RepeatNode::close(RuntimeState* state) { return Status::OK(); } _child_row_batch.reset(nullptr); + for (int i = 0; i < _expr_evals.size(); i++) { + _expr_evals[i]->close(state); + } + _expr_evals.clear(); + Expr::close(_exprs); RETURN_IF_ERROR(child(0)->close(state)); return ExecNode::close(state); } @@ -216,6 +204,7 @@ void RepeatNode::debug_string(int indentation_level, std::stringstream* out) con *out << "RepeatNode("; *out << "repeat pattern: [" << JoinElements(_repeat_id_list, ",") << "]\n"; *out << "add " << _grouping_list.size() << " columns. \n"; + *out << "_exprs: " << Expr::debug_string(_exprs); *out << "added column values: "; for (const std::vector<int64_t>& v : _grouping_list) { *out << "[" << JoinElements(v, ",") << "] "; diff --git a/be/src/exec/repeat_node.h b/be/src/exec/repeat_node.h index d9dce75278..9c43a33c86 100644 --- a/be/src/exec/repeat_node.h +++ b/be/src/exec/repeat_node.h @@ -18,6 +18,8 @@ #pragma once #include "exec/exec_node.h" +#include "exprs/expr.h" +#include "exprs/expr_context.h" namespace doris { @@ -32,6 +34,7 @@ public: RepeatNode(ObjectPool* pool, const TPlanNode& tnode, const DescriptorTbl& descs); ~RepeatNode(); + virtual Status init(const TPlanNode& tnode, RuntimeState* state = nullptr) override; virtual Status prepare(RuntimeState* state) override; virtual Status open(RuntimeState* state) override; virtual Status get_next(RuntimeState* state, RowBatch* row_batch, bool* eos) override; @@ -52,12 +55,17 @@ protected: std::vector<std::vector<int64_t>> _grouping_list; // Tuple id used for output, it has new slots. TupleId _output_tuple_id; - const TupleDescriptor* _tuple_desc; + const TupleDescriptor* _output_tuple_desc; std::unique_ptr<RowBatch> _child_row_batch; bool _child_eos; int _repeat_id_idx; RuntimeState* _runtime_state; + + // Exprs used to evaluate input rows + std::vector<Expr*> _exprs; + + std::vector<ExprContext*> _expr_evals; }; } // namespace doris diff --git a/be/src/vec/exec/vrepeat_node.cpp b/be/src/vec/exec/vrepeat_node.cpp index 4ccbd60750..3c180d417d 100644 --- a/be/src/vec/exec/vrepeat_node.cpp +++ b/be/src/vec/exec/vrepeat_node.cpp @@ -17,52 +17,30 @@ #include "vec/exec/vrepeat_node.h" -#include "exprs/expr.h" #include "gutil/strings/join.h" #include "runtime/runtime_state.h" #include "util/runtime_profile.h" +#include "vec/exprs/vexpr.h" namespace doris::vectorized { VRepeatNode::VRepeatNode(ObjectPool* pool, const TPlanNode& tnode, const DescriptorTbl& descs) - : RepeatNode(pool, tnode, descs), - _child_block(nullptr), - _virtual_tuple_id(tnode.repeat_node.output_tuple_id) {} + : RepeatNode(pool, tnode, descs) {} + +Status VRepeatNode::init(const TPlanNode& tnode, RuntimeState* state) { + RETURN_IF_ERROR(ExecNode::init(tnode, state)); + RETURN_IF_ERROR(VExpr::create_expr_trees(_pool, tnode.repeat_node.exprs, &_expr_ctxs)); + return Status::OK(); +} Status VRepeatNode::prepare(RuntimeState* state) { VLOG_CRITICAL << "VRepeatNode::prepare"; SCOPED_TIMER(_runtime_profile->total_time_counter()); RETURN_IF_ERROR(RepeatNode::prepare(state)); + RETURN_IF_ERROR(VExpr::prepare(_expr_ctxs, state, child(0)->row_desc())); - // get current all output slots - for (const auto& tuple_desc : this->row_desc().tuple_descriptors()) { - for (const auto& slot_desc : tuple_desc->slots()) { - _output_slots.push_back(slot_desc); - } - } - - // get all input slots - for (const auto& child_tuple_desc : child(0)->row_desc().tuple_descriptors()) { - for (const auto& child_slot_desc : child_tuple_desc->slots()) { - _child_slots.push_back(child_slot_desc); - } + for (const auto& slot_desc : _output_tuple_desc->slots()) { + _output_slots.push_back(slot_desc); } - - _virtual_tuple_desc = state->desc_tbl().get_tuple_descriptor(_virtual_tuple_id); - if (_virtual_tuple_desc == NULL) { - return Status::InternalError("Failed to get virtual tuple descriptor."); - } - - std::stringstream ss; - ss << "The output slots size " << _output_slots.size() - << " is not equal to the sum of child_slots_size " << _child_slots.size() - << ",virtual_slots_size " << _virtual_tuple_desc->slots().size(); - if (_output_slots.size() != (_child_slots.size() + _virtual_tuple_desc->slots().size())) { - return Status::InternalError( - "The output slots size {} is not equal to the sum of child_slots_size {}" - ",virtual_slots_size {}", - _output_slots.size(), _child_slots.size(), _virtual_tuple_desc->slots().size()); - } - _child_block.reset(new Block()); return Status::OK(); @@ -73,6 +51,7 @@ Status VRepeatNode::open(RuntimeState* state) { VLOG_CRITICAL << "VRepeatNode::open"; SCOPED_TIMER(_runtime_profile->total_time_counter()); RETURN_IF_ERROR(RepeatNode::open(state)); + RETURN_IF_ERROR(VExpr::open(_expr_ctxs, state)); return Status::OK(); } @@ -84,7 +63,6 @@ Status VRepeatNode::get_repeated_block(Block* child_block, int repeat_id_idx, Bl size_t child_column_size = child_block->columns(); size_t column_size = _output_slots.size(); bool mem_reuse = output_block->mem_reuse(); - DCHECK_EQ(child_column_size, _child_slots.size()); DCHECK_LT(child_column_size, column_size); std::vector<vectorized::MutableColumnPtr> columns(column_size); for (size_t i = 0; i < column_size; i++) { @@ -105,9 +83,6 @@ Status VRepeatNode::get_repeated_block(Block* child_block, int repeat_id_idx, Bl for (size_t i = 0; i < child_column_size; i++) { const ColumnWithTypeAndName& src_column = child_block->get_by_position(i); - DCHECK_EQ(_child_slots[i]->type().type, _output_slots[cur_col]->type().type); - DCHECK_EQ(_child_slots[i]->col_name(), _output_slots[cur_col]->col_name()); - std::set<SlotId>& repeat_ids = _slot_id_set_list[repeat_id_idx]; bool is_repeat_slot = _all_slot_ids.find(_output_slots[cur_col]->id()) != _all_slot_ids.end(); @@ -141,8 +116,8 @@ Status VRepeatNode::get_repeated_block(Block* child_block, int repeat_id_idx, Bl // Fill grouping ID to block for (auto slot_idx = 0; slot_idx < _grouping_list.size(); slot_idx++) { - DCHECK_LT(slot_idx, _virtual_tuple_desc->slots().size()); - const SlotDescriptor* _virtual_slot_desc = _virtual_tuple_desc->slots()[slot_idx]; + DCHECK_LT(slot_idx, _output_tuple_desc->slots().size()); + const SlotDescriptor* _virtual_slot_desc = _output_tuple_desc->slots()[cur_col]; DCHECK_EQ(_virtual_slot_desc->type().type, _output_slots[cur_col]->type().type); DCHECK_EQ(_virtual_slot_desc->col_name(), _output_slots[cur_col]->col_name()); int64_t val = _grouping_list[slot_idx][repeat_id_idx]; @@ -200,15 +175,29 @@ Status VRepeatNode::get_next(RuntimeState* state, Block* block, bool* eos) { *eos = true; return Status::OK(); } + + DCHECK(!_expr_ctxs.empty()); + _intermediate_block.reset(new Block()); + for (auto vexpr_ctx : _expr_ctxs) { + int result_column_id = -1; + RETURN_IF_ERROR(vexpr_ctx->execute(_child_block.get(), &result_column_id)); + DCHECK(result_column_id != -1); + _child_block->get_by_position(result_column_id).column = + _child_block->get_by_position(result_column_id) + .column->convert_to_full_column_if_const(); + _intermediate_block->insert(_child_block->get_by_position(result_column_id)); + } + DCHECK_EQ(_expr_ctxs.size(), _intermediate_block->columns()); } - RETURN_IF_ERROR(get_repeated_block(_child_block.get(), _repeat_id_idx, block)); + RETURN_IF_ERROR(get_repeated_block(_intermediate_block.get(), _repeat_id_idx, block)); _repeat_id_idx++; int size = _repeat_id_list.size(); if (_repeat_id_idx >= size) { - release_block_memory(*_child_block.get()); + _intermediate_block->clear(); + release_block_memory(*_child_block); _repeat_id_idx = 0; } @@ -224,7 +213,7 @@ Status VRepeatNode::close(RuntimeState* state) { return Status::OK(); } START_AND_SCOPE_SPAN(state->get_tracer(), span, "VRepeatNode::close"); - release_block_memory(*_child_block.get()); + VExpr::close(_expr_ctxs, state); RETURN_IF_ERROR(child(0)->close(state)); return ExecNode::close(state); } diff --git a/be/src/vec/exec/vrepeat_node.h b/be/src/vec/exec/vrepeat_node.h index 26efa9a1fe..cc857dc33a 100644 --- a/be/src/vec/exec/vrepeat_node.h +++ b/be/src/vec/exec/vrepeat_node.h @@ -28,11 +28,14 @@ class RuntimeState; class Status; namespace vectorized { +class VExprContext; + class VRepeatNode : public RepeatNode { public: VRepeatNode(ObjectPool* pool, const TPlanNode& tnode, const DescriptorTbl& descs); ~VRepeatNode() override = default; + virtual Status init(const TPlanNode& tnode, RuntimeState* state = nullptr) override; virtual Status prepare(RuntimeState* state) override; virtual Status open(RuntimeState* state) override; virtual Status get_next(RuntimeState* state, Block* block, bool* eos) override; @@ -45,13 +48,12 @@ private: using RepeatNode::get_next; Status get_repeated_block(Block* child_block, int repeat_id_idx, Block* output_block); - std::unique_ptr<Block> _child_block; - std::vector<SlotDescriptor*> _child_slots; + std::unique_ptr<Block> _child_block {}; + std::unique_ptr<Block> _intermediate_block {}; + std::vector<SlotDescriptor*> _output_slots; - // _virtual_tuple_id id used for GROUPING_ID(). - TupleId _virtual_tuple_id; - const TupleDescriptor* _virtual_tuple_desc; + std::vector<VExprContext*> _expr_ctxs; }; } // namespace vectorized } // namespace doris diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/GroupingInfo.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/GroupingInfo.java index 3387d17dcb..8921cd07e6 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/analysis/GroupingInfo.java +++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/GroupingInfo.java @@ -22,6 +22,7 @@ import org.apache.doris.catalog.Type; import org.apache.doris.common.AnalysisException; import com.google.common.base.Preconditions; +import com.google.common.collect.Lists; import java.util.ArrayList; import java.util.BitSet; @@ -36,47 +37,76 @@ public class GroupingInfo { public static final String GROUPING_PREFIX = "GROUPING_PREFIX_"; private VirtualSlotRef groupingIDSlot; private TupleDescriptor virtualTuple; - private Set<VirtualSlotRef> groupingSlots; + private TupleDescriptor outputTupleDesc; + private ExprSubstitutionMap outputTupleSmap; + private List<SlotDescriptor> groupingSlotDescList; + private Set<VirtualSlotRef> virtualSlotRefs; private List<BitSet> groupingIdList; private GroupByClause.GroupingType groupingType; private BitSet bitSetAll; + private List<Expr> preRepeatExprs; + public GroupingInfo(Analyzer analyzer, GroupByClause groupByClause) throws AnalysisException { this.groupingType = groupByClause.getGroupingType(); - groupingSlots = new LinkedHashSet<>(); + virtualSlotRefs = new LinkedHashSet<>(); virtualTuple = analyzer.getDescTbl().createTupleDescriptor("VIRTUAL_TUPLE"); groupingIDSlot = new VirtualSlotRef(COL_GROUPING_ID, Type.BIGINT, virtualTuple, new ArrayList<>()); groupingIDSlot.analyze(analyzer); - groupingSlots.add(groupingIDSlot); + virtualSlotRefs.add(groupingIDSlot); + + outputTupleDesc = analyzer.getDescTbl().createTupleDescriptor("repeat-tuple"); + outputTupleSmap = new ExprSubstitutionMap(); + groupingSlotDescList = Lists.newArrayList(); + preRepeatExprs = Lists.newArrayList(); } - public Set<VirtualSlotRef> getGroupingSlots() { - return groupingSlots; + public Set<VirtualSlotRef> getVirtualSlotRefs() { + return virtualSlotRefs; } public TupleDescriptor getVirtualTuple() { return virtualTuple; } + public TupleDescriptor getOutputTupleDesc() { + return outputTupleDesc; + } + + public ExprSubstitutionMap getOutputTupleSmap() { + return outputTupleSmap; + } + + public List<SlotDescriptor> getGroupingSlotDescList() { + return groupingSlotDescList; + } + public List<BitSet> getGroupingIdList() { return groupingIdList; } + public List<Expr> getPreRepeatExprs() { + return preRepeatExprs; + } + + public void substitutePreRepeatExprs(ExprSubstitutionMap smap, Analyzer analyzer) { + preRepeatExprs = Expr.substituteList(preRepeatExprs, smap, analyzer, true); + } + // generate virtual slots for grouping or grouping_id functions public VirtualSlotRef addGroupingSlots(List<Expr> realSlots, Analyzer analyzer) throws AnalysisException { - String colName = realSlots.stream().map(expr -> expr.toSql()).collect(Collectors.joining( - "_")); + String colName = realSlots.stream().map(expr -> expr.toSql()).collect(Collectors.joining("_")); colName = GROUPING_PREFIX + colName; VirtualSlotRef virtualSlot = new VirtualSlotRef(colName, Type.BIGINT, virtualTuple, realSlots); virtualSlot.analyze(analyzer); - if (groupingSlots.contains(virtualSlot)) { - for (VirtualSlotRef vs : groupingSlots) { + if (virtualSlotRefs.contains(virtualSlot)) { + for (VirtualSlotRef vs : virtualSlotRefs) { if (vs.equals(virtualSlot)) { return vs; } } } - groupingSlots.add(virtualSlot); + virtualSlotRefs.add(virtualSlot); return virtualSlot; } @@ -124,13 +154,13 @@ public class GroupingInfo { default: Preconditions.checkState(false); } - groupingExprs.addAll(groupingSlots); + groupingExprs.addAll(virtualSlotRefs); } // generate grouping function's value public List<List<Long>> genGroupingList(ArrayList<Expr> groupingExprs) throws AnalysisException { List<List<Long>> groupingList = new ArrayList<>(); - for (SlotRef slot : groupingSlots) { + for (SlotRef slot : virtualSlotRefs) { List<Long> glist = new ArrayList<>(); for (BitSet bitSet : groupingIdList) { long l = 0L; @@ -150,7 +180,7 @@ public class GroupingInfo { int slotSize = ((VirtualSlotRef) slot).getRealSlots().size(); for (int i = 0; i < slotSize; ++i) { int j = groupingExprs.indexOf(((VirtualSlotRef) slot).getRealSlots().get(i)); - if (j < 0 || j >= bitSet.size()) { + if (j < 0 || j >= bitSet.size()) { throw new AnalysisException("Column " + ((VirtualSlotRef) slot).getRealColumnName() + " in GROUP_ID() does not exist in GROUP BY clause."); } @@ -164,6 +194,68 @@ public class GroupingInfo { return groupingList; } + public void genOutputTupleDescAndSMap(Analyzer analyzer, ArrayList<Expr> groupingAndVirtualSlotExprs, + List<FunctionCallExpr> aggExprs) { + List<Expr> groupingExprs = Lists.newArrayList(); + List<Expr> virtualSlotExprs = Lists.newArrayList(); + for (Expr expr : groupingAndVirtualSlotExprs) { + if (expr instanceof VirtualSlotRef) { + virtualSlotExprs.add(expr); + } else { + groupingExprs.add(expr); + } + } + for (Expr expr : groupingExprs) { + SlotDescriptor slotDesc = addSlot(analyzer, expr); + slotDesc.setIsNullable(true); + groupingSlotDescList.add(slotDesc); + preRepeatExprs.add(expr); + // register equivalence between grouping slot and grouping expr; + // do this only when the grouping expr isn't a constant, otherwise + // it'll simply show up as a gratuitous HAVING predicate + // (which would actually be incorrect if the constant happens to be NULL) + if (!expr.isConstant()) { + analyzer.createAuxEquivPredicate(new SlotRef(slotDesc), expr.clone()); + } + } + List<SlotRef> aggSlot = Lists.newArrayList(); + aggExprs.forEach(expr -> aggSlot.addAll(getSlotRefChildren(expr))); + for (SlotRef slotRef : aggSlot) { + addSlot(analyzer, slotRef); + preRepeatExprs.add(slotRef); + } + for (Expr expr : virtualSlotExprs) { + addSlot(analyzer, expr); + } + } + + private SlotDescriptor addSlot(Analyzer analyzer, Expr expr) { + SlotDescriptor slotDesc = analyzer.addSlotDescriptor(outputTupleDesc); + slotDesc.initFromExpr(expr); + slotDesc.setIsMaterialized(true); + if (expr instanceof SlotRef) { + slotDesc.setColumn(((SlotRef) expr).getColumn()); + } + if (expr instanceof VirtualSlotRef) { + outputTupleSmap.put(expr.clone(), new VirtualSlotRef(slotDesc)); + } else { + outputTupleSmap.put(expr.clone(), new SlotRef(slotDesc)); + } + return slotDesc; + } + + private List<SlotRef> getSlotRefChildren(Expr root) { + List<SlotRef> result = new ArrayList<>(); + for (Expr child : root.getChildren()) { + if (child instanceof SlotRef) { + result.add((SlotRef) child); + } else { + result.addAll(getSlotRefChildren(child)); + } + } + return result; + } + public void substituteGroupingFn(List<Expr> exprs, Analyzer analyzer) throws AnalysisException { if (groupingType == GroupByClause.GroupingType.GROUP_BY) { throw new AnalysisException("cannot use GROUPING functions without [grouping sets|rollup|cube] a" diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java index 4b930acd0c..b087ff451b 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java +++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java @@ -1047,15 +1047,16 @@ public class SelectStmt extends QueryStmt { List<TupleId> groupingByTupleIds = new ArrayList<>(); if (groupByClause != null) { - // must do it before copying for createAggInfo() - if (groupingInfo != null) { - groupingByTupleIds.add(groupingInfo.getVirtualTuple().getId()); - } groupByClause.genGroupingExprs(); if (groupingInfo != null) { groupingInfo.buildRepeat(groupByClause.getGroupingExprs(), groupByClause.getGroupingSetList()); } substituteOrdinalsAliases(groupByClause.getGroupingExprs(), "GROUP BY", analyzer); + if (groupingInfo != null) { + groupingInfo.genOutputTupleDescAndSMap(analyzer, groupByClause.getGroupingExprs(), aggExprs); + // must do it before copying for createAggInfo() + groupingByTupleIds.add(groupingInfo.getOutputTupleDesc().getId()); + } groupByClause.analyze(analyzer); createAggInfo(groupByClause.getGroupingExprs(), aggExprs, analyzer); } else { diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/VirtualSlotRef.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/VirtualSlotRef.java index 9696babe05..4cfd7a62cc 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/analysis/VirtualSlotRef.java +++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/VirtualSlotRef.java @@ -29,6 +29,8 @@ import java.io.DataOutput; import java.io.IOException; import java.util.ArrayList; import java.util.List; +import java.util.Map; +import java.util.Set; /** * It like a SlotRef except that it is not a real column exist in table. @@ -55,6 +57,10 @@ public class VirtualSlotRef extends SlotRef { tupleDescriptor = other.tupleDescriptor; } + public VirtualSlotRef(SlotDescriptor desc) { + super(desc); + } + public static VirtualSlotRef read(DataInput in) throws IOException { VirtualSlotRef virtualSlotRef = new VirtualSlotRef(null, Type.BIGINT, null, new ArrayList<>()); virtualSlotRef.readFields(in); @@ -68,6 +74,10 @@ public class VirtualSlotRef extends SlotRef { return getColumnName(); } + @Override + public void getTableIdToColumnNames(Map<Long, Set<String>> tableIdToColumnNames) { + } + @Override public void write(DataOutput out) throws IOException { super.write(out); diff --git a/fe/fe-core/src/main/java/org/apache/doris/planner/RepeatNode.java b/fe/fe-core/src/main/java/org/apache/doris/planner/RepeatNode.java index 47dbe46f32..5fef371ec8 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/planner/RepeatNode.java +++ b/fe/fe-core/src/main/java/org/apache/doris/planner/RepeatNode.java @@ -19,9 +19,8 @@ package org.apache.doris.planner; import org.apache.doris.analysis.Analyzer; import org.apache.doris.analysis.Expr; -import org.apache.doris.analysis.FunctionCallExpr; +import org.apache.doris.analysis.ExprSubstitutionMap; import org.apache.doris.analysis.GroupByClause; -import org.apache.doris.analysis.GroupingFunctionCallExpr; import org.apache.doris.analysis.GroupingInfo; import org.apache.doris.analysis.SlotDescriptor; import org.apache.doris.analysis.SlotId; @@ -68,17 +67,16 @@ public class RepeatNode extends PlanNode { private GroupByClause groupByClause; protected RepeatNode(PlanNodeId id, PlanNode input, GroupingInfo groupingInfo, GroupByClause groupByClause) { - super(id, input.getTupleIds(), "REPEAT_NODE", StatisticalType.REPEAT_NODE); + super(id, groupingInfo.getOutputTupleDesc().getId().asList(), "REPEAT_NODE", StatisticalType.REPEAT_NODE); this.children.add(input); this.groupingInfo = groupingInfo; this.input = input; this.groupByClause = groupByClause; - } // only for unittest protected RepeatNode(PlanNodeId id, PlanNode input, List<Set<SlotId>> repeatSlotIdList, - TupleDescriptor outputTupleDesc, List<List<Long>> groupingList) { + TupleDescriptor outputTupleDesc, List<List<Long>> groupingList) { super(id, input.getTupleIds(), "REPEAT_NODE", StatisticalType.REPEAT_NODE); this.children.add(input); this.repeatSlotIdList = buildIdSetList(repeatSlotIdList); @@ -116,25 +114,18 @@ public class RepeatNode extends PlanNode { @Override public void init(Analyzer analyzer) throws UserException { Preconditions.checkState(conjuncts.isEmpty()); - groupByClause.substituteGroupingExprs(groupingInfo.getGroupingSlots(), input.getOutputSmap(), - analyzer); - - for (Expr expr : groupByClause.getGroupingExprs()) { - if (expr instanceof SlotRef || (expr instanceof GroupingFunctionCallExpr)) { - continue; - } - // throw new AnalysisException("function or expr is not allowed in grouping sets clause."); + ExprSubstitutionMap childSmap = getCombinedChildSmap(); + groupByClause.substituteGroupingExprs(groupingInfo.getVirtualSlotRefs(), childSmap, analyzer); + groupingInfo.substitutePreRepeatExprs(childSmap, analyzer); + outputSmap = groupingInfo.getOutputTupleSmap(); + conjuncts = Expr.substituteList(conjuncts, outputSmap, analyzer, false); + outputTupleDesc = groupingInfo.getOutputTupleDesc(); + List<TupleId> inputTupleIds = input.getOutputTupleIds(); + if (inputTupleIds.size() == 1) { + // used for MaterializedViewSelector getTableIdToColumnNames + outputTupleDesc.setTable(analyzer.getTupleDesc(inputTupleIds.get(0)).getTable()); } - // build new BitSet List for tupleDesc - Set<SlotDescriptor> slotDescSet = new HashSet<>(); - for (TupleId tupleId : input.getTupleIds()) { - TupleDescriptor tupleDescriptor = analyzer.getDescTbl().getTupleDesc(tupleId); - slotDescSet.addAll(tupleDescriptor.getSlots()); - } - - // build tupleDesc according to child's tupleDesc info - outputTupleDesc = groupingInfo.getVirtualTuple(); //set aggregate nullable for (Expr slot : groupByClause.getGroupingExprs()) { if (slot instanceof SlotRef && !(slot instanceof VirtualSlotRef)) { @@ -144,74 +135,42 @@ public class RepeatNode extends PlanNode { outputTupleDesc.computeStatAndMemLayout(); List<Set<SlotId>> groupingIdList = new ArrayList<>(); - List<Expr> exprList = groupByClause.getGroupingExprs(); - Preconditions.checkState(exprList.size() >= 2); - allSlotId = new HashSet<>(); + List<SlotDescriptor> groupingSlotDescList = groupingInfo.getGroupingSlotDescList(); for (BitSet bitSet : Collections.unmodifiableList(groupingInfo.getGroupingIdList())) { Set<SlotId> slotIdSet = new HashSet<>(); - for (SlotDescriptor slotDesc : slotDescSet) { - SlotId slotId = slotDesc.getId(); - if (slotId == null) { - continue; - } - for (int i = 0; i < exprList.size(); i++) { - if (exprList.get(i) instanceof SlotRef) { - SlotRef slotRef = (SlotRef) (exprList.get(i)); - if (bitSet.get(i) && slotRef.getSlotId() == slotId) { - slotIdSet.add(slotId); - break; - } - } else if (exprList.get(i) instanceof FunctionCallExpr) { - List<SlotRef> slotRefs = getSlotRefChildren(exprList.get(i)); - for (SlotRef slotRef : slotRefs) { - if (bitSet.get(i) && slotRef.getSlotId() == slotId) { - slotIdSet.add(slotId); - break; - } - } - } + for (int i = 0; i < groupingSlotDescList.size(); i++) { + if (bitSet.get(i)) { + slotIdSet.add(groupingSlotDescList.get(i).getId()); } } groupingIdList.add(slotIdSet); } this.repeatSlotIdList = buildIdSetList(groupingIdList); + allSlotId = new HashSet<>(); for (Set<Integer> s : this.repeatSlotIdList) { allSlotId.addAll(s); } this.groupingList = groupingInfo.genGroupingList(groupByClause.getGroupingExprs()); - tupleIds.add(outputTupleDesc.getId()); for (TupleId id : tupleIds) { analyzer.getTupleDesc(id).setIsMaterialized(true); } computeTupleStatAndMemLayout(analyzer); computeStats(analyzer); - createDefaultSmap(analyzer); - } - - private List<SlotRef> getSlotRefChildren(Expr root) { - List<SlotRef> result = new ArrayList<>(); - for (Expr child : root.getChildren()) { - if (child instanceof SlotRef) { - result.add((SlotRef) child); - } else { - result.addAll(getSlotRefChildren(child)); - } - } - return result; } @Override protected void toThrift(TPlanNode msg) { msg.node_type = TPlanNodeType.REPEAT_NODE; - msg.repeat_node = new TRepeatNode(outputTupleDesc.getId().asInt(), repeatSlotIdList, groupingList.get(0), - groupingList, allSlotId); + msg.repeat_node = + new TRepeatNode(outputTupleDesc.getId().asInt(), repeatSlotIdList, groupingList.get(0), groupingList, + allSlotId, Expr.treesToThrift(groupingInfo.getPreRepeatExprs())); } @Override protected String debugString() { - return MoreObjects.toStringHelper(this).add("Repeat", repeatSlotIdList.size()).addValue( - super.debugString()).toString(); + return MoreObjects.toStringHelper(this).add("Repeat", repeatSlotIdList.size()).addValue(super.debugString()) + .toString(); } @Override @@ -223,11 +182,12 @@ public class RepeatNode extends PlanNode { output.append(detailPrefix + "repeat: repeat "); output.append(repeatSlotIdList.size() - 1); output.append(" lines "); - output.append(repeatSlotIdList); + output.append(repeatSlotIdList).append("\n"); + output.append(detailPrefix).append("exprs: ").append(getExplainString(groupingInfo.getPreRepeatExprs())); output.append("\n"); if (CollectionUtils.isNotEmpty(outputTupleDesc.getSlots())) { - output.append(detailPrefix + "generate: "); - output.append(outputTupleDesc.getSlots().stream().map(slot -> "`" + slot.getColumn().getName() + "`") + output.append(detailPrefix + "output slots: "); + output.append(outputTupleDesc.getSlots().stream().map(slot -> "`" + slot.getLabel() + "`") .collect(Collectors.joining(", ")) + "\n"); } return output.toString(); diff --git a/fe/fe-core/src/test/java/org/apache/doris/planner/QueryPlanTest.java b/fe/fe-core/src/test/java/org/apache/doris/planner/QueryPlanTest.java index 8ed644ebac..1d13186779 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/planner/QueryPlanTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/planner/QueryPlanTest.java @@ -421,7 +421,7 @@ public class QueryPlanTest extends TestWithFeService { public void testFunctionViewGroupingSet() throws Exception { String queryStr = "select query_id, client_ip, concat from test.function_view group by rollup(" + "query_id, client_ip, concat);"; - assertSQLPlanOrErrorMsgContains(queryStr, "repeat: repeat 3 lines [[], [0], [0, 1], [0, 1, 2, 3]]"); + assertSQLPlanOrErrorMsgContains(queryStr, "repeat: repeat 3 lines [[], [8], [8, 9], [8, 9, 10]]"); } @Test diff --git a/fe/fe-core/src/test/java/org/apache/doris/planner/RepeatNodeTest.java b/fe/fe-core/src/test/java/org/apache/doris/planner/RepeatNodeTest.java index 9e6f5f6027..a819e13081 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/planner/RepeatNodeTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/planner/RepeatNodeTest.java @@ -17,79 +17,64 @@ package org.apache.doris.planner; -import org.apache.doris.analysis.AccessTestUtil; -import org.apache.doris.analysis.Analyzer; -import org.apache.doris.analysis.DescriptorTable; -import org.apache.doris.analysis.SlotId; -import org.apache.doris.analysis.SlotRef; -import org.apache.doris.analysis.TableName; -import org.apache.doris.analysis.TupleDescriptor; -import org.apache.doris.analysis.TupleId; -import org.apache.doris.datasource.InternalDataSource; -import org.apache.doris.thrift.TExplainLevel; -import org.apache.doris.thrift.TPlanNode; -import org.apache.doris.thrift.TPlanNodeType; +import org.apache.doris.utframe.TestWithFeService; -import com.google.common.collect.ArrayListMultimap; -import com.google.common.collect.Multimap; -import org.junit.Assert; -import org.junit.Before; -import org.junit.Test; +import org.junit.jupiter.api.Assertions; +import org.junit.jupiter.api.Test; -import java.lang.reflect.Field; -import java.util.ArrayList; -import java.util.Arrays; -import java.util.List; -import java.util.Set; +public class RepeatNodeTest extends TestWithFeService { -public class RepeatNodeTest { - private Analyzer analyzer; - private RepeatNode node; - private TupleDescriptor virtualTuple; - private List<Set<SlotId>> groupingIdList = new ArrayList<>(); - private List<List<Long>> groupingList = new ArrayList<>(); + @Override + protected void runBeforeAll() throws Exception { + createDatabase("testdb"); + useDatabase("testdb"); + createTable(" CREATE TABLE `testdb`.`mycost` (\n" + " `id` tinyint(4) NULL,\n" + " `name` varchar(20) NULL,\n" + + " `date` date NULL,\n" + " `cost` bigint(20) SUM NULL\n" + ") ENGINE=OLAP\n" + + "AGGREGATE KEY(`id`, `name`, `date`)\n" + "COMMENT 'OLAP'\n" + "PARTITION BY RANGE(`date`)\n" + + "(PARTITION p2020 VALUES [('0000-01-01'), ('2021-01-01')),\n" + + "PARTITION p2021 VALUES [('2021-01-01'), ('2022-01-01')),\n" + + "PARTITION p2022 VALUES [('2022-01-01'), ('2023-01-01')))\n" + "DISTRIBUTED BY HASH(`id`) BUCKETS 8\n" + + "PROPERTIES (\n" + "\"replication_allocation\" = \"tag.location.default: 1\",\n" + + "\"in_memory\" = \"false\",\n" + "\"storage_format\" = \"V2\"\n" + ");"); - @Before - public void setUp() throws Exception { - Analyzer analyzerBase = AccessTestUtil.fetchTableAnalyzer(); - analyzer = new Analyzer(analyzerBase.getEnv(), analyzerBase.getContext()); - String[] cols = {"k1", "k2", "k3"}; - List<SlotRef> slots = new ArrayList<>(); - for (String col : cols) { - SlotRef expr = new SlotRef(new TableName(InternalDataSource.INTERNAL_DS_NAME, "testdb", "t"), col); - slots.add(expr); - } - try { - Field f = analyzer.getClass().getDeclaredField("tupleByAlias"); - f.setAccessible(true); - Multimap<String, TupleDescriptor> tupleByAlias = ArrayListMultimap.create(); - TupleDescriptor td = new TupleDescriptor(new TupleId(0)); - td.setTable(analyzerBase.getTableOrAnalysisException(new TableName(InternalDataSource.INTERNAL_DS_NAME, "testdb", "t"))); - tupleByAlias.put("testdb.t", td); - f.set(analyzer, tupleByAlias); - } catch (NoSuchFieldException | IllegalAccessException e) { - e.printStackTrace(); - } - virtualTuple = analyzer.getDescTbl().createTupleDescriptor("VIRTUAL_TUPLE"); - groupingList.add(Arrays.asList(0L, 7L, 3L, 5L, 1L, 6L, 2L, 4L)); - groupingList.add(Arrays.asList(0L, 7L, 3L, 5L, 1L, 6L, 2L, 4L)); - DescriptorTable descTable = new DescriptorTable(); - TupleDescriptor tuple = descTable.createTupleDescriptor("DstTable"); - node = new RepeatNode(new PlanNodeId(1), - new OlapScanNode(new PlanNodeId(0), tuple, "null"), groupingIdList, virtualTuple, groupingList); + createTable( + " CREATE TABLE `testdb`.`mypeople` (\n" + " `id` bigint(20) NULL,\n" + " `name` varchar(20) NULL,\n" + + " `sex` varchar(10) NULL,\n" + " `age` int(11) NULL,\n" + " `phone` char(15) NULL,\n" + + " `address` varchar(50) NULL\n" + ") ENGINE=OLAP\n" + "DUPLICATE KEY(`id`, `name`)\n" + + "COMMENT 'OLAP'\n" + "DISTRIBUTED BY HASH(`id`) BUCKETS 8\n" + "PROPERTIES (\n" + + "\"replication_allocation\" = \"tag.location.default: 1\",\n" + "\"in_memory\" = \"false\",\n" + + "\"storage_format\" = \"V2\"\n" + ");"); + } + @Test + public void testNormal() throws Exception { + String sql = "select id, name, sum(cost), grouping_id(id, name) from mycost group by cube(id, name);"; + String explainString = getSQLPlanOrErrorMsg("explain " + sql); + Assertions.assertTrue(explainString.contains("exprs: `id`, `name`, `cost`")); + Assertions.assertTrue(explainString.contains( + "output slots: ``id``, ``name``, ``cost``, ``GROUPING_ID``, ``GROUPING_PREFIX_`id`_`name```")); } @Test - public void testNormal() { - try { - TPlanNode msg = new TPlanNode(); - node.toThrift(msg); - node.getNodeExplainString("", TExplainLevel.NORMAL); - node.debugString(); - Assert.assertEquals(TPlanNodeType.REPEAT_NODE, msg.node_type); - } catch (Exception e) { - Assert.fail("throw exceptions"); - } + public void testExpr() throws Exception { + String sql1 = "select if(c.id > 0, 1, 0) as id_, p.name, sum(c.cost) from mycost c " + + "join mypeople p on c.id = p.id group by grouping sets((id_, name),());"; + String explainString1 = getSQLPlanOrErrorMsg("explain " + sql1); + System.out.println(explainString1); + Assertions.assertTrue(explainString1.contains( + "output slots: `if(`c`.`id` > 0, 1, 0)`, ``p`.`name``, ``c`.`cost``, ``GROUPING_ID``")); + + String sql2 = "select (id + 1) id_, name, sum(cost) from mycost group by grouping sets((id_, name),());"; + String explainString2 = getSQLPlanOrErrorMsg("explain " + sql2); + System.out.println(explainString2); + Assertions.assertTrue(explainString2.contains("exprs: (`id` + 1), `name`, `cost`")); + Assertions.assertTrue( + explainString2.contains(" output slots: `(`id` + 1)`, ``name``, ``cost``, ``GROUPING_ID``")); + + String sql3 = "select 1 as id_, name, sum(cost) from mycost group by grouping sets((id_, name),());"; + String explainString3 = getSQLPlanOrErrorMsg("explain " + sql3); + System.out.println(explainString3); + Assertions.assertTrue(explainString3.contains("exprs: 1, `name`, `cost`")); + Assertions.assertTrue(explainString3.contains("output slots: `1`, ``name``, ``cost``, ``GROUPING_ID``")); } } diff --git a/gensrc/thrift/PlanNodes.thrift b/gensrc/thrift/PlanNodes.thrift index 72967bf07c..ddd63bc92a 100644 --- a/gensrc/thrift/PlanNodes.thrift +++ b/gensrc/thrift/PlanNodes.thrift @@ -559,6 +559,7 @@ struct TRepeatNode { 4: required list<list<i64>> grouping_list // A list of all slot 5: required set<Types.TSlotId> all_slot_ids + 6: required list<Exprs.TExpr> exprs } struct TPreAggregationNode { diff --git a/regression-test/data/query/grouping_sets/test_grouping_sets.out b/regression-test/data/query/grouping_sets/test_grouping_sets.out index 71b1d5b44c..a11a1a577b 100644 --- a/regression-test/data/query/grouping_sets/test_grouping_sets.out +++ b/regression-test/data/query/grouping_sets/test_grouping_sets.out @@ -10,3 +10,40 @@ 3 \N 1002 3 1989 1002 +-- !select2 -- +\N \N 3004 +2 \N 1001 +2 1989 1001 +3 \N 1001 +3 1986 1001 +4 \N 1002 +4 1989 1002 + +-- !select3 -- +\N \N 6 +\N 1001 3 +\N 1002 3 +1 \N 6 +1 1001 3 +1 1002 3 + +-- !select4 -- +\N \N 6 +1986 wangynnsfstring12345 2 +1989 wangjuoo4string12345 1 +1989 yunlj8@nkstring12345 3 + +-- !select5 -- +\N 1986 1001 +\N 1989 2003 +2 1989 1001 +3 1986 1001 +4 1989 1002 + +-- !select6 -- +\N \N \N +\N \N 5973 +0 4 1991 +2 9 1991 +2 10 1991 + diff --git a/regression-test/suites/query/grouping_sets/test_grouping_sets.groovy b/regression-test/suites/query/grouping_sets/test_grouping_sets.groovy index a157836e59..2c8b64e010 100644 --- a/regression-test/suites/query/grouping_sets/test_grouping_sets.groovy +++ b/regression-test/suites/query/grouping_sets/test_grouping_sets.groovy @@ -20,4 +20,26 @@ suite("test_grouping_sets", "query") { SELECT k1, k2, SUM(k3) FROM test_query_db.test GROUP BY GROUPING SETS ((k1, k2), (k1), (k2), ( ) ) order by k1, k2 """ + + qt_select2 """ + select (k1 + 1) k1_, k2, sum(k3) from test_query_db.test group by + rollup(k1_, k2) order by k1_, k2 + """ + + qt_select3 "select 1 as k, k3, sum(k1) from test_query_db.test group by cube(k, k3) order by k, k3" + + qt_select4 """ + select k2, concat(k7, k12) as k_concat, sum(k1) from test_query_db.test group by + grouping sets((k2, k_concat),()) order by k2, k_concat + """ + + qt_select5 """ + select k1_, k2_, sum(k3_) from (select (k1 + 1) k1_, k2 k2_, k3 k3_ from test_query_db.test) as test + group by grouping sets((k1_, k2_), (k2_)) order by k1_, k2_ + """ + + qt_select6 """ + select if(k0 = 1, 2, k0) k_if, k1, sum(k2) k2_sum from test_query_db.baseall where k0 is null or k2 = 1991 + group by grouping sets((k_if, k1),()) order by k_if, k1, k2_sum + """ } --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org