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

englefly 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 423002b20a [fix](nereids) partitionTopN & Window estimation (#22953)
423002b20a is described below

commit 423002b20a3ae9da74898051835b3514a7f87db2
Author: minghong <engle...@gmail.com>
AuthorDate: Tue Aug 15 20:19:03 2023 +0800

    [fix](nereids) partitionTopN & Window estimation (#22953)
    
    * partitionTopN & winExpr estimation
    
    * tpcds 44/47/57
---
 .../doris/nereids/stats/ExpressionEstimation.java  |  18 ++--
 .../doris/nereids/stats/StatsCalculator.java       | 109 ++++++++++++++-------
 .../plans/physical/PhysicalPartitionTopN.java      |   3 +-
 .../trees/plans/physical/PhysicalQuickSort.java    |   2 +-
 .../trees/plans/physical/PhysicalWindow.java       |   2 +-
 .../nereids_tpcds_shape_sf100_p0/shape/query44.out |  62 ++++++------
 .../nereids_tpcds_shape_sf100_p0/shape/query47.out |  19 ++--
 .../nereids_tpcds_shape_sf100_p0/shape/query57.out |  20 ++--
 8 files changed, 135 insertions(+), 100 deletions(-)

diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/ExpressionEstimation.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/ExpressionEstimation.java
index 03e4be4d63..5324fce9e6 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/ExpressionEstimation.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/ExpressionEstimation.java
@@ -290,13 +290,12 @@ public class ExpressionEstimation extends 
ExpressionVisitor<ColumnStatistic, Sta
         }
         /*
         we keep columnStat.min and columnStat.max, but set ndv=1.
-        if there is group-by keys, we will update ndv when visiting group 
clause
+        if there is group-by keys, we will update count when visiting group 
clause
         */
         double width = min.child().getDataType().width();
-        return new 
ColumnStatisticBuilder().setCount(1).setNdv(1).setAvgSizeByte(width).setNumNulls(width)
-                
.setDataSize(child.getDataType().width()).setMinValue(columnStat.minValue)
-                .setMaxValue(columnStat.maxValue).setSelectivity(1.0)
-                .setMinExpr(null).build();
+        return new 
ColumnStatisticBuilder().setCount(1).setNdv(1).setAvgSizeByte(width)
+                
.setMinValue(columnStat.minValue).setMinExpr(columnStat.minExpr)
+                
.setMaxValue(columnStat.maxValue).setMaxExpr(columnStat.maxExpr).build();
     }
 
     @Override
@@ -308,12 +307,13 @@ public class ExpressionEstimation extends 
ExpressionVisitor<ColumnStatistic, Sta
         }
         /*
         we keep columnStat.min and columnStat.max, but set ndv=1.
-        if there is group-by keys, we will update ndv when visiting group 
clause
+        if there is group-by keys, we will update count when visiting group 
clause
         */
         int width = max.child().getDataType().width();
-        return new 
ColumnStatisticBuilder().setCount(1D).setNdv(1D).setAvgSizeByte(width).setNumNulls(0)
-                
.setDataSize(width).setMinValue(columnStat.minValue).setMaxValue(columnStat.maxValue)
-                .setSelectivity(1.0).setMaxExpr(null).setMinExpr(null).build();
+        return new 
ColumnStatisticBuilder().setCount(1D).setNdv(1D).setAvgSizeByte(width)
+                
.setMinValue(columnStat.minValue).setMinExpr(columnStat.minExpr)
+                
.setMaxValue(columnStat.maxValue).setMaxExpr(columnStat.maxExpr)
+                .build();
     }
 
     @Override
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/StatsCalculator.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/StatsCalculator.java
index 5f7b4cc3de..380d34dd1d 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/StatsCalculator.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/StatsCalculator.java
@@ -17,6 +17,7 @@
 
 package org.apache.doris.nereids.stats;
 
+import org.apache.doris.analysis.IntLiteral;
 import org.apache.doris.catalog.Column;
 import org.apache.doris.catalog.Env;
 import org.apache.doris.catalog.OlapTable;
@@ -38,7 +39,9 @@ import org.apache.doris.nereids.trees.expressions.Slot;
 import org.apache.doris.nereids.trees.expressions.SlotReference;
 import org.apache.doris.nereids.trees.expressions.WindowExpression;
 import 
org.apache.doris.nereids.trees.expressions.functions.agg.AggregateFunction;
-import org.apache.doris.nereids.trees.expressions.functions.window.Rank;
+import org.apache.doris.nereids.trees.expressions.functions.agg.Count;
+import org.apache.doris.nereids.trees.expressions.functions.agg.Max;
+import org.apache.doris.nereids.trees.expressions.functions.agg.Min;
 import org.apache.doris.nereids.trees.plans.Plan;
 import org.apache.doris.nereids.trees.plans.algebra.Aggregate;
 import org.apache.doris.nereids.trees.plans.algebra.CatalogRelation;
@@ -681,16 +684,20 @@ public class StatsCalculator extends 
DefaultPlanVisitor<Statistics, Void> {
     }
 
     private Statistics computePartitionTopN(PartitionTopN partitionTopN) {
-        Statistics stats = groupExpression.childStatistics(0);
-        double rowCount = stats.getRowCount();
+        Statistics childStats = groupExpression.childStatistics(0);
+        double rowCount = childStats.getRowCount();
         List<Expression> partitionKeys = partitionTopN.getPartitionKeys();
         if (!partitionTopN.hasGlobalLimit() && !partitionKeys.isEmpty()) {
             // If there is no global limit. So result for the cardinality 
estimation is:
             // NDV(partition key) * partitionLimit
-            Map<Expression, ColumnStatistic> childSlotToColumnStats = 
stats.columnStatistics();
             List<ColumnStatistic> partitionByKeyStats = partitionKeys.stream()
-                    .filter(childSlotToColumnStats::containsKey)
-                    .map(childSlotToColumnStats::get)
+                    .map(partitionKey -> {
+                        ColumnStatistic partitionKeyStats = 
childStats.findColumnStatistics(partitionKey);
+                        if (partitionKeyStats == null) {
+                            partitionKeyStats = new 
ExpressionEstimation().visit(partitionKey, childStats);
+                        }
+                        return partitionKeyStats;
+                    })
                     .filter(s -> !s.isUnKnown)
                     .collect(Collectors.toList());
             if (partitionByKeyStats.isEmpty()) {
@@ -698,7 +705,7 @@ public class StatsCalculator extends 
DefaultPlanVisitor<Statistics, Void> {
                 rowCount = rowCount * DEFAULT_COLUMN_NDV_RATIO;
             } else {
                 rowCount = Math.min(rowCount, 
partitionByKeyStats.stream().map(s -> s.ndv)
-                    .max(Double::compare).get());
+                    .max(Double::compare).get() * 
partitionTopN.getPartitionLimit());
             }
         } else {
             rowCount = Math.min(rowCount, partitionTopN.getPartitionLimit());
@@ -706,7 +713,7 @@ public class StatsCalculator extends 
DefaultPlanVisitor<Statistics, Void> {
         // TODO: for the filter push down window situation, we will prune the 
row count twice
         //  because we keep the pushed down filter. And it will be calculated 
twice, one of them in 'PartitionTopN'
         //  and the other is in 'Filter'. It's hard to dismiss.
-        return stats.updateRowCountOnly(rowCount);
+        return childStats.updateRowCountOnly(rowCount);
     }
 
     private Statistics computeLimit(Limit limit) {
@@ -961,42 +968,72 @@ public class StatsCalculator extends 
DefaultPlanVisitor<Statistics, Void> {
     }
 
     private Statistics computeWindow(Window windowOperator) {
-        Statistics stats = groupExpression.childStatistics(0);
-        Map<Expression, ColumnStatistic> childColumnStats = 
stats.columnStatistics();
+        Statistics childStats = groupExpression.childStatistics(0);
+        Map<Expression, ColumnStatistic> childColumnStats = 
childStats.columnStatistics();
         Map<Expression, ColumnStatistic> columnStatisticMap = 
windowOperator.getWindowExpressions().stream()
                 .map(expr -> {
-                    //estimate rank()
-                    if (expr instanceof Alias && expr.child(0) instanceof 
WindowExpression
-                            && ((WindowExpression) 
expr.child(0)).getFunction() instanceof Rank) {
-                        ColumnStatisticBuilder colBuilder = new 
ColumnStatisticBuilder();
-                        colBuilder.setNdv(stats.getRowCount())
-                                .setOriginal(null)
-                                .setCount(stats.getRowCount())
-                                .setMinValue(0)
-                                .setMaxValue(stats.getRowCount());
-                        return Pair.of(expr.toSlot(), colBuilder.build());
-                    }
-                    //estimate other expressions
-                    ColumnStatistic value = null;
-                    Set<Slot> slots = expr.getInputSlots();
-                    if (slots.isEmpty()) {
-                        value = ColumnStatistic.UNKNOWN;
+                    Preconditions.checkArgument(expr instanceof Alias
+                            && expr.child(0) instanceof WindowExpression,
+                            "need WindowExpression, but we meet " + expr);
+                    WindowExpression windExpr = (WindowExpression) 
expr.child(0);
+                    ColumnStatisticBuilder colStatsBuilder = new 
ColumnStatisticBuilder();
+                    colStatsBuilder.setCount(childStats.getRowCount())
+                            .setOriginal(null);
+
+                    Double partitionCount = 
windExpr.getPartitionKeys().stream().map(key -> {
+                        ColumnStatistic keyStats = 
childStats.findColumnStatistics(key);
+                        if (keyStats == null) {
+                            keyStats = new ExpressionEstimation().visit(key, 
childStats);
+                        }
+                        return keyStats;
+                    })
+                            .filter(columnStatistic -> 
!columnStatistic.isUnKnown)
+                            .map(colStats -> colStats.ndv).max(Double::compare)
+                            .orElseGet(() -> -1.0);
+
+                    if (partitionCount == -1.0) {
+                        // partition key stats are all unknown
+                        colStatsBuilder.setCount(childStats.getRowCount())
+                                .setNdv(1)
+                                .setMinValue(Double.NEGATIVE_INFINITY)
+                                .setMaxValue(Double.POSITIVE_INFINITY);
                     } else {
-                        for (Slot slot : slots) {
-                            if (childColumnStats.containsKey(slot)) {
-                                value = childColumnStats.get(slot);
-                                break;
+                        partitionCount = Math.max(1, partitionCount);
+                        if (windExpr.getFunction() instanceof 
AggregateFunction) {
+                            if (windExpr.getFunction() instanceof Count) {
+                                colStatsBuilder.setNdv(1)
+                                        .setMinValue(0)
+                                        .setMinExpr(new IntLiteral(0))
+                                        .setMaxValue(childStats.getRowCount())
+                                        .setMaxExpr(new IntLiteral((long) 
childStats.getRowCount()));
+                            } else if (windExpr.getFunction() instanceof Min
+                                    || windExpr.getFunction() instanceof Max) {
+                                Expression minmaxChild = 
windExpr.getFunction().child(0);
+                                ColumnStatistic minChildStats = new 
ExpressionEstimation()
+                                        .visit(minmaxChild, childStats);
+                                colStatsBuilder.setNdv(1)
+                                        .setMinValue(minChildStats.minValue)
+                                        .setMinExpr(minChildStats.minExpr)
+                                        .setMaxValue(minChildStats.maxValue)
+                                        .setMaxExpr(minChildStats.maxExpr);
+                            } else {
+                                // sum/avg
+                                
colStatsBuilder.setNdv(1).setMinValue(Double.NEGATIVE_INFINITY)
+                                        .setMaxValue(Double.POSITIVE_INFINITY);
                             }
-                        }
-                        if (value == null) {
-                            // todo: how to set stats?
-                            value = ColumnStatistic.UNKNOWN;
+                        } else {
+                            // rank/dense_rank/row_num ...
+                            colStatsBuilder.setNdv(childStats.getRowCount() / 
partitionCount)
+                                    .setMinValue(0)
+                                    .setMinExpr(new IntLiteral(0))
+                                    .setMaxValue(childStats.getRowCount())
+                                    .setMaxExpr(new IntLiteral((long) 
childStats.getRowCount()));
                         }
                     }
-                    return Pair.of(expr.toSlot(), value);
+                    return Pair.of(expr.toSlot(), colStatsBuilder.build());
                 }).collect(Collectors.toMap(Pair::key, Pair::value));
         columnStatisticMap.putAll(childColumnStats);
-        return new Statistics(stats.getRowCount(), columnStatisticMap);
+        return new Statistics(childStats.getRowCount(), columnStatisticMap);
     }
 
     private ColumnStatistic unionColumn(ColumnStatistic leftStats, double 
leftRowCount, ColumnStatistic rightStats,
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalPartitionTopN.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalPartitionTopN.java
index ce57ad0782..4166e7d903 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalPartitionTopN.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalPartitionTopN.java
@@ -183,7 +183,8 @@ public class PhysicalPartitionTopN<CHILD_TYPE extends Plan> 
extends PhysicalUnar
             "partitionKeys", partitionKeys,
             "orderKeys", orderKeys,
             "hasGlobalLimit", hasGlobalLimit,
-            "partitionLimit", partitionLimit
+            "partitionLimit", partitionLimit,
+            "stats", statistics
         );
     }
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalQuickSort.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalQuickSort.java
index 5981c8e538..57c77cc4fb 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalQuickSort.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalQuickSort.java
@@ -103,7 +103,7 @@ public class PhysicalQuickSort<CHILD_TYPE extends Plan> 
extends AbstractPhysical
     public String toString() {
         return Utils.toSqlString("PhysicalQuickSort[" + id.asInt() + "]" + 
getGroupIdWithPrefix(),
                 "orderKeys", orderKeys,
-                "phase", phase.toString()
+                "phase", phase.toString(), "stats", statistics
         );
     }
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalWindow.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalWindow.java
index 5632e970a3..b1703f4749 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalWindow.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalWindow.java
@@ -106,7 +106,7 @@ public class PhysicalWindow<CHILD_TYPE extends Plan> 
extends PhysicalUnary<CHILD
     public String toString() {
         return Utils.toSqlString("PhysicalWindow[" + id.asInt() + "]" + 
getGroupIdWithPrefix(),
             "windowFrameGroup", windowFrameGroup,
-            "requiredProperties", requireProperties
+            "requiredProperties", requireProperties, "stats", statistics
         );
     }
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query44.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query44.out
index ef988f8e86..8028e752fd 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query44.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query44.out
@@ -5,11 +5,11 @@ PhysicalResultSink
 ----PhysicalDistribute
 ------PhysicalTopN
 --------PhysicalProject
-----------hashJoin[INNER_JOIN](i1.i_item_sk = asceding.item_sk)
-------------PhysicalProject
---------------PhysicalOlapScan[item]
+----------hashJoin[INNER_JOIN](asceding.rnk = descending.rnk)
 ------------PhysicalDistribute
---------------hashJoin[INNER_JOIN](asceding.rnk = descending.rnk)
+--------------hashJoin[INNER_JOIN](i1.i_item_sk = asceding.item_sk)
+----------------PhysicalProject
+------------------PhysicalOlapScan[item]
 ----------------PhysicalDistribute
 ------------------PhysicalProject
 --------------------filter((rnk < 11))
@@ -36,34 +36,34 @@ PhysicalResultSink
 --------------------------------------------------PhysicalProject
 ----------------------------------------------------filter(ss_addr_sk IS 
NULL(store_sales.ss_store_sk = 146))
 
------------------------------------------------------PhysicalOlapScan[store_sales]
+------------PhysicalDistribute
+--------------hashJoin[INNER_JOIN](i2.i_item_sk = descending.item_sk)
+----------------PhysicalProject
+------------------PhysicalOlapScan[item]
 ----------------PhysicalDistribute
-------------------hashJoin[INNER_JOIN](i2.i_item_sk = descending.item_sk)
---------------------PhysicalProject
-----------------------PhysicalOlapScan[item]
---------------------PhysicalDistribute
-----------------------PhysicalProject
-------------------------filter((rnk < 11))
---------------------------PhysicalWindow
+------------------PhysicalProject
+--------------------filter((rnk < 11))
+----------------------PhysicalWindow
+------------------------PhysicalQuickSort
+--------------------------PhysicalDistribute
 ----------------------------PhysicalQuickSort
-------------------------------PhysicalDistribute
---------------------------------PhysicalQuickSort
-----------------------------------PhysicalPartitionTopN
-------------------------------------PhysicalProject
---------------------------------------NestedLoopJoin[INNER_JOIN](cast(rank_col 
as DOUBLE) > cast((0.9 * rank_col) as DOUBLE))
-----------------------------------------hashAgg[GLOBAL]
-------------------------------------------PhysicalDistribute
---------------------------------------------hashAgg[LOCAL]
-----------------------------------------------PhysicalProject
-------------------------------------------------filter((ss1.ss_store_sk = 146))
---------------------------------------------------PhysicalOlapScan[store_sales]
+------------------------------PhysicalPartitionTopN
+--------------------------------PhysicalProject
+----------------------------------NestedLoopJoin[INNER_JOIN](cast(rank_col as 
DOUBLE) > cast((0.9 * rank_col) as DOUBLE))
+------------------------------------hashAgg[GLOBAL]
+--------------------------------------PhysicalDistribute
+----------------------------------------hashAgg[LOCAL]
+------------------------------------------PhysicalProject
+--------------------------------------------filter((ss1.ss_store_sk = 146))
+----------------------------------------------PhysicalOlapScan[store_sales]
+------------------------------------PhysicalDistribute
+--------------------------------------PhysicalAssertNumRows
 ----------------------------------------PhysicalDistribute
-------------------------------------------PhysicalAssertNumRows
---------------------------------------------PhysicalDistribute
-----------------------------------------------PhysicalProject
-------------------------------------------------hashAgg[GLOBAL]
---------------------------------------------------PhysicalDistribute
-----------------------------------------------------hashAgg[LOCAL]
-------------------------------------------------------PhysicalProject
---------------------------------------------------------filter(ss_addr_sk IS 
NULL(store_sales.ss_store_sk = 146))
-----------------------------------------------------------PhysicalOlapScan[store_sales]
+------------------------------------------PhysicalProject
+--------------------------------------------hashAgg[GLOBAL]
+----------------------------------------------PhysicalDistribute
+------------------------------------------------hashAgg[LOCAL]
+--------------------------------------------------PhysicalProject
+----------------------------------------------------filter(ss_addr_sk IS 
NULL(store_sales.ss_store_sk = 146))
+------------------------------------------------------PhysicalOlapScan[store_sales]
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query47.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query47.out
index b705f7a2cb..3c6bf21b99 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query47.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query47.out
@@ -36,17 +36,16 @@ PhysicalCteAnchor ( cteId=CTEId#0 )
 ----------PhysicalTopN
 ------------PhysicalProject
 --------------hashJoin[INNER_JOIN](v1.i_category = 
v1_lead.i_category)(v1.i_brand = v1_lead.i_brand)(v1.s_store_name = 
v1_lead.s_store_name)(v1.s_company_name = v1_lead.s_company_name)(v1.rn = 
expr_(rn - 1))
+----------------PhysicalProject
+------------------hashJoin[INNER_JOIN](v1.i_category = 
v1_lag.i_category)(v1.i_brand = v1_lag.i_brand)(v1.s_store_name = 
v1_lag.s_store_name)(v1.s_company_name = v1_lag.s_company_name)(v1.rn = 
expr_(rn + 1))
+--------------------PhysicalDistribute
+----------------------PhysicalProject
+------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
+--------------------PhysicalDistribute
+----------------------PhysicalProject
+------------------------filter((CASE WHEN (avg_monthly_sales > 0.0000) THEN 
(abs((cast(sum_sales as DOUBLE) - cast(avg_monthly_sales as DOUBLE))) / 
cast(avg_monthly_sales as DOUBLE)) ELSE NULL END > 0.1)(v2.d_year = 
2001)(v2.avg_monthly_sales > 0.0000))
+--------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
 ----------------PhysicalDistribute
 ------------------PhysicalProject
 --------------------PhysicalCteConsumer ( cteId=CTEId#0 )
-----------------PhysicalDistribute
-------------------PhysicalProject
---------------------hashJoin[INNER_JOIN](v1.i_category = 
v1_lag.i_category)(v1.i_brand = v1_lag.i_brand)(v1.s_store_name = 
v1_lag.s_store_name)(v1.s_company_name = v1_lag.s_company_name)(v1.rn = 
expr_(rn + 1))
-----------------------PhysicalDistribute
-------------------------PhysicalProject
---------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
-----------------------PhysicalDistribute
-------------------------PhysicalProject
---------------------------filter((CASE WHEN (avg_monthly_sales > 0.0000) THEN 
(abs((cast(sum_sales as DOUBLE) - cast(avg_monthly_sales as DOUBLE))) / 
cast(avg_monthly_sales as DOUBLE)) ELSE NULL END > 0.1)(v2.d_year = 
2001)(v2.avg_monthly_sales > 0.0000))
-----------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query57.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query57.out
index 2e7f06968a..71c2cdd060 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query57.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query57.out
@@ -36,18 +36,16 @@ PhysicalCteAnchor ( cteId=CTEId#0 )
 --------PhysicalDistribute
 ----------PhysicalTopN
 ------------PhysicalProject
---------------hashJoin[INNER_JOIN](v1.i_category = 
v1_lead.i_category)(v1.i_brand = v1_lead.i_brand)(v1.cc_name = 
v1_lead.cc_name)(v1.rn = expr_(rn - 1))
+--------------hashJoin[INNER_JOIN](v1.i_category = 
v1_lag.i_category)(v1.i_brand = v1_lag.i_brand)(v1.cc_name = 
v1_lag.cc_name)(v1.rn = expr_(rn + 1))
+----------------hashJoin[INNER_JOIN](v1.i_category = 
v1_lead.i_category)(v1.i_brand = v1_lead.i_brand)(v1.cc_name = 
v1_lead.cc_name)(v1.rn = expr_(rn - 1))
+------------------PhysicalDistribute
+--------------------PhysicalProject
+----------------------PhysicalCteConsumer ( cteId=CTEId#0 )
+------------------PhysicalDistribute
+--------------------PhysicalProject
+----------------------filter((CASE WHEN (avg_monthly_sales > 0.0000) THEN 
(abs((cast(sum_sales as DOUBLE) - cast(avg_monthly_sales as DOUBLE))) / 
cast(avg_monthly_sales as DOUBLE)) ELSE NULL END > 0.1)(v2.d_year = 
1999)(v2.avg_monthly_sales > 0.0000))
+------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
 ----------------PhysicalDistribute
 ------------------PhysicalProject
 --------------------PhysicalCteConsumer ( cteId=CTEId#0 )
-----------------PhysicalDistribute
-------------------PhysicalProject
---------------------hashJoin[INNER_JOIN](v1.i_category = 
v1_lag.i_category)(v1.i_brand = v1_lag.i_brand)(v1.cc_name = 
v1_lag.cc_name)(v1.rn = expr_(rn + 1))
-----------------------PhysicalDistribute
-------------------------PhysicalProject
---------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
-----------------------PhysicalDistribute
-------------------------PhysicalProject
---------------------------filter((CASE WHEN (avg_monthly_sales > 0.0000) THEN 
(abs((cast(sum_sales as DOUBLE) - cast(avg_monthly_sales as DOUBLE))) / 
cast(avg_monthly_sales as DOUBLE)) ELSE NULL END > 0.1)(v2.d_year = 
1999)(v2.avg_monthly_sales > 0.0000))
-----------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
 


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

Reply via email to