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

jakevin 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 8a03a073399 [feature](Nereids): pull up Project under Limit/TopN 
(#25866)
8a03a073399 is described below

commit 8a03a0733992a4d163a321631eb12eef0e0276c7
Author: jakevin <jakevin...@gmail.com>
AuthorDate: Wed Oct 25 16:44:32 2023 +0800

    [feature](Nereids): pull up Project under Limit/TopN (#25866)
    
    If project contains expression-eval, we need pull up through limit, it can 
improve performance.
---
 .../doris/nereids/jobs/executor/Rewriter.java      |   8 +-
 .../org/apache/doris/nereids/rules/RuleType.java   |   2 +
 .../rules/rewrite/PullUpProjectUnderLimit.java     |  55 +++++++
 .../rules/rewrite/PullUpProjectUnderTopN.java      |  60 ++++++++
 .../rules/rewrite/PullUpProjectUnderLimitTest.java |  60 ++++++++
 .../rules/rewrite/PullUpProjectUnderTopNTest.java  |  60 ++++++++
 .../shape/query10.out                              |   8 +-
 .../shape/query17.out                              |   8 +-
 .../shape/query27.out                              |  61 ++++----
 .../shape/query35.out                              |   8 +-
 .../shape/query54.out                              |   8 +-
 .../shape/query58.out                              | 165 +++++++++++----------
 .../shape/query59.out                              |  61 ++++----
 .../shape/query61.out                              |   4 +-
 .../shape/query69.out                              |   8 +-
 .../shape/query83.out                              | 119 +++++++--------
 .../nereids_tpcds_shape_sf100_p0/shape/query10.out |   8 +-
 .../nereids_tpcds_shape_sf100_p0/shape/query17.out |   8 +-
 .../nereids_tpcds_shape_sf100_p0/shape/query27.out |  61 ++++----
 .../nereids_tpcds_shape_sf100_p0/shape/query35.out |   8 +-
 .../nereids_tpcds_shape_sf100_p0/shape/query54.out |   8 +-
 .../nereids_tpcds_shape_sf100_p0/shape/query58.out | 129 ++++++++--------
 .../nereids_tpcds_shape_sf100_p0/shape/query59.out |  61 ++++----
 .../nereids_tpcds_shape_sf100_p0/shape/query61.out |   4 +-
 .../nereids_tpcds_shape_sf100_p0/shape/query69.out |   8 +-
 .../nereids_tpcds_shape_sf100_p0/shape/query83.out | 119 +++++++--------
 26 files changed, 680 insertions(+), 429 deletions(-)

diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Rewriter.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Rewriter.java
index e1db7bfb8d0..5003e5e1ea6 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Rewriter.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Rewriter.java
@@ -85,6 +85,8 @@ import 
org.apache.doris.nereids.rules.rewrite.PruneOlapScanPartition;
 import org.apache.doris.nereids.rules.rewrite.PruneOlapScanTablet;
 import org.apache.doris.nereids.rules.rewrite.PullUpCteAnchor;
 import org.apache.doris.nereids.rules.rewrite.PullUpProjectUnderApply;
+import org.apache.doris.nereids.rules.rewrite.PullUpProjectUnderLimit;
+import org.apache.doris.nereids.rules.rewrite.PullUpProjectUnderTopN;
 import org.apache.doris.nereids.rules.rewrite.PushConjunctsIntoEsScan;
 import org.apache.doris.nereids.rules.rewrite.PushConjunctsIntoJdbcScan;
 import org.apache.doris.nereids.rules.rewrite.PushFilterInsideJoin;
@@ -288,6 +290,10 @@ public class Rewriter extends AbstractBatchJobExecutor {
                             new PushdownLimitDistinctThroughJoin(),
                             new PushdownTopNThroughWindow(),
                             new CreatePartitionTopNFromWindow()
+                    ),
+                    topDown(
+                            new PullUpProjectUnderTopN(),
+                            new PullUpProjectUnderLimit()
                     )
             ),
             // TODO: these rules should be implementation rules, and generate 
alternative physical plans.
@@ -341,7 +347,7 @@ public class Rewriter extends AbstractBatchJobExecutor {
             ),
 
             topic("eliminate empty relation",
-                bottomUp(new EliminateEmptyRelation())
+                    bottomUp(new EliminateEmptyRelation())
             )
     );
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java
index f3b862f0c6d..b8078cdae9c 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java
@@ -133,6 +133,8 @@ public enum RuleType {
     ELIMINATE_SORT_UNDER_APPLY(RuleTypeClass.REWRITE),
     ELIMINATE_SORT_UNDER_APPLY_PROJECT(RuleTypeClass.REWRITE),
     PULL_UP_PROJECT_UNDER_APPLY(RuleTypeClass.REWRITE),
+    PULL_UP_PROJECT_UNDER_LIMIT(RuleTypeClass.REWRITE),
+    PULL_UP_PROJECT_UNDER_TOPN(RuleTypeClass.REWRITE),
     AGG_SCALAR_SUBQUERY_TO_WINDOW_FUNCTION(RuleTypeClass.REWRITE),
     UN_CORRELATED_APPLY_FILTER(RuleTypeClass.REWRITE),
     UN_CORRELATED_APPLY_PROJECT_FILTER(RuleTypeClass.REWRITE),
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderLimit.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderLimit.java
new file mode 100644
index 00000000000..a1a4ac4d219
--- /dev/null
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderLimit.java
@@ -0,0 +1,55 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+package org.apache.doris.nereids.rules.rewrite;
+
+import org.apache.doris.nereids.rules.Rule;
+import org.apache.doris.nereids.rules.RuleType;
+import org.apache.doris.nereids.trees.expressions.Slot;
+import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.trees.plans.logical.LogicalProject;
+import org.apache.doris.nereids.util.PlanUtils;
+
+import com.google.common.base.Preconditions;
+import com.google.common.collect.ImmutableList;
+
+import java.util.Set;
+import java.util.stream.Collectors;
+
+/**
+ * Pull up Project under Limit.
+ */
+public class PullUpProjectUnderLimit extends OneRewriteRuleFactory {
+    @Override
+    public Rule build() {
+        return logicalLimit(logicalProject().whenNot(p -> p.isAllSlots()))
+                .then(limit -> {
+                    LogicalProject<Plan> project = limit.child();
+                    Set<Slot> allUsedSlots = 
project.getProjects().stream().flatMap(ne -> ne.getInputSlots().stream())
+                            .collect(Collectors.toSet());
+                    Set<Slot> outputSet = project.child().getOutputSet();
+                    if (outputSet.size() == allUsedSlots.size()) {
+                        
Preconditions.checkState(outputSet.equals(allUsedSlots));
+                        return 
project.withChildren(limit.withChildren(project.child()));
+                    } else {
+                        Plan columnProject = 
PlanUtils.projectOrSelf(ImmutableList.copyOf(allUsedSlots),
+                                project.child());
+                        return 
project.withChildren(limit.withChildren(columnProject));
+                    }
+                }).toRule(RuleType.PULL_UP_PROJECT_UNDER_LIMIT);
+    }
+}
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderTopN.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderTopN.java
new file mode 100644
index 00000000000..84ac06c153d
--- /dev/null
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderTopN.java
@@ -0,0 +1,60 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+package org.apache.doris.nereids.rules.rewrite;
+
+import org.apache.doris.nereids.properties.OrderKey;
+import org.apache.doris.nereids.rules.Rule;
+import org.apache.doris.nereids.rules.RuleType;
+import org.apache.doris.nereids.trees.expressions.Slot;
+import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.trees.plans.logical.LogicalProject;
+import org.apache.doris.nereids.util.PlanUtils;
+
+import com.google.common.base.Preconditions;
+import com.google.common.collect.ImmutableList;
+
+import java.util.Set;
+import java.util.stream.Collectors;
+
+/**
+ * Pull up Project under TopN.
+ */
+public class PullUpProjectUnderTopN extends OneRewriteRuleFactory {
+    @Override
+    public Rule build() {
+        return logicalTopN(logicalProject().whenNot(p -> p.isAllSlots()))
+                .then(topN -> {
+                    LogicalProject<Plan> project = topN.child();
+                    Set<Slot> outputSet = project.child().getOutputSet();
+                    if 
(!topN.getOrderKeys().stream().map(OrderKey::getExpr).flatMap(e -> 
e.getInputSlots().stream())
+                            .allMatch(outputSet::contains)) {
+                        return null;
+                    }
+                    Set<Slot> allUsedSlots = 
project.getProjects().stream().flatMap(ne -> ne.getInputSlots().stream())
+                            .collect(Collectors.toSet());
+                    if (outputSet.size() == allUsedSlots.size()) {
+                        
Preconditions.checkState(outputSet.equals(allUsedSlots));
+                        return 
project.withChildren(topN.withChildren(project.child()));
+                    } else {
+                        Plan columnProject = 
PlanUtils.projectOrSelf(ImmutableList.copyOf(allUsedSlots),
+                                project.child());
+                        return 
project.withChildren(topN.withChildren(columnProject));
+                    }
+                }).toRule(RuleType.PULL_UP_PROJECT_UNDER_TOPN);
+    }
+}
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderLimitTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderLimitTest.java
new file mode 100644
index 00000000000..4c0be6e7a30
--- /dev/null
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderLimitTest.java
@@ -0,0 +1,60 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+package org.apache.doris.nereids.rules.rewrite;
+
+import org.apache.doris.nereids.trees.expressions.Cast;
+import org.apache.doris.nereids.trees.expressions.NamedExpression;
+import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan;
+import org.apache.doris.nereids.trees.plans.logical.LogicalPlan;
+import org.apache.doris.nereids.types.VarcharType;
+import org.apache.doris.nereids.util.LogicalPlanBuilder;
+import org.apache.doris.nereids.util.MemoPatternMatchSupported;
+import org.apache.doris.nereids.util.MemoTestUtils;
+import org.apache.doris.nereids.util.PlanChecker;
+import org.apache.doris.nereids.util.PlanConstructor;
+
+import com.google.common.collect.ImmutableList;
+import org.junit.jupiter.api.Test;
+
+import java.util.List;
+
+class PullUpProjectUnderLimitTest implements MemoPatternMatchSupported {
+    private final LogicalOlapScan scan1 = 
PlanConstructor.newLogicalOlapScan(0, "t1", 0);
+
+    @Test
+    void test() {
+        List<NamedExpression> exprs = ImmutableList.of(
+                scan1.getOutput().get(0),
+                new Cast(scan1.getOutput().get(1), 
VarcharType.SYSTEM_DEFAULT).alias("cast")
+        );
+        LogicalPlan limit = new LogicalPlanBuilder(scan1)
+                .projectExprs(exprs)
+                .limit(0, 0)
+                .build();
+
+        PlanChecker.from(MemoTestUtils.createConnectContext(), limit)
+                .applyTopDown(new PullUpProjectUnderLimit())
+                .matches(
+                        logicalProject(
+                                logicalLimit(
+                                        logicalOlapScan()
+                                )
+                        )
+                );
+    }
+}
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderTopNTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderTopNTest.java
new file mode 100644
index 00000000000..bcc6107c056
--- /dev/null
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/PullUpProjectUnderTopNTest.java
@@ -0,0 +1,60 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+package org.apache.doris.nereids.rules.rewrite;
+
+import org.apache.doris.nereids.trees.expressions.Cast;
+import org.apache.doris.nereids.trees.expressions.NamedExpression;
+import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan;
+import org.apache.doris.nereids.trees.plans.logical.LogicalPlan;
+import org.apache.doris.nereids.types.VarcharType;
+import org.apache.doris.nereids.util.LogicalPlanBuilder;
+import org.apache.doris.nereids.util.MemoPatternMatchSupported;
+import org.apache.doris.nereids.util.MemoTestUtils;
+import org.apache.doris.nereids.util.PlanChecker;
+import org.apache.doris.nereids.util.PlanConstructor;
+
+import com.google.common.collect.ImmutableList;
+import org.junit.jupiter.api.Test;
+
+import java.util.List;
+
+class PullUpProjectUnderTopNTest implements MemoPatternMatchSupported {
+    private final LogicalOlapScan scan1 = 
PlanConstructor.newLogicalOlapScan(0, "t1", 0);
+
+    @Test
+    void test() {
+        List<NamedExpression> exprs = ImmutableList.of(
+                scan1.getOutput().get(0),
+                new Cast(scan1.getOutput().get(1), 
VarcharType.SYSTEM_DEFAULT).alias("cast")
+        );
+        LogicalPlan limit = new LogicalPlanBuilder(scan1)
+                .projectExprs(exprs)
+                .topN(0, 0, ImmutableList.of(0))
+                .build();
+
+        PlanChecker.from(MemoTestUtils.createConnectContext(), limit)
+                .applyTopDown(new PullUpProjectUnderTopN())
+                .matches(
+                        logicalProject(
+                                logicalTopN(
+                                        logicalOlapScan()
+                                )
+                        )
+                );
+    }
+}
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query10.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query10.out
index 934ba291062..62c923c29b0 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query10.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query10.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_10 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query17.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query17.out
index b37e58c9331..418fc060fb3 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query17.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query17.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_17 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query27.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query27.out
index 24991d8bf4c..3ce04df2fda 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query27.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query27.out
@@ -1,37 +1,38 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_27 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
-----------hashAgg[GLOBAL]
-------------PhysicalDistribute
---------------hashAgg[LOCAL]
-----------------PhysicalRepeat
-------------------PhysicalProject
---------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_item_sk = item.i_item_sk))otherCondition=()
-----------------------PhysicalDistribute
-------------------------PhysicalProject
---------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
-----------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_store_sk = store.s_store_sk))otherCondition=()
-------------------------------PhysicalProject
---------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_cdemo_sk = 
customer_demographics.cd_demo_sk))otherCondition=()
-----------------------------------PhysicalProject
-------------------------------------PhysicalOlapScan[store_sales]
-----------------------------------PhysicalDistribute
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
+----------PhysicalProject
+------------hashAgg[GLOBAL]
+--------------PhysicalDistribute
+----------------hashAgg[LOCAL]
+------------------PhysicalRepeat
+--------------------PhysicalProject
+----------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_item_sk = item.i_item_sk))otherCondition=()
+------------------------PhysicalDistribute
+--------------------------PhysicalProject
+----------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
+------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_store_sk = store.s_store_sk))otherCondition=()
+--------------------------------PhysicalProject
+----------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_cdemo_sk = 
customer_demographics.cd_demo_sk))otherCondition=()
 ------------------------------------PhysicalProject
---------------------------------------filter((customer_demographics.cd_education_status
 = 'Secondary') and (customer_demographics.cd_gender = 'M') and 
(customer_demographics.cd_marital_status = 'W'))
-----------------------------------------PhysicalOlapScan[customer_demographics]
+--------------------------------------PhysicalOlapScan[store_sales]
+------------------------------------PhysicalDistribute
+--------------------------------------PhysicalProject
+----------------------------------------filter((customer_demographics.cd_education_status
 = 'Secondary') and (customer_demographics.cd_gender = 'M') and 
(customer_demographics.cd_marital_status = 'W'))
+------------------------------------------PhysicalOlapScan[customer_demographics]
+--------------------------------PhysicalDistribute
+----------------------------------PhysicalProject
+------------------------------------filter((store.s_state = 'TN'))
+--------------------------------------PhysicalOlapScan[store]
 ------------------------------PhysicalDistribute
 --------------------------------PhysicalProject
-----------------------------------filter((store.s_state = 'TN'))
-------------------------------------PhysicalOlapScan[store]
-----------------------------PhysicalDistribute
-------------------------------PhysicalProject
---------------------------------filter((date_dim.d_year = 1999))
-----------------------------------PhysicalOlapScan[date_dim]
-----------------------PhysicalDistribute
-------------------------PhysicalProject
---------------------------PhysicalOlapScan[item]
+----------------------------------filter((date_dim.d_year = 1999))
+------------------------------------PhysicalOlapScan[date_dim]
+------------------------PhysicalDistribute
+--------------------------PhysicalProject
+----------------------------PhysicalOlapScan[item]
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query35.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query35.out
index b946c0e966d..c4edf4f650f 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query35.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query35.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_35 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query54.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query54.out
index bd3a8b8c38d..352165e257b 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query54.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query54.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_54 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query58.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query58.out
index 945474b33ae..d0ac61aee6a 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query58.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query58.out
@@ -1,98 +1,99 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_58 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
-----------hashJoin[INNER_JOIN] hashCondition=((ss_items.item_id = 
ws_items.item_id))otherCondition=((cast(cs_item_rev as DOUBLE) <= cast((1.1 * 
ws_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) >= cast((0.9 * 
ws_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) <= cast((1.1 * 
ws_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) >= cast((0.9 * 
ws_item_rev) as DOUBLE)) and (cast(ws_item_rev as DOUBLE) <= cast((1.1 * 
cs_item_rev) as DOUBLE)) and (cast(ws_item_rev as DOUBLE [...]
-------------PhysicalProject
---------------hashAgg[GLOBAL]
-----------------PhysicalDistribute
-------------------hashAgg[LOCAL]
---------------------PhysicalProject
-----------------------hashJoin[INNER_JOIN] 
hashCondition=((web_sales.ws_item_sk = item.i_item_sk))otherCondition=()
-------------------------PhysicalDistribute
---------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_sales.ws_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
-----------------------------PhysicalProject
-------------------------------PhysicalOlapScan[web_sales]
-----------------------------PhysicalDistribute
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
+----------PhysicalProject
+------------hashJoin[INNER_JOIN] hashCondition=((ss_items.item_id = 
ws_items.item_id))otherCondition=((cast(cs_item_rev as DOUBLE) <= cast((1.1 * 
ws_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) >= cast((0.9 * 
ws_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) <= cast((1.1 * 
ws_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) >= cast((0.9 * 
ws_item_rev) as DOUBLE)) and (cast(ws_item_rev as DOUBLE) <= cast((1.1 * 
cs_item_rev) as DOUBLE)) and (cast(ws_item_rev as DOUB [...]
+--------------PhysicalProject
+----------------hashAgg[GLOBAL]
+------------------PhysicalDistribute
+--------------------hashAgg[LOCAL]
+----------------------PhysicalProject
+------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_sales.ws_item_sk = item.i_item_sk))otherCondition=()
+--------------------------PhysicalDistribute
+----------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_sales.ws_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
 ------------------------------PhysicalProject
---------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
+--------------------------------PhysicalOlapScan[web_sales]
+------------------------------PhysicalDistribute
+--------------------------------PhysicalProject
+----------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
+------------------------------------PhysicalProject
+--------------------------------------PhysicalOlapScan[date_dim]
+------------------------------------PhysicalDistribute
+--------------------------------------PhysicalProject
+----------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
+------------------------------------------PhysicalProject
+--------------------------------------------PhysicalOlapScan[date_dim]
+------------------------------------------PhysicalDistribute
+--------------------------------------------PhysicalAssertNumRows
+----------------------------------------------PhysicalDistribute
+------------------------------------------------PhysicalProject
+--------------------------------------------------filter((date_dim.d_date = 
'2001-06-16'))
+----------------------------------------------------PhysicalOlapScan[date_dim]
+--------------------------PhysicalDistribute
+----------------------------PhysicalProject
+------------------------------PhysicalOlapScan[item]
+--------------PhysicalProject
+----------------hashJoin[INNER_JOIN] hashCondition=((ss_items.item_id = 
cs_items.item_id))otherCondition=((cast(cs_item_rev as DOUBLE) <= cast((1.1 * 
ss_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) >= cast((0.9 * 
ss_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) <= cast((1.1 * 
cs_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) >= cast((0.9 * 
cs_item_rev) as DOUBLE)))
+------------------PhysicalProject
+--------------------hashAgg[GLOBAL]
+----------------------PhysicalDistribute
+------------------------hashAgg[LOCAL]
+--------------------------PhysicalProject
+----------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_item_sk = item.i_item_sk))otherCondition=()
+------------------------------PhysicalDistribute
+--------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
 ----------------------------------PhysicalProject
-------------------------------------PhysicalOlapScan[date_dim]
+------------------------------------PhysicalOlapScan[store_sales]
 ----------------------------------PhysicalDistribute
 ------------------------------------PhysicalProject
---------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
+--------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
 ----------------------------------------PhysicalProject
 ------------------------------------------PhysicalOlapScan[date_dim]
 ----------------------------------------PhysicalDistribute
-------------------------------------------PhysicalAssertNumRows
---------------------------------------------PhysicalDistribute
+------------------------------------------PhysicalProject
+--------------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
 ----------------------------------------------PhysicalProject
-------------------------------------------------filter((date_dim.d_date = 
'2001-06-16'))
---------------------------------------------------PhysicalOlapScan[date_dim]
-------------------------PhysicalDistribute
---------------------------PhysicalProject
-----------------------------PhysicalOlapScan[item]
-------------PhysicalProject
---------------hashJoin[INNER_JOIN] hashCondition=((ss_items.item_id = 
cs_items.item_id))otherCondition=((cast(cs_item_rev as DOUBLE) <= cast((1.1 * 
ss_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) >= cast((0.9 * 
ss_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) <= cast((1.1 * 
cs_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) >= cast((0.9 * 
cs_item_rev) as DOUBLE)))
-----------------PhysicalProject
-------------------hashAgg[GLOBAL]
---------------------PhysicalDistribute
-----------------------hashAgg[LOCAL]
-------------------------PhysicalProject
---------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_item_sk = item.i_item_sk))otherCondition=()
-----------------------------PhysicalDistribute
-------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
+------------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------------------------PhysicalDistribute
+------------------------------------------------PhysicalAssertNumRows
+--------------------------------------------------PhysicalDistribute
+----------------------------------------------------PhysicalProject
+------------------------------------------------------filter((date_dim.d_date 
= '2001-06-16'))
+--------------------------------------------------------PhysicalOlapScan[date_dim]
+------------------------------PhysicalDistribute
 --------------------------------PhysicalProject
-----------------------------------PhysicalOlapScan[store_sales]
---------------------------------PhysicalDistribute
+----------------------------------PhysicalOlapScan[item]
+------------------PhysicalProject
+--------------------hashAgg[GLOBAL]
+----------------------PhysicalDistribute
+------------------------hashAgg[LOCAL]
+--------------------------PhysicalProject
+----------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_sales.cs_item_sk = item.i_item_sk))otherCondition=()
+------------------------------PhysicalDistribute
+--------------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_sales.cs_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
 ----------------------------------PhysicalProject
-------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
---------------------------------------PhysicalProject
-----------------------------------------PhysicalOlapScan[date_dim]
---------------------------------------PhysicalDistribute
+------------------------------------PhysicalOlapScan[catalog_sales]
+----------------------------------PhysicalDistribute
+------------------------------------PhysicalProject
+--------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
 ----------------------------------------PhysicalProject
-------------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
---------------------------------------------PhysicalProject
-----------------------------------------------PhysicalOlapScan[date_dim]
---------------------------------------------PhysicalDistribute
-----------------------------------------------PhysicalAssertNumRows
-------------------------------------------------PhysicalDistribute
---------------------------------------------------PhysicalProject
-----------------------------------------------------filter((date_dim.d_date = 
'2001-06-16'))
-------------------------------------------------------PhysicalOlapScan[date_dim]
-----------------------------PhysicalDistribute
-------------------------------PhysicalProject
---------------------------------PhysicalOlapScan[item]
-----------------PhysicalProject
-------------------hashAgg[GLOBAL]
---------------------PhysicalDistribute
-----------------------hashAgg[LOCAL]
-------------------------PhysicalProject
---------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_sales.cs_item_sk = item.i_item_sk))otherCondition=()
-----------------------------PhysicalDistribute
-------------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_sales.cs_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
+------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------------------PhysicalDistribute
+------------------------------------------PhysicalProject
+--------------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
+----------------------------------------------PhysicalProject
+------------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------------------------PhysicalDistribute
+------------------------------------------------PhysicalAssertNumRows
+--------------------------------------------------PhysicalDistribute
+----------------------------------------------------PhysicalProject
+------------------------------------------------------filter((date_dim.d_date 
= '2001-06-16'))
+--------------------------------------------------------PhysicalOlapScan[date_dim]
+------------------------------PhysicalDistribute
 --------------------------------PhysicalProject
-----------------------------------PhysicalOlapScan[catalog_sales]
---------------------------------PhysicalDistribute
-----------------------------------PhysicalProject
-------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
---------------------------------------PhysicalProject
-----------------------------------------PhysicalOlapScan[date_dim]
---------------------------------------PhysicalDistribute
-----------------------------------------PhysicalProject
-------------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
---------------------------------------------PhysicalProject
-----------------------------------------------PhysicalOlapScan[date_dim]
---------------------------------------------PhysicalDistribute
-----------------------------------------------PhysicalAssertNumRows
-------------------------------------------------PhysicalDistribute
---------------------------------------------------PhysicalProject
-----------------------------------------------------filter((date_dim.d_date = 
'2001-06-16'))
-------------------------------------------------------PhysicalOlapScan[date_dim]
-----------------------------PhysicalDistribute
-------------------------------PhysicalProject
---------------------------------PhysicalOlapScan[item]
+----------------------------------PhysicalOlapScan[item]
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query59.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query59.out
index ba352f6d16e..83a98a113c5 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query59.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query59.out
@@ -13,40 +13,41 @@ PhysicalCteAnchor ( cteId=CTEId#0 )
 ----------------PhysicalProject
 ------------------PhysicalOlapScan[date_dim]
 --PhysicalResultSink
-----PhysicalTopN
-------PhysicalDistribute
---------PhysicalTopN
-----------PhysicalProject
-------------hashJoin[INNER_JOIN] hashCondition=((expr_cast(d_week_seq1 as 
BIGINT) = expr_(d_week_seq2 - 52)) and (y.s_store_id1 = 
x.s_store_id2))otherCondition=()
---------------PhysicalDistribute
-----------------PhysicalProject
-------------------hashJoin[INNER_JOIN] hashCondition=((wss.ss_store_sk = 
store.s_store_sk))otherCondition=()
---------------------PhysicalDistribute
-----------------------hashJoin[INNER_JOIN] hashCondition=((d.d_week_seq = 
d_week_seq1))otherCondition=()
-------------------------PhysicalDistribute
---------------------------PhysicalProject
-----------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
-------------------------PhysicalDistribute
---------------------------PhysicalProject
-----------------------------filter((d.d_month_seq <= 1206) and (d.d_month_seq 
>= 1195))
-------------------------------PhysicalOlapScan[date_dim]
---------------------PhysicalDistribute
-----------------------PhysicalProject
-------------------------PhysicalOlapScan[store]
---------------PhysicalDistribute
-----------------PhysicalProject
-------------------hashJoin[INNER_JOIN] hashCondition=((wss.ss_store_sk = 
store.s_store_sk))otherCondition=()
---------------------PhysicalDistribute
-----------------------PhysicalProject
-------------------------hashJoin[INNER_JOIN] hashCondition=((d.d_week_seq = 
d_week_seq2))otherCondition=()
+----PhysicalProject
+------PhysicalTopN
+--------PhysicalDistribute
+----------PhysicalTopN
+------------PhysicalProject
+--------------hashJoin[INNER_JOIN] hashCondition=((expr_cast(d_week_seq1 as 
BIGINT) = expr_(d_week_seq2 - 52)) and (y.s_store_id1 = 
x.s_store_id2))otherCondition=()
+----------------PhysicalDistribute
+------------------PhysicalProject
+--------------------hashJoin[INNER_JOIN] hashCondition=((wss.ss_store_sk = 
store.s_store_sk))otherCondition=()
+----------------------PhysicalDistribute
+------------------------hashJoin[INNER_JOIN] hashCondition=((d.d_week_seq = 
d_week_seq1))otherCondition=()
 --------------------------PhysicalDistribute
 ----------------------------PhysicalProject
 ------------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
 --------------------------PhysicalDistribute
 ----------------------------PhysicalProject
-------------------------------filter((d.d_month_seq <= 1218) and 
(d.d_month_seq >= 1207))
+------------------------------filter((d.d_month_seq <= 1206) and 
(d.d_month_seq >= 1195))
 --------------------------------PhysicalOlapScan[date_dim]
---------------------PhysicalDistribute
-----------------------PhysicalProject
-------------------------PhysicalOlapScan[store]
+----------------------PhysicalDistribute
+------------------------PhysicalProject
+--------------------------PhysicalOlapScan[store]
+----------------PhysicalDistribute
+------------------PhysicalProject
+--------------------hashJoin[INNER_JOIN] hashCondition=((wss.ss_store_sk = 
store.s_store_sk))otherCondition=()
+----------------------PhysicalDistribute
+------------------------PhysicalProject
+--------------------------hashJoin[INNER_JOIN] hashCondition=((d.d_week_seq = 
d_week_seq2))otherCondition=()
+----------------------------PhysicalDistribute
+------------------------------PhysicalProject
+--------------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
+----------------------------PhysicalDistribute
+------------------------------PhysicalProject
+--------------------------------filter((d.d_month_seq <= 1218) and 
(d.d_month_seq >= 1207))
+----------------------------------PhysicalOlapScan[date_dim]
+----------------------PhysicalDistribute
+------------------------PhysicalProject
+--------------------------PhysicalOlapScan[store]
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query61.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query61.out
index 56361e99201..7d61c9ee51a 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query61.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query61.out
@@ -1,9 +1,9 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_61 --
 PhysicalResultSink
---PhysicalTopN
+--PhysicalProject
 ----PhysicalTopN
-------PhysicalProject
+------PhysicalTopN
 --------NestedLoopJoin[CROSS_JOIN]
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query69.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query69.out
index 7b67283e545..165c8346e2f 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query69.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query69.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_69 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query83.out 
b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query83.out
index b86e7f8a643..4d3363e6c48 100644
--- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query83.out
+++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query83.out
@@ -1,49 +1,22 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_83 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
-----------hashJoin[INNER_JOIN] hashCondition=((sr_items.item_id = 
cr_items.item_id))otherCondition=()
-------------PhysicalProject
---------------hashAgg[GLOBAL]
-----------------PhysicalDistribute
-------------------hashAgg[LOCAL]
---------------------PhysicalProject
-----------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_returns.cr_item_sk = item.i_item_sk))otherCondition=()
-------------------------PhysicalDistribute
---------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_returns.cr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
-----------------------------PhysicalProject
-------------------------------PhysicalOlapScan[catalog_returns]
-----------------------------PhysicalDistribute
-------------------------------PhysicalProject
---------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
-----------------------------------PhysicalProject
-------------------------------------PhysicalOlapScan[date_dim]
-----------------------------------PhysicalDistribute
-------------------------------------PhysicalProject
---------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
-----------------------------------------PhysicalProject
-------------------------------------------PhysicalOlapScan[date_dim]
-----------------------------------------PhysicalDistribute
-------------------------------------------PhysicalProject
---------------------------------------------filter(d_date IN ('2001-07-13', 
'2001-09-10', '2001-11-16'))
-----------------------------------------------PhysicalOlapScan[date_dim]
-------------------------PhysicalDistribute
---------------------------PhysicalProject
-----------------------------PhysicalOlapScan[item]
-------------hashJoin[INNER_JOIN] hashCondition=((sr_items.item_id = 
wr_items.item_id))otherCondition=()
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
+----------PhysicalProject
+------------hashJoin[INNER_JOIN] hashCondition=((sr_items.item_id = 
cr_items.item_id))otherCondition=()
 --------------PhysicalProject
 ----------------hashAgg[GLOBAL]
 ------------------PhysicalDistribute
 --------------------hashAgg[LOCAL]
 ----------------------PhysicalProject
-------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_returns.sr_item_sk = item.i_item_sk))otherCondition=()
+------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_returns.cr_item_sk = item.i_item_sk))otherCondition=()
 --------------------------PhysicalDistribute
-----------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_returns.sr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
+----------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_returns.cr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
 ------------------------------PhysicalProject
---------------------------------PhysicalOlapScan[store_returns]
+--------------------------------PhysicalOlapScan[catalog_returns]
 ------------------------------PhysicalDistribute
 --------------------------------PhysicalProject
 ----------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
@@ -61,31 +34,59 @@ PhysicalResultSink
 --------------------------PhysicalDistribute
 ----------------------------PhysicalProject
 ------------------------------PhysicalOlapScan[item]
---------------PhysicalProject
-----------------hashAgg[GLOBAL]
-------------------PhysicalDistribute
---------------------hashAgg[LOCAL]
-----------------------PhysicalProject
-------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_returns.wr_item_sk = item.i_item_sk))otherCondition=()
---------------------------PhysicalDistribute
-----------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_returns.wr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
+--------------hashJoin[INNER_JOIN] hashCondition=((sr_items.item_id = 
wr_items.item_id))otherCondition=()
+----------------PhysicalProject
+------------------hashAgg[GLOBAL]
+--------------------PhysicalDistribute
+----------------------hashAgg[LOCAL]
+------------------------PhysicalProject
+--------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_returns.sr_item_sk = item.i_item_sk))otherCondition=()
+----------------------------PhysicalDistribute
+------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_returns.sr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
+--------------------------------PhysicalProject
+----------------------------------PhysicalOlapScan[store_returns]
+--------------------------------PhysicalDistribute
+----------------------------------PhysicalProject
+------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
+--------------------------------------PhysicalProject
+----------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------PhysicalDistribute
+----------------------------------------PhysicalProject
+------------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
+--------------------------------------------PhysicalProject
+----------------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------------PhysicalDistribute
+----------------------------------------------PhysicalProject
+------------------------------------------------filter(d_date IN 
('2001-07-13', '2001-09-10', '2001-11-16'))
+--------------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------PhysicalDistribute
 ------------------------------PhysicalProject
---------------------------------PhysicalOlapScan[web_returns]
-------------------------------PhysicalDistribute
+--------------------------------PhysicalOlapScan[item]
+----------------PhysicalProject
+------------------hashAgg[GLOBAL]
+--------------------PhysicalDistribute
+----------------------hashAgg[LOCAL]
+------------------------PhysicalProject
+--------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_returns.wr_item_sk = item.i_item_sk))otherCondition=()
+----------------------------PhysicalDistribute
+------------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_returns.wr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
 --------------------------------PhysicalProject
-----------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
-------------------------------------PhysicalProject
---------------------------------------PhysicalOlapScan[date_dim]
-------------------------------------PhysicalDistribute
+----------------------------------PhysicalOlapScan[web_returns]
+--------------------------------PhysicalDistribute
+----------------------------------PhysicalProject
+------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
 --------------------------------------PhysicalProject
-----------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
-------------------------------------------PhysicalProject
---------------------------------------------PhysicalOlapScan[date_dim]
-------------------------------------------PhysicalDistribute
+----------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------PhysicalDistribute
+----------------------------------------PhysicalProject
+------------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
 --------------------------------------------PhysicalProject
-----------------------------------------------filter(d_date IN ('2001-07-13', 
'2001-09-10', '2001-11-16'))
-------------------------------------------------PhysicalOlapScan[date_dim]
---------------------------PhysicalDistribute
-----------------------------PhysicalProject
-------------------------------PhysicalOlapScan[item]
+----------------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------------PhysicalDistribute
+----------------------------------------------PhysicalProject
+------------------------------------------------filter(d_date IN 
('2001-07-13', '2001-09-10', '2001-11-16'))
+--------------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------PhysicalDistribute
+------------------------------PhysicalProject
+--------------------------------PhysicalOlapScan[item]
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query10.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query10.out
index f732357c250..c4a144a5458 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query10.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query10.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_10 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query17.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query17.out
index 54d30213060..07d153a378c 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query17.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query17.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_17 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query27.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query27.out
index 17b027ff75d..caceba8398c 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query27.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query27.out
@@ -1,37 +1,38 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_27 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
-----------hashAgg[GLOBAL]
-------------PhysicalDistribute
---------------hashAgg[LOCAL]
-----------------PhysicalRepeat
-------------------PhysicalProject
---------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_store_sk = store.s_store_sk))otherCondition=()
-----------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_item_sk = item.i_item_sk))otherCondition=()
-------------------------PhysicalDistribute
---------------------------PhysicalProject
-----------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
-------------------------------PhysicalProject
---------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_cdemo_sk = 
customer_demographics.cd_demo_sk))otherCondition=()
-----------------------------------PhysicalProject
-------------------------------------PhysicalOlapScan[store_sales]
-----------------------------------PhysicalDistribute
-------------------------------------PhysicalProject
---------------------------------------filter((customer_demographics.cd_education_status
 = 'Secondary') and (customer_demographics.cd_gender = 'F') and 
(customer_demographics.cd_marital_status = 'D'))
-----------------------------------------PhysicalOlapScan[customer_demographics]
-------------------------------PhysicalDistribute
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
+----------PhysicalProject
+------------hashAgg[GLOBAL]
+--------------PhysicalDistribute
+----------------hashAgg[LOCAL]
+------------------PhysicalRepeat
+--------------------PhysicalProject
+----------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_store_sk = store.s_store_sk))otherCondition=()
+------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_item_sk = item.i_item_sk))otherCondition=()
+--------------------------PhysicalDistribute
+----------------------------PhysicalProject
+------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
 --------------------------------PhysicalProject
-----------------------------------filter((date_dim.d_year = 1999))
-------------------------------------PhysicalOlapScan[date_dim]
+----------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_cdemo_sk = 
customer_demographics.cd_demo_sk))otherCondition=()
+------------------------------------PhysicalProject
+--------------------------------------PhysicalOlapScan[store_sales]
+------------------------------------PhysicalDistribute
+--------------------------------------PhysicalProject
+----------------------------------------filter((customer_demographics.cd_education_status
 = 'Secondary') and (customer_demographics.cd_gender = 'F') and 
(customer_demographics.cd_marital_status = 'D'))
+------------------------------------------PhysicalOlapScan[customer_demographics]
+--------------------------------PhysicalDistribute
+----------------------------------PhysicalProject
+------------------------------------filter((date_dim.d_year = 1999))
+--------------------------------------PhysicalOlapScan[date_dim]
+--------------------------PhysicalDistribute
+----------------------------PhysicalProject
+------------------------------PhysicalOlapScan[item]
 ------------------------PhysicalDistribute
 --------------------------PhysicalProject
-----------------------------PhysicalOlapScan[item]
-----------------------PhysicalDistribute
-------------------------PhysicalProject
---------------------------filter(s_state IN ('AL', 'LA', 'MI', 'MO', 'SC', 
'TN'))
-----------------------------PhysicalOlapScan[store]
+----------------------------filter(s_state IN ('AL', 'LA', 'MI', 'MO', 'SC', 
'TN'))
+------------------------------PhysicalOlapScan[store]
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query35.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query35.out
index 4bf6857ee71..3455b7d1fcd 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query35.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query35.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_35 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query54.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query54.out
index dc15ec76069..b7c4c2b1c3a 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query54.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query54.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_54 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query58.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query58.out
index 4f3eab55f9f..95960c610a4 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query58.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query58.out
@@ -1,51 +1,22 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_58 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
-----------hashJoin[INNER_JOIN] hashCondition=((ss_items.item_id = 
cs_items.item_id))otherCondition=((cast(cs_item_rev as DOUBLE) <= cast((1.1 * 
ss_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) <= cast((1.1 * 
ws_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) >= cast((0.9 * 
ss_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) >= cast((0.9 * 
ws_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) <= cast((1.1 * 
cs_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE [...]
-------------PhysicalProject
---------------hashAgg[GLOBAL]
-----------------PhysicalDistribute
-------------------hashAgg[LOCAL]
---------------------PhysicalProject
-----------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_sales.cs_item_sk = item.i_item_sk))otherCondition=()
-------------------------PhysicalDistribute
---------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_sales.cs_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
-----------------------------PhysicalProject
-------------------------------PhysicalOlapScan[catalog_sales]
-----------------------------PhysicalDistribute
-------------------------------PhysicalProject
---------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
-----------------------------------PhysicalProject
-------------------------------------PhysicalOlapScan[date_dim]
-----------------------------------PhysicalDistribute
-------------------------------------PhysicalProject
---------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
-----------------------------------------PhysicalProject
-------------------------------------------PhysicalOlapScan[date_dim]
-----------------------------------------PhysicalDistribute
-------------------------------------------PhysicalAssertNumRows
---------------------------------------------PhysicalDistribute
-----------------------------------------------PhysicalProject
-------------------------------------------------filter((date_dim.d_date = 
'2001-03-24'))
---------------------------------------------------PhysicalOlapScan[date_dim]
-------------------------PhysicalDistribute
---------------------------PhysicalProject
-----------------------------PhysicalOlapScan[item]
-------------hashJoin[INNER_JOIN] hashCondition=((ss_items.item_id = 
ws_items.item_id))otherCondition=((cast(ss_item_rev as DOUBLE) <= cast((1.1 * 
ws_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) >= cast((0.9 * 
ws_item_rev) as DOUBLE)) and (cast(ws_item_rev as DOUBLE) <= cast((1.1 * 
ss_item_rev) as DOUBLE)) and (cast(ws_item_rev as DOUBLE) >= cast((0.9 * 
ss_item_rev) as DOUBLE)))
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
+----------PhysicalProject
+------------hashJoin[INNER_JOIN] hashCondition=((ss_items.item_id = 
cs_items.item_id))otherCondition=((cast(cs_item_rev as DOUBLE) <= cast((1.1 * 
ss_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) <= cast((1.1 * 
ws_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) >= cast((0.9 * 
ss_item_rev) as DOUBLE)) and (cast(cs_item_rev as DOUBLE) >= cast((0.9 * 
ws_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) <= cast((1.1 * 
cs_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUB [...]
 --------------PhysicalProject
 ----------------hashAgg[GLOBAL]
 ------------------PhysicalDistribute
 --------------------hashAgg[LOCAL]
 ----------------------PhysicalProject
-------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_sales.ws_item_sk = item.i_item_sk))otherCondition=()
+------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_sales.cs_item_sk = item.i_item_sk))otherCondition=()
 --------------------------PhysicalDistribute
-----------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_sales.ws_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
+----------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_sales.cs_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
 ------------------------------PhysicalProject
---------------------------------PhysicalOlapScan[web_sales]
+--------------------------------PhysicalOlapScan[catalog_sales]
 ------------------------------PhysicalDistribute
 --------------------------------PhysicalProject
 ----------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
@@ -65,33 +36,63 @@ PhysicalResultSink
 --------------------------PhysicalDistribute
 ----------------------------PhysicalProject
 ------------------------------PhysicalOlapScan[item]
---------------PhysicalProject
-----------------hashAgg[GLOBAL]
-------------------PhysicalDistribute
---------------------hashAgg[LOCAL]
-----------------------PhysicalProject
-------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_item_sk = item.i_item_sk))otherCondition=()
---------------------------PhysicalDistribute
-----------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
+--------------hashJoin[INNER_JOIN] hashCondition=((ss_items.item_id = 
ws_items.item_id))otherCondition=((cast(ss_item_rev as DOUBLE) <= cast((1.1 * 
ws_item_rev) as DOUBLE)) and (cast(ss_item_rev as DOUBLE) >= cast((0.9 * 
ws_item_rev) as DOUBLE)) and (cast(ws_item_rev as DOUBLE) <= cast((1.1 * 
ss_item_rev) as DOUBLE)) and (cast(ws_item_rev as DOUBLE) >= cast((0.9 * 
ss_item_rev) as DOUBLE)))
+----------------PhysicalProject
+------------------hashAgg[GLOBAL]
+--------------------PhysicalDistribute
+----------------------hashAgg[LOCAL]
+------------------------PhysicalProject
+--------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_sales.ws_item_sk = item.i_item_sk))otherCondition=()
+----------------------------PhysicalDistribute
+------------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_sales.ws_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
+--------------------------------PhysicalProject
+----------------------------------PhysicalOlapScan[web_sales]
+--------------------------------PhysicalDistribute
+----------------------------------PhysicalProject
+------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
+--------------------------------------PhysicalProject
+----------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------PhysicalDistribute
+----------------------------------------PhysicalProject
+------------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
+--------------------------------------------PhysicalProject
+----------------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------------PhysicalDistribute
+----------------------------------------------PhysicalAssertNumRows
+------------------------------------------------PhysicalDistribute
+--------------------------------------------------PhysicalProject
+----------------------------------------------------filter((date_dim.d_date = 
'2001-03-24'))
+------------------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------PhysicalDistribute
 ------------------------------PhysicalProject
---------------------------------PhysicalOlapScan[store_sales]
-------------------------------PhysicalDistribute
+--------------------------------PhysicalOlapScan[item]
+----------------PhysicalProject
+------------------hashAgg[GLOBAL]
+--------------------PhysicalDistribute
+----------------------hashAgg[LOCAL]
+------------------------PhysicalProject
+--------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_item_sk = item.i_item_sk))otherCondition=()
+----------------------------PhysicalDistribute
+------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_sales.ss_sold_date_sk = 
date_dim.d_date_sk))otherCondition=()
 --------------------------------PhysicalProject
-----------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
-------------------------------------PhysicalProject
---------------------------------------PhysicalOlapScan[date_dim]
-------------------------------------PhysicalDistribute
+----------------------------------PhysicalOlapScan[store_sales]
+--------------------------------PhysicalDistribute
+----------------------------------PhysicalProject
+------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
 --------------------------------------PhysicalProject
-----------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
-------------------------------------------PhysicalProject
---------------------------------------------PhysicalOlapScan[date_dim]
-------------------------------------------PhysicalDistribute
---------------------------------------------PhysicalAssertNumRows
-----------------------------------------------PhysicalDistribute
-------------------------------------------------PhysicalProject
---------------------------------------------------filter((date_dim.d_date = 
'2001-03-24'))
-----------------------------------------------------PhysicalOlapScan[date_dim]
---------------------------PhysicalDistribute
-----------------------------PhysicalProject
-------------------------------PhysicalOlapScan[item]
+----------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------PhysicalDistribute
+----------------------------------------PhysicalProject
+------------------------------------------hashJoin[INNER_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
+--------------------------------------------PhysicalProject
+----------------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------------PhysicalDistribute
+----------------------------------------------PhysicalAssertNumRows
+------------------------------------------------PhysicalDistribute
+--------------------------------------------------PhysicalProject
+----------------------------------------------------filter((date_dim.d_date = 
'2001-03-24'))
+------------------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------PhysicalDistribute
+------------------------------PhysicalProject
+--------------------------------PhysicalOlapScan[item]
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query59.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query59.out
index 0e4dcd07bd3..df51fbd7382 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query59.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query59.out
@@ -13,40 +13,41 @@ PhysicalCteAnchor ( cteId=CTEId#0 )
 ----------------PhysicalProject
 ------------------PhysicalOlapScan[date_dim]
 --PhysicalResultSink
-----PhysicalTopN
-------PhysicalDistribute
---------PhysicalTopN
-----------PhysicalProject
-------------hashJoin[INNER_JOIN] hashCondition=((expr_cast(d_week_seq1 as 
BIGINT) = expr_(d_week_seq2 - 52)) and (y.s_store_id1 = 
x.s_store_id2))otherCondition=()
---------------PhysicalDistribute
-----------------PhysicalProject
-------------------hashJoin[INNER_JOIN] hashCondition=((wss.ss_store_sk = 
store.s_store_sk))otherCondition=()
---------------------PhysicalDistribute
-----------------------hashJoin[INNER_JOIN] hashCondition=((d.d_week_seq = 
d_week_seq1))otherCondition=()
-------------------------PhysicalDistribute
---------------------------PhysicalProject
-----------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
-------------------------PhysicalDistribute
---------------------------PhysicalProject
-----------------------------filter((d.d_month_seq <= 1207) and (d.d_month_seq 
>= 1196))
-------------------------------PhysicalOlapScan[date_dim]
---------------------PhysicalDistribute
-----------------------PhysicalProject
-------------------------PhysicalOlapScan[store]
---------------PhysicalDistribute
-----------------PhysicalProject
-------------------hashJoin[INNER_JOIN] hashCondition=((wss.ss_store_sk = 
store.s_store_sk))otherCondition=()
---------------------PhysicalDistribute
-----------------------PhysicalProject
-------------------------hashJoin[INNER_JOIN] hashCondition=((d.d_week_seq = 
d_week_seq2))otherCondition=()
+----PhysicalProject
+------PhysicalTopN
+--------PhysicalDistribute
+----------PhysicalTopN
+------------PhysicalProject
+--------------hashJoin[INNER_JOIN] hashCondition=((expr_cast(d_week_seq1 as 
BIGINT) = expr_(d_week_seq2 - 52)) and (y.s_store_id1 = 
x.s_store_id2))otherCondition=()
+----------------PhysicalDistribute
+------------------PhysicalProject
+--------------------hashJoin[INNER_JOIN] hashCondition=((wss.ss_store_sk = 
store.s_store_sk))otherCondition=()
+----------------------PhysicalDistribute
+------------------------hashJoin[INNER_JOIN] hashCondition=((d.d_week_seq = 
d_week_seq1))otherCondition=()
 --------------------------PhysicalDistribute
 ----------------------------PhysicalProject
 ------------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
 --------------------------PhysicalDistribute
 ----------------------------PhysicalProject
-------------------------------filter((d.d_month_seq <= 1219) and 
(d.d_month_seq >= 1208))
+------------------------------filter((d.d_month_seq <= 1207) and 
(d.d_month_seq >= 1196))
 --------------------------------PhysicalOlapScan[date_dim]
---------------------PhysicalDistribute
-----------------------PhysicalProject
-------------------------PhysicalOlapScan[store]
+----------------------PhysicalDistribute
+------------------------PhysicalProject
+--------------------------PhysicalOlapScan[store]
+----------------PhysicalDistribute
+------------------PhysicalProject
+--------------------hashJoin[INNER_JOIN] hashCondition=((wss.ss_store_sk = 
store.s_store_sk))otherCondition=()
+----------------------PhysicalDistribute
+------------------------PhysicalProject
+--------------------------hashJoin[INNER_JOIN] hashCondition=((d.d_week_seq = 
d_week_seq2))otherCondition=()
+----------------------------PhysicalDistribute
+------------------------------PhysicalProject
+--------------------------------PhysicalCteConsumer ( cteId=CTEId#0 )
+----------------------------PhysicalDistribute
+------------------------------PhysicalProject
+--------------------------------filter((d.d_month_seq <= 1219) and 
(d.d_month_seq >= 1208))
+----------------------------------PhysicalOlapScan[date_dim]
+----------------------PhysicalDistribute
+------------------------PhysicalProject
+--------------------------PhysicalOlapScan[store]
 
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query61.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query61.out
index 368b6cfb780..61b94d29964 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query61.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query61.out
@@ -1,9 +1,9 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_61 --
 PhysicalResultSink
---PhysicalTopN
+--PhysicalProject
 ----PhysicalTopN
-------PhysicalProject
+------PhysicalTopN
 --------NestedLoopJoin[CROSS_JOIN]
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query69.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query69.out
index 1c43abde780..dbf40c620b7 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query69.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query69.out
@@ -1,10 +1,10 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_69 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
 ----------hashAgg[GLOBAL]
 ------------PhysicalDistribute
 --------------hashAgg[LOCAL]
diff --git 
a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query83.out 
b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query83.out
index 35b51af2346..242460d1c05 100644
--- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query83.out
+++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query83.out
@@ -1,48 +1,24 @@
 -- This file is automatically generated. You should know what you did if you 
want to edit this
 -- !ds_shape_83 --
 PhysicalResultSink
---PhysicalTopN
-----PhysicalDistribute
-------PhysicalTopN
---------PhysicalProject
-----------hashJoin[INNER_JOIN] hashCondition=((sr_items.item_id = 
cr_items.item_id))otherCondition=()
-------------PhysicalProject
---------------hashAgg[GLOBAL]
-----------------PhysicalDistribute
-------------------hashAgg[LOCAL]
---------------------PhysicalProject
-----------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_returns.cr_item_sk = item.i_item_sk))otherCondition=()
-------------------------PhysicalProject
---------------------------PhysicalOlapScan[item]
-------------------------PhysicalDistribute
---------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_returns.cr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
-----------------------------PhysicalProject
-------------------------------PhysicalOlapScan[catalog_returns]
-----------------------------PhysicalDistribute
-------------------------------PhysicalProject
---------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
-----------------------------------PhysicalProject
-------------------------------------PhysicalOlapScan[date_dim]
-----------------------------------PhysicalDistribute
-------------------------------------PhysicalProject
---------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
-----------------------------------------PhysicalProject
-------------------------------------------PhysicalOlapScan[date_dim]
-----------------------------------------PhysicalDistribute
-------------------------------------------PhysicalProject
---------------------------------------------filter(d_date IN ('2001-06-06', 
'2001-09-02', '2001-11-11'))
-----------------------------------------------PhysicalOlapScan[date_dim]
-------------hashJoin[INNER_JOIN] hashCondition=((sr_items.item_id = 
wr_items.item_id))otherCondition=()
+--PhysicalProject
+----PhysicalTopN
+------PhysicalDistribute
+--------PhysicalTopN
+----------PhysicalProject
+------------hashJoin[INNER_JOIN] hashCondition=((sr_items.item_id = 
cr_items.item_id))otherCondition=()
 --------------PhysicalProject
 ----------------hashAgg[GLOBAL]
 ------------------PhysicalDistribute
 --------------------hashAgg[LOCAL]
 ----------------------PhysicalProject
-------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_returns.sr_item_sk = item.i_item_sk))otherCondition=()
+------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_returns.cr_item_sk = item.i_item_sk))otherCondition=()
+--------------------------PhysicalProject
+----------------------------PhysicalOlapScan[item]
 --------------------------PhysicalDistribute
-----------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_returns.sr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
+----------------------------hashJoin[INNER_JOIN] 
hashCondition=((catalog_returns.cr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
 ------------------------------PhysicalProject
---------------------------------PhysicalOlapScan[store_returns]
+--------------------------------PhysicalOlapScan[catalog_returns]
 ------------------------------PhysicalDistribute
 --------------------------------PhysicalProject
 ----------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
@@ -57,33 +33,58 @@ PhysicalResultSink
 --------------------------------------------PhysicalProject
 ----------------------------------------------filter(d_date IN ('2001-06-06', 
'2001-09-02', '2001-11-11'))
 ------------------------------------------------PhysicalOlapScan[date_dim]
---------------------------PhysicalDistribute
+--------------hashJoin[INNER_JOIN] hashCondition=((sr_items.item_id = 
wr_items.item_id))otherCondition=()
+----------------PhysicalProject
+------------------hashAgg[GLOBAL]
+--------------------PhysicalDistribute
+----------------------hashAgg[LOCAL]
+------------------------PhysicalProject
+--------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_returns.sr_item_sk = item.i_item_sk))otherCondition=()
+----------------------------PhysicalDistribute
+------------------------------hashJoin[INNER_JOIN] 
hashCondition=((store_returns.sr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
+--------------------------------PhysicalProject
+----------------------------------PhysicalOlapScan[store_returns]
+--------------------------------PhysicalDistribute
+----------------------------------PhysicalProject
+------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
+--------------------------------------PhysicalProject
+----------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------PhysicalDistribute
+----------------------------------------PhysicalProject
+------------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
+--------------------------------------------PhysicalProject
+----------------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------------PhysicalDistribute
+----------------------------------------------PhysicalProject
+------------------------------------------------filter(d_date IN 
('2001-06-06', '2001-09-02', '2001-11-11'))
+--------------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------PhysicalDistribute
+------------------------------PhysicalProject
+--------------------------------PhysicalOlapScan[item]
+----------------PhysicalProject
+------------------hashAgg[GLOBAL]
+--------------------PhysicalDistribute
+----------------------hashAgg[LOCAL]
+------------------------PhysicalProject
+--------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_returns.wr_item_sk = item.i_item_sk))otherCondition=()
 ----------------------------PhysicalProject
 ------------------------------PhysicalOlapScan[item]
---------------PhysicalProject
-----------------hashAgg[GLOBAL]
-------------------PhysicalDistribute
---------------------hashAgg[LOCAL]
-----------------------PhysicalProject
-------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_returns.wr_item_sk = item.i_item_sk))otherCondition=()
---------------------------PhysicalProject
-----------------------------PhysicalOlapScan[item]
---------------------------PhysicalDistribute
-----------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_returns.wr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
-------------------------------PhysicalProject
---------------------------------PhysicalOlapScan[web_returns]
-------------------------------PhysicalDistribute
+----------------------------PhysicalDistribute
+------------------------------hashJoin[INNER_JOIN] 
hashCondition=((web_returns.wr_returned_date_sk = 
date_dim.d_date_sk))otherCondition=()
 --------------------------------PhysicalProject
-----------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
-------------------------------------PhysicalProject
---------------------------------------PhysicalOlapScan[date_dim]
-------------------------------------PhysicalDistribute
+----------------------------------PhysicalOlapScan[web_returns]
+--------------------------------PhysicalDistribute
+----------------------------------PhysicalProject
+------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_date = date_dim.d_date))otherCondition=()
 --------------------------------------PhysicalProject
-----------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
-------------------------------------------PhysicalProject
---------------------------------------------PhysicalOlapScan[date_dim]
-------------------------------------------PhysicalDistribute
+----------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------PhysicalDistribute
+----------------------------------------PhysicalProject
+------------------------------------------hashJoin[LEFT_SEMI_JOIN] 
hashCondition=((date_dim.d_week_seq = date_dim.d_week_seq))otherCondition=()
 --------------------------------------------PhysicalProject
-----------------------------------------------filter(d_date IN ('2001-06-06', 
'2001-09-02', '2001-11-11'))
-------------------------------------------------PhysicalOlapScan[date_dim]
+----------------------------------------------PhysicalOlapScan[date_dim]
+--------------------------------------------PhysicalDistribute
+----------------------------------------------PhysicalProject
+------------------------------------------------filter(d_date IN 
('2001-06-06', '2001-09-02', '2001-11-11'))
+--------------------------------------------------PhysicalOlapScan[date_dim]
 


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

Reply via email to