You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by da...@apache.org on 2022/10/18 14:09:11 UTC

[doris] branch master updated: [fix](sort)the sort expr's nullability property may not be right (#13328)

This is an automated email from the ASF dual-hosted git repository.

dataroaring 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 ac037e57f5 [fix](sort)the sort expr's nullability property may not be right (#13328)
ac037e57f5 is described below

commit ac037e57f5dd7eaf69dafb907794462da275a733
Author: starocean999 <40...@users.noreply.github.com>
AuthorDate: Tue Oct 18 22:09:02 2022 +0800

    [fix](sort)the sort expr's nullability property may not be right (#13328)
---
 be/src/vec/common/sort/heap_sorter.cpp               | 11 ++++++++++-
 be/src/vec/common/sort/sorter.cpp                    | 11 ++++++++++-
 be/src/vec/common/sort/vsort_exec_exprs.cpp          |  8 ++++++++
 be/src/vec/common/sort/vsort_exec_exprs.h            | 20 ++++++++++++++------
 .../main/java/org/apache/doris/planner/SortNode.java | 15 +++++++++++++++
 gensrc/thrift/PlanNodes.thrift                       |  3 +++
 .../data/correctness_p0/test_outer_join_sort.out     |  3 +++
 .../correctness_p0/test_outer_join_sort.groovy       | 20 ++++++++++++++++++++
 8 files changed, 83 insertions(+), 8 deletions(-)

diff --git a/be/src/vec/common/sort/heap_sorter.cpp b/be/src/vec/common/sort/heap_sorter.cpp
index 6520b005a4..6ca7fae7b2 100644
--- a/be/src/vec/common/sort/heap_sorter.cpp
+++ b/be/src/vec/common/sort/heap_sorter.cpp
@@ -41,8 +41,17 @@ Status HeapSorter::append_block(Block* block) {
             }
 
             Block new_block;
+            int i = 0;
+            const auto& convert_nullable_flags = _vsort_exec_exprs.get_convert_nullable_flags();
             for (auto column_id : valid_column_ids) {
-                new_block.insert(block->get_by_position(column_id));
+                if (convert_nullable_flags[i]) {
+                    auto column_ptr = make_nullable(block->get_by_position(column_id).column);
+                    new_block.insert({column_ptr,
+                                      make_nullable(block->get_by_position(column_id).type), ""});
+                } else {
+                    new_block.insert(block->get_by_position(column_id));
+                }
+                i++;
             }
             block->swap(new_block);
         }
diff --git a/be/src/vec/common/sort/sorter.cpp b/be/src/vec/common/sort/sorter.cpp
index 5de7499a2e..4fb99cb507 100644
--- a/be/src/vec/common/sort/sorter.cpp
+++ b/be/src/vec/common/sort/sorter.cpp
@@ -82,8 +82,17 @@ Status Sorter::partial_sort(Block& src_block, Block& dest_block) {
         }
 
         Block new_block;
+        int i = 0;
+        const auto& convert_nullable_flags = _vsort_exec_exprs.get_convert_nullable_flags();
         for (auto column_id : valid_column_ids) {
-            new_block.insert(src_block.get_by_position(column_id));
+            if (convert_nullable_flags[i]) {
+                auto column_ptr = make_nullable(src_block.get_by_position(column_id).column);
+                new_block.insert(
+                        {column_ptr, make_nullable(src_block.get_by_position(column_id).type), ""});
+            } else {
+                new_block.insert(src_block.get_by_position(column_id));
+            }
+            i++;
         }
         dest_block.swap(new_block);
     }
diff --git a/be/src/vec/common/sort/vsort_exec_exprs.cpp b/be/src/vec/common/sort/vsort_exec_exprs.cpp
index e542a897c3..373b6b4c34 100644
--- a/be/src/vec/common/sort/vsort_exec_exprs.cpp
+++ b/be/src/vec/common/sort/vsort_exec_exprs.cpp
@@ -20,6 +20,14 @@
 namespace doris::vectorized {
 
 Status VSortExecExprs::init(const TSortInfo& sort_info, ObjectPool* pool) {
+    if (sort_info.__isset.slot_exprs_nullability_changed_flags) {
+        _need_convert_to_nullable_flags = sort_info.slot_exprs_nullability_changed_flags;
+    } else {
+        _need_convert_to_nullable_flags.resize(sort_info.__isset.sort_tuple_slot_exprs
+                                                       ? sort_info.sort_tuple_slot_exprs.size()
+                                                       : 0,
+                                               false);
+    }
     return init(sort_info.ordering_exprs,
                 sort_info.__isset.sort_tuple_slot_exprs ? &sort_info.sort_tuple_slot_exprs : NULL,
                 pool);
diff --git a/be/src/vec/common/sort/vsort_exec_exprs.h b/be/src/vec/common/sort/vsort_exec_exprs.h
index 5e5b044d16..c030c86fb2 100644
--- a/be/src/vec/common/sort/vsort_exec_exprs.h
+++ b/be/src/vec/common/sort/vsort_exec_exprs.h
@@ -37,12 +37,6 @@ public:
     // Initialize the expressions from a TSortInfo using the specified pool.
     Status init(const TSortInfo& sort_info, ObjectPool* pool);
 
-    // Initialize the ordering and (optionally) materialization expressions from the thrift
-    // TExprs into the specified pool. sort_tuple_slot_exprs is NULL if the tuple is not
-    // materialized.
-    Status init(const std::vector<TExpr>& ordering_exprs,
-                const std::vector<TExpr>* sort_tuple_slot_exprs, ObjectPool* pool);
-
     // prepare all expressions used for sorting and tuple materialization.
     Status prepare(RuntimeState* state, const RowDescriptor& child_row_desc,
                    const RowDescriptor& output_row_desc);
@@ -69,6 +63,10 @@ public:
 
     bool need_materialize_tuple() const { return _materialize_tuple; }
 
+    const std::vector<bool>& get_convert_nullable_flags() const {
+        return _need_convert_to_nullable_flags;
+    }
+
 private:
     // Create two VExprContexts for evaluating over the TupleRows.
     std::vector<VExprContext*> _lhs_ordering_expr_ctxs;
@@ -83,11 +81,21 @@ private:
     // _materialize_tuple is true.
     std::vector<VExprContext*> _sort_tuple_slot_expr_ctxs;
 
+    // for some reason, _sort_tuple_slot_expr_ctxs is not-null but _lhs_ordering_expr_ctxs is nullable
+    // this flag list would be used to convert column to nullable.
+    std::vector<bool> _need_convert_to_nullable_flags;
+
     // Initialize directly from already-created VExprContexts. Callers should manually call
     // Prepare(), Open(), and Close() on input VExprContexts (instead of calling the
     // analogous functions in this class). Used for testing.
     Status init(const std::vector<VExprContext*>& lhs_ordering_expr_ctxs,
                 const std::vector<VExprContext*>& rhs_ordering_expr_ctxs);
+
+    // Initialize the ordering and (optionally) materialization expressions from the thrift
+    // TExprs into the specified pool. sort_tuple_slot_exprs is NULL if the tuple is not
+    // materialized.
+    Status init(const std::vector<TExpr>& ordering_exprs,
+                const std::vector<TExpr>* sort_tuple_slot_exprs, ObjectPool* pool);
 };
 
 } // namespace vectorized
diff --git a/fe/fe-core/src/main/java/org/apache/doris/planner/SortNode.java b/fe/fe-core/src/main/java/org/apache/doris/planner/SortNode.java
index d5ab71573f..f2359b43d4 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/planner/SortNode.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/planner/SortNode.java
@@ -71,6 +71,8 @@ public class SortNode extends PlanNode {
 
     private boolean isUnusedExprRemoved = false;
 
+    private ArrayList<Boolean> nullabilityChangedFlags = Lists.newArrayList();
+
     /**
      * Constructor.
      */
@@ -218,11 +220,16 @@ public class SortNode extends PlanNode {
         for (int i = 0; i < slotExprs.size(); ++i) {
             resolvedTupleExprs.add(slotExprs.get(i));
             outputSmap.put(slotExprs.get(i), new SlotRef(sortTupleSlots.get(i)));
+            nullabilityChangedFlags.add(slotExprs.get(i).isNullable());
         }
 
         ExprSubstitutionMap childSmap = getCombinedChildSmap();
         resolvedTupleExprs = Expr.substituteList(resolvedTupleExprs, childSmap, analyzer, false);
 
+        for (int i = 0; i < resolvedTupleExprs.size(); ++i) {
+            nullabilityChangedFlags.set(i, nullabilityChangedFlags.get(i) ^ resolvedTupleExprs.get(i).isNullable());
+        }
+
         // Remap the ordering exprs to the tuple materialized by this sort node. The mapping
         // is a composition of the childSmap and the outputSmap_ because the child node may
         // have also remapped its input (e.g., as in a series of (sort->analytic)* nodes).
@@ -250,6 +257,7 @@ public class SortNode extends PlanNode {
                 for (int i = slotDescriptorList.size() - 1; i >= 0; i--) {
                     if (!slotDescriptorList.get(i).isMaterialized()) {
                         resolvedTupleExprs.remove(i);
+                        nullabilityChangedFlags.remove(i);
                     }
                 }
             }
@@ -266,6 +274,9 @@ public class SortNode extends PlanNode {
         removeUnusedExprs();
         if (resolvedTupleExprs != null) {
             sortInfo.setSortTupleSlotExprs(Expr.treesToThrift(resolvedTupleExprs));
+            // FIXME this is a bottom line solution for wrong nullability of resolvedTupleExprs
+            // remove the following line after nereids online
+            sortInfo.setSlotExprsNullabilityChangedFlags(nullabilityChangedFlags);
         }
         TSortNode sortNode = new TSortNode(sortInfo, useTopN);
 
@@ -313,5 +324,9 @@ public class SortNode extends PlanNode {
         info.setSortTupleDesc(tupleDescriptor);
         info.setSortTupleSlotExprs(resolvedTupleExprs);
 
+        nullabilityChangedFlags.clear();
+        for (int i = 0; i < resolvedTupleExprs.size(); i++) {
+            nullabilityChangedFlags.add(false);
+        }
     }
 }
diff --git a/gensrc/thrift/PlanNodes.thrift b/gensrc/thrift/PlanNodes.thrift
index 2ff101acee..0f70c29ae2 100644
--- a/gensrc/thrift/PlanNodes.thrift
+++ b/gensrc/thrift/PlanNodes.thrift
@@ -482,6 +482,9 @@ struct TSortInfo {
   // Expressions evaluated over the input row that materialize the tuple to be sorted.
   // Contains one expr per slot in the materialized tuple.
   4: optional list<Exprs.TExpr> sort_tuple_slot_exprs
+
+  // Indicates the nullable info of sort_tuple_slot_exprs is changed after substitute by child's smap
+  5: optional list<bool> slot_exprs_nullability_changed_flags   
 }
 
 enum TPushAggOp {
diff --git a/regression-test/data/correctness_p0/test_outer_join_sort.out b/regression-test/data/correctness_p0/test_outer_join_sort.out
index 72d126351a..875d88ccc5 100644
--- a/regression-test/data/correctness_p0/test_outer_join_sort.out
+++ b/regression-test/data/correctness_p0/test_outer_join_sort.out
@@ -2,3 +2,6 @@
 -- !select --
 1
 
+-- !select --
+1
+
diff --git a/regression-test/suites/correctness_p0/test_outer_join_sort.groovy b/regression-test/suites/correctness_p0/test_outer_join_sort.groovy
index 4d8bd9a76b..e5e7e0567f 100644
--- a/regression-test/suites/correctness_p0/test_outer_join_sort.groovy
+++ b/regression-test/suites/correctness_p0/test_outer_join_sort.groovy
@@ -92,6 +92,26 @@ suite("test_outer_join_sort") {
         order by subq_0.`c3`;
     """
 
+    qt_select """
+        select
+        case
+            when outerjoin_C.a is  NULL then subq_0.`c1`
+            else subq_0.`c2`
+        end as c0
+        from
+        (
+            select
+            1 as c0,
+            version() as c1,
+            a as c2
+            from
+            test_test_outer_join_sort_outerjoin_A
+        ) as subq_0
+        right join outerjoin_C on (subq_0.`c0` = outerjoin_C.a)
+        order by
+        subq_0.`c1`;
+    """
+
     sql """
         drop table if exists test_test_outer_join_sort_outerjoin_A;
     """


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org
For additional commands, e-mail: commits-help@doris.apache.org