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

liulijia pushed a commit to branch branch-3.0
in repository https://gitbox.apache.org/repos/asf/doris.git


The following commit(s) were added to refs/heads/branch-3.0 by this push:
     new 1297e72e2d8 branch-3.0: [fix](nereids)scalar subquery should not show 
error message when there are multiple agg functions in top-level agg node 
#52667 (#57022)
1297e72e2d8 is described below

commit 1297e72e2d8dbd164c652cb9674125b6b9f11e9b
Author: Lijia Liu <[email protected]>
AuthorDate: Wed Oct 29 11:10:27 2025 +0800

    branch-3.0: [fix](nereids)scalar subquery should not show error message 
when there are multiple agg functions in top-level agg node #52667 (#57022)
    
    pick from master #52667
    
    ---------
    
    Co-authored-by: starocean999 <[email protected]>
    Co-authored-by: liulijia <[email protected]>
---
 .../nereids/rules/analysis/SubqueryToApply.java    | 218 ++++++++++++---------
 .../rewrite/UnCorrelatedApplyAggregateFilter.java  |   3 +-
 .../rules/rewrite/UnCorrelatedApplyFilter.java     |   2 +-
 .../rewrite/UnCorrelatedApplyProjectFilter.java    |   2 +-
 .../trees/copier/LogicalPlanDeepCopier.java        |   2 +-
 .../nereids/trees/expressions/ScalarSubquery.java  |  30 +--
 .../nereids/trees/plans/logical/LogicalApply.java  |  22 +--
 .../rules/analysis/SubqueryToApplyTest.java        |  67 +++++++
 .../rules/rewrite/ExistsApplyToJoinTest.java       |   8 +-
 .../subquery/correlated_scalar_subquery.out        |  23 +++
 .../subquery/correlated_scalar_subquery.groovy     |   5 +
 11 files changed, 254 insertions(+), 128 deletions(-)

diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/SubqueryToApply.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/SubqueryToApply.java
index 27e6f446682..021c4e72064 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/SubqueryToApply.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/SubqueryToApply.java
@@ -27,7 +27,6 @@ import 
org.apache.doris.nereids.rules.expression.ExpressionRewriteContext;
 import 
org.apache.doris.nereids.rules.expression.rules.TrySimplifyPredicateWithMarkJoinSlot;
 import org.apache.doris.nereids.trees.TreeNode;
 import org.apache.doris.nereids.trees.expressions.Alias;
-import org.apache.doris.nereids.trees.expressions.And;
 import org.apache.doris.nereids.trees.expressions.BinaryOperator;
 import org.apache.doris.nereids.trees.expressions.Exists;
 import org.apache.doris.nereids.trees.expressions.Expression;
@@ -37,7 +36,6 @@ import 
org.apache.doris.nereids.trees.expressions.LessThanEqual;
 import org.apache.doris.nereids.trees.expressions.ListQuery;
 import org.apache.doris.nereids.trees.expressions.MarkJoinSlotReference;
 import org.apache.doris.nereids.trees.expressions.NamedExpression;
-import org.apache.doris.nereids.trees.expressions.Not;
 import org.apache.doris.nereids.trees.expressions.Or;
 import org.apache.doris.nereids.trees.expressions.ScalarSubquery;
 import org.apache.doris.nereids.trees.expressions.Slot;
@@ -64,6 +62,7 @@ import 
org.apache.doris.nereids.trees.plans.logical.LogicalSort;
 import org.apache.doris.nereids.util.ExpressionUtils;
 import org.apache.doris.nereids.util.Utils;
 
+import com.google.common.annotations.VisibleForTesting;
 import com.google.common.base.Preconditions;
 import com.google.common.collect.ImmutableList;
 import com.google.common.collect.ImmutableSet;
@@ -140,7 +139,7 @@ public class SubqueryToApply implements AnalysisRuleFactory 
{
                                     .collect(ImmutableList.toImmutableList()), 
tmpPlan,
                                 context.getSubqueryToMarkJoinSlot(),
                                 ctx.cascadesContext,
-                                Optional.of(conjunct), false, 
isMarkSlotNotNull);
+                                Optional.of(conjunct), isMarkSlotNotNull);
                         applyPlan = result.first;
                         tmpPlan = applyPlan;
                         newConjuncts.add(result.second.isPresent() ? 
result.second.get() : conjunct);
@@ -182,7 +181,7 @@ public class SubqueryToApply implements AnalysisRuleFactory 
{
                     Pair<LogicalPlan, Optional<Expression>> result =
                             
subqueryToApply(Utils.fastToImmutableList(subqueryExprs), childPlan,
                                     context.getSubqueryToMarkJoinSlot(), 
ctx.cascadesContext,
-                                    Optional.of(newProject), true, false);
+                                    Optional.of(newProject), false);
                     applyPlan = result.first;
                     childPlan = applyPlan;
                     newProjects.add(
@@ -266,7 +265,7 @@ public class SubqueryToApply implements AnalysisRuleFactory 
{
                                 
subqueryExprs.stream().collect(ImmutableList.toImmutableList()),
                                 relatedInfoList.get(i) == 
RelatedInfo.RelatedToLeft ? leftChildPlan : rightChildPlan,
                                 context.getSubqueryToMarkJoinSlot(),
-                                ctx.cascadesContext, Optional.of(conjunct), 
false, isMarkSlotNotNull);
+                                ctx.cascadesContext, Optional.of(conjunct), 
isMarkSlotNotNull);
                         applyPlan = result.first;
                         if (relatedInfoList.get(i) == 
RelatedInfo.RelatedToLeft) {
                             leftChildPlan = applyPlan;
@@ -368,22 +367,20 @@ public class SubqueryToApply implements 
AnalysisRuleFactory {
     private Pair<LogicalPlan, Optional<Expression>> subqueryToApply(
             List<SubqueryExpr> subqueryExprs, LogicalPlan childPlan,
             Map<SubqueryExpr, Optional<MarkJoinSlotReference>> 
subqueryToMarkJoinSlot,
-            CascadesContext ctx, Optional<Expression> conjunct, boolean 
isProject,
-            boolean isMarkJoinSlotNotNull) {
-        Pair<LogicalPlan, Optional<Expression>> tmpPlan = Pair.of(childPlan, 
conjunct);
+            CascadesContext ctx, Optional<Expression> correlatedOuterExpr, 
boolean isMarkJoinSlotNotNull) {
+        Pair<LogicalPlan, Optional<Expression>> tmpPlan = Pair.of(childPlan, 
correlatedOuterExpr);
         for (int i = 0; i < subqueryExprs.size(); ++i) {
             SubqueryExpr subqueryExpr = subqueryExprs.get(i);
             if (subqueryExpr instanceof Exists && 
hasTopLevelScalarAgg(subqueryExpr.getQueryPlan())) {
                 // because top level scalar agg always returns a value or 
null(for empty input)
-                // so Exists and Not Exists conjunct are always evaluated to 
True and false literals respectively
-                // we don't create apply node for it
+                // so Exists and Not Exists correlatedOuterExpr are always 
evaluated to
+                // True and false literals respectively, we don't create apply 
node for it
                 continue;
             }
 
             if (!ctx.subqueryIsAnalyzed(subqueryExpr)) {
                 tmpPlan = addApply(subqueryExpr, tmpPlan.first,
-                    subqueryToMarkJoinSlot, ctx, tmpPlan.second,
-                    isProject, subqueryExprs.size() == 1, 
isMarkJoinSlotNotNull);
+                    subqueryToMarkJoinSlot, ctx, tmpPlan.second, 
isMarkJoinSlotNotNull);
             }
         }
         return tmpPlan;
@@ -398,22 +395,30 @@ public class SubqueryToApply implements 
AnalysisRuleFactory {
         return false;
     }
 
-    private Pair<LogicalPlan, Optional<Expression>> addApply(SubqueryExpr 
subquery,
-            LogicalPlan childPlan,
+    private Pair<LogicalPlan, Optional<Expression>> addApply(SubqueryExpr 
subquery, LogicalPlan childPlan,
             Map<SubqueryExpr, Optional<MarkJoinSlotReference>> 
subqueryToMarkJoinSlot,
-            CascadesContext ctx, Optional<Expression> conjunct, boolean 
isProject,
-            boolean singleSubquery, boolean isMarkJoinSlotNotNull) {
+            CascadesContext ctx, Optional<Expression> correlatedOuterExpr, 
boolean isMarkJoinSlotNotNull) {
         ctx.setSubqueryExprIsAnalyzed(subquery, true);
         Optional<MarkJoinSlotReference> markJoinSlot = 
subqueryToMarkJoinSlot.get(subquery);
-        boolean needAddScalarSubqueryOutputToProjects = 
isConjunctContainsScalarSubqueryOutput(
-                subquery, conjunct, isProject, singleSubquery);
+
+        boolean needAddScalarSubqueryOutputToProjects = 
isScalarSubqueryOutputUsedInOuterScope(
+                subquery, correlatedOuterExpr);
         boolean needRuntimeAssertCount = false;
-        NamedExpression oldSubqueryOutput = 
subquery.getQueryPlan().getOutput().get(0);
+        // In #50256, needRuntimeAssertCount has been replaced by 
needRuntimeAnyValue
+        // for scalar subquery, we need ensure it output at most 1 row
+        // by doing that, we add an aggregate function any_value() to the 
project list
+        // we use needRuntimeAnyValue to indicate if any_value() is needed
+        // if needRuntimeAnyValue is true, we will add it to the project list
+        // boolean needRuntimeAnyValue = false;
+        NamedExpression subqueryOutput = 
subquery.getQueryPlan().getOutput().get(0);
+        // if (subquery instanceof ScalarSubquery) {  // #51928
+        //     // scalar sub query may adjust output slot's nullable.
+        //     subqueryOutput = ((ScalarSubquery) 
subquery).getOutputSlotAdjustNullable();
+        // }
         Slot countSlot = null;
         Slot anyValueSlot = null;
-        Optional<Expression> newConjunct = conjunct;
-        if (needAddScalarSubqueryOutputToProjects && subquery instanceof 
ScalarSubquery
-                && !subquery.getCorrelateSlots().isEmpty()) {
+        Optional<Expression> newCorrelatedOuterExpr = correlatedOuterExpr;
+        if (needAddScalarSubqueryOutputToProjects && 
!subquery.getCorrelateSlots().isEmpty()) {
             if (((ScalarSubquery) subquery).hasTopLevelScalarAgg()) {
                 // consider sql: SELECT * FROM t1 WHERE t1.a <= (SELECT 
COUNT(t2.a) FROM t2 WHERE (t1.b = t2.b));
                 // when unnest correlated subquery, we create a left join node.
@@ -421,36 +426,14 @@ public class SubqueryToApply implements 
AnalysisRuleFactory {
                 // if there is no match, the row from right table is filled 
with nulls
                 // but COUNT function is always not nullable.
                 // so wrap COUNT with Nvl to ensure its result is 0 instead of 
null to get the correct result
-                if (conjunct.isPresent()) {
-                    Map<Expression, Expression> replaceMap = new HashMap<>();
-                    NamedExpression agg = ((ScalarSubquery) 
subquery).getTopLevelScalarAggFunction().get();
-                    if (agg instanceof Alias) {
-                        if (((Alias) agg).child() instanceof 
NotNullableAggregateFunction) {
-                            NotNullableAggregateFunction notNullableAggFunc =
-                                    (NotNullableAggregateFunction) ((Alias) 
agg).child();
-                            if (subquery.getQueryPlan() instanceof 
LogicalProject) {
-                                LogicalProject logicalProject =
-                                        (LogicalProject) 
subquery.getQueryPlan();
-                                
Preconditions.checkState(logicalProject.getOutputs().size() == 1,
-                                        "Scalar subuqery's should only output 
1 column");
-                                Slot aggSlot = agg.toSlot();
-                                replaceMap.put(aggSlot, new Alias(new 
Nvl(aggSlot,
-                                        
notNullableAggFunc.resultForEmptyInput())));
-                                NamedExpression newOutput = (NamedExpression) 
ExpressionUtils
-                                        .replace((NamedExpression) 
logicalProject.getProjects().get(0), replaceMap);
-                                replaceMap.clear();
-                                replaceMap.put(oldSubqueryOutput, 
newOutput.toSlot());
-                                oldSubqueryOutput = newOutput;
-                                subquery = subquery.withSubquery((LogicalPlan) 
logicalProject.child());
-                            } else {
-                                replaceMap.put(oldSubqueryOutput, new 
Nvl(oldSubqueryOutput,
-                                        
notNullableAggFunc.resultForEmptyInput()));
-                            }
-                        }
-                        if (!replaceMap.isEmpty()) {
-                            newConjunct = 
Optional.of(ExpressionUtils.replace(conjunct.get(), replaceMap));
-                        }
-                    }
+                if (correlatedOuterExpr.isPresent()) {
+                    List<NamedExpression> aggFunctions = 
ScalarSubquery.getTopLevelScalarAggFunctions(
+                            subquery.getQueryPlan(), 
subquery.getCorrelateSlots());
+                    SubQueryRewriteResult result = 
addNvlForScalarSubqueryOutput(aggFunctions, subqueryOutput,
+                            subquery, correlatedOuterExpr);
+                    subqueryOutput = result.subqueryOutput;
+                    subquery = result.subquery;
+                    newCorrelatedOuterExpr = result.correlatedOuterExpr;
                 }
             } else {
                 // if scalar subquery doesn't have top level scalar agg we 
will create one, for example
@@ -458,16 +441,17 @@ public class SubqueryToApply implements 
AnalysisRuleFactory {
                 // the original output of the correlate subquery is t2.c1, 
after adding a scalar agg, it will be
                 // select (select count(*), any_value(t2.c1) from t2 where 
t2.c2 = t1.c2) from t1;
                 Alias countAlias = new Alias(new Count());
-                Alias anyValueAlias = new Alias(new 
AnyValue(oldSubqueryOutput));
+                Alias anyValueAlias = new Alias(new AnyValue(subqueryOutput));
                 LogicalAggregate<Plan> aggregate = new 
LogicalAggregate<>(ImmutableList.of(),
                         ImmutableList.of(countAlias, anyValueAlias), 
subquery.getQueryPlan());
                 countSlot = countAlias.toSlot();
                 anyValueSlot = anyValueAlias.toSlot();
                 subquery = subquery.withSubquery(aggregate);
-                if (conjunct.isPresent()) {
+                if (correlatedOuterExpr.isPresent()) {
                     Map<Expression, Expression> replaceMap = new HashMap<>();
-                    replaceMap.put(oldSubqueryOutput, anyValueSlot);
-                    newConjunct = 
Optional.of(ExpressionUtils.replace(conjunct.get(), replaceMap));
+                    replaceMap.put(subqueryOutput, anyValueSlot);
+                    newCorrelatedOuterExpr = 
Optional.of(ExpressionUtils.replace(correlatedOuterExpr.get(),
+                            replaceMap));
                 }
                 needRuntimeAssertCount = true;
             }
@@ -491,7 +475,7 @@ public class SubqueryToApply implements AnalysisRuleFactory 
{
                 subquery.getCorrelateSlots(),
                 subQueryType, isNot, compareExpr, 
subquery.getTypeCoercionExpr(), Optional.empty(),
                 markJoinSlot,
-                needAddScalarSubqueryOutputToProjects, isProject, 
isMarkJoinSlotNotNull,
+                needAddScalarSubqueryOutputToProjects, isMarkJoinSlotNotNull,
                 childPlan, subquery.getQueryPlan());
 
         ImmutableList.Builder<NamedExpression> projects =
@@ -510,19 +494,101 @@ public class SubqueryToApply implements 
AnalysisRuleFactory {
                                 new LessThanEqual(countSlot, new 
IntegerLiteral(1))),
                         new VarcharLiteral("correlate scalar subquery must 
return only 1 row"))));
             } else {
-                projects.add(oldSubqueryOutput);
+                projects.add(subqueryOutput);
             }
         }
 
-        return Pair.of(new LogicalProject(projects.build(), newApply), 
newConjunct);
+        return Pair.of(new LogicalProject(projects.build(), newApply), 
newCorrelatedOuterExpr);
+    }
+
+    /**
+     * SubQueryRewriteResult
+     */
+    @VisibleForTesting
+    protected class SubQueryRewriteResult {
+        public SubqueryExpr subquery;
+        public NamedExpression subqueryOutput;
+        public Optional<Expression> correlatedOuterExpr;
+
+        public SubQueryRewriteResult(SubqueryExpr subquery, NamedExpression 
subqueryOutput,
+                                     Optional<Expression> correlatedOuterExpr) 
{
+            this.subquery = subquery;
+            this.subqueryOutput = subqueryOutput;
+            this.correlatedOuterExpr = correlatedOuterExpr;
+        }
     }
 
-    private boolean isConjunctContainsScalarSubqueryOutput(
-            SubqueryExpr subqueryExpr, Optional<Expression> conjunct, boolean 
isProject, boolean singleSubquery) {
+    /**
+     * for correlated scalar subquery like select c1, (select count(c1) from 
t2 where t1.c2 = t2.c2) as c from t1
+     * if we don't add extra nvl for not nullable agg functions, the plan will 
be like bellow:
+     * +--LogicalProject(projects=[c1#0, count(c1)#4 AS `c`#5])
+     *    +--LogicalJoin(type=LEFT_OUTER_JOIN, hashJoinConjuncts=[(c2#1 = 
c2#3)])
+     *       |--LogicalOlapScan (t1)
+     *       +--LogicalAggregate[108] (groupByExpr=[c2#3], outputExpr=[c2#3, 
count(c1#2) AS `count(c1)`#4])
+     *          +--LogicalOlapScan (t2)
+     *
+     * the count(c1)#4 may be null because of unmatched row of left outer 
join, but count is not nullable agg function,
+     * it should never be null, we need use nvl to wrap it and change the plan 
like bellow:
+     * +--LogicalProject(projects=[c1#0, ifnull(count(c1)#4, 0) AS `c`#5])
+     *    +--LogicalJoin(type=LEFT_OUTER_JOIN, hashJoinConjuncts=[(c2#1 = 
c2#3)])
+     *       |--LogicalOlapScan (t1)
+     *       +--LogicalAggregate[108] (groupByExpr=[c2#3], outputExpr=[c2#3, 
count(c1#2) AS `count(c1)`#4])
+     *          +--LogicalOlapScan (t2)
+     *
+     * in order to do that, we need change subquery's output and replace the 
correlated outer expr
+     */
+    @VisibleForTesting
+    protected SubQueryRewriteResult 
addNvlForScalarSubqueryOutput(List<NamedExpression> aggFunctions,
+                                                                
NamedExpression subqueryOutput,
+                                                                SubqueryExpr 
subquery,
+                                                                
Optional<Expression> correlatedOuterExpr) {
+        SubQueryRewriteResult result = new SubQueryRewriteResult(subquery, 
subqueryOutput, correlatedOuterExpr);
+        Map<Expression, Expression> replaceMapForSubqueryProject = new 
HashMap<>();
+        Map<Expression, Expression> replaceMapForCorrelatedOuterExpr = new 
HashMap<>();
+        for (NamedExpression agg : aggFunctions) {
+            if (agg instanceof Alias && ((Alias) agg).child() instanceof 
NotNullableAggregateFunction) {
+                NotNullableAggregateFunction notNullableAggFunc =
+                        (NotNullableAggregateFunction) ((Alias) agg).child();
+                if (subquery.getQueryPlan() instanceof LogicalProject) {
+                    // if the top node of subquery is LogicalProject, we need 
replace the agg slot in
+                    // project list by nvl(agg), and this project will be 
placed above LogicalApply node
+                    Slot aggSlot = agg.toSlot();
+                    replaceMapForSubqueryProject.put(aggSlot, new Alias(new 
Nvl(aggSlot,
+                            notNullableAggFunc.resultForEmptyInput())));
+                } else {
+                    replaceMapForCorrelatedOuterExpr.put(subqueryOutput, new 
Nvl(subqueryOutput,
+                            notNullableAggFunc.resultForEmptyInput()));
+                }
+            }
+        }
+        if (!replaceMapForSubqueryProject.isEmpty()) {
+            Preconditions.checkState(subquery.getQueryPlan() instanceof 
LogicalProject,
+                    "Scalar subquery's top plan node should be 
LogicalProject");
+            LogicalProject logicalProject =
+                    (LogicalProject) subquery.getQueryPlan();
+            Preconditions.checkState(logicalProject.getOutputs().size() == 1,
+                    "Scalar subuqery's should only output 1 column");
+            NamedExpression newOutput = (NamedExpression) ExpressionUtils
+                    .replace((NamedExpression) 
logicalProject.getProjects().get(0),
+                            replaceMapForSubqueryProject);
+            replaceMapForCorrelatedOuterExpr.put(subqueryOutput, 
newOutput.toSlot());
+            result.subqueryOutput = newOutput;
+            // logicalProject will be placed above LogicalApply later, so we 
remove it from subquery
+            result.subquery = subquery.withSubquery((LogicalPlan) 
logicalProject.child());
+        }
+        if (!replaceMapForCorrelatedOuterExpr.isEmpty()) {
+            result.correlatedOuterExpr = 
Optional.of(ExpressionUtils.replace(correlatedOuterExpr.get(),
+                    replaceMapForCorrelatedOuterExpr));
+        }
+        return result;
+    }
+
+    private boolean isScalarSubqueryOutputUsedInOuterScope(
+            SubqueryExpr subqueryExpr, Optional<Expression> 
correlatedOuterExpr) {
         return subqueryExpr instanceof ScalarSubquery
-            && ((conjunct.isPresent() && ((ImmutableSet) 
conjunct.get().collect(SlotReference.class::isInstance))
-                    .contains(subqueryExpr.getQueryPlan().getOutput().get(0)))
-                || isProject);
+            && ((correlatedOuterExpr.isPresent()
+                && ((ImmutableSet) 
correlatedOuterExpr.get().collect(SlotReference.class::isInstance))
+                    
.contains(subqueryExpr.getQueryPlan().getOutput().get(0))));
     }
 
     /**
@@ -653,30 +719,6 @@ public class SubqueryToApply implements 
AnalysisRuleFactory {
 
     }
 
-    private enum SearchState {
-        SearchNot,
-        SearchAnd,
-        SearchExistsOrInSubquery
-    }
-
-    private boolean shouldOutputMarkJoinSlot(Expression expr, SearchState 
searchState) {
-        if (searchState == SearchState.SearchNot && expr instanceof Not) {
-            if (shouldOutputMarkJoinSlot(((Not) expr).child(), 
SearchState.SearchAnd)) {
-                return true;
-            }
-        } else if (searchState == SearchState.SearchAnd && expr instanceof 
And) {
-            for (Expression child : expr.children()) {
-                if (shouldOutputMarkJoinSlot(child, 
SearchState.SearchExistsOrInSubquery)) {
-                    return true;
-                }
-            }
-        } else if (searchState == SearchState.SearchExistsOrInSubquery
-                && (expr instanceof InSubquery || expr instanceof Exists)) {
-            return true;
-        }
-        return false;
-    }
-
     private List<Boolean> shouldOutputMarkJoinSlot(Collection<Expression> 
conjuncts) {
         ImmutableList.Builder<Boolean> result = 
ImmutableList.builderWithExpectedSize(conjuncts.size());
         for (Expression expr : conjuncts) {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyAggregateFilter.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyAggregateFilter.java
index 258698b1f7f..ddc6a17b171 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyAggregateFilter.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyAggregateFilter.java
@@ -118,8 +118,7 @@ public class UnCorrelatedApplyAggregateFilter implements 
RewriteRuleFactory {
         return new LogicalApply<>(apply.getCorrelationSlot(), 
apply.getSubqueryType(), apply.isNot(),
                 apply.getCompareExpr(), apply.getTypeCoercionExpr(),
                 ExpressionUtils.optionalAnd(correlatedPredicate), 
apply.getMarkJoinSlotReference(),
-                apply.isNeedAddSubOutputToProjects(), apply.isInProject(),
-                apply.isMarkJoinSlotNotNull(), apply.left(),
+                apply.isNeedAddSubOutputToProjects(), 
apply.isMarkJoinSlotNotNull(), apply.left(),
                 isRightChildAgg ? newAgg : apply.right().withChildren(newAgg));
     }
 }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyFilter.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyFilter.java
index b5732a604ca..10bdf2f0046 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyFilter.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyFilter.java
@@ -70,7 +70,7 @@ public class UnCorrelatedApplyFilter extends 
OneRewriteRuleFactory {
                     apply.getCompareExpr(), apply.getTypeCoercionExpr(),
                     ExpressionUtils.optionalAnd(correlatedPredicate), 
apply.getMarkJoinSlotReference(),
                     apply.isNeedAddSubOutputToProjects(),
-                    apply.isInProject(), apply.isMarkJoinSlotNotNull(), 
apply.left(), child);
+                    apply.isMarkJoinSlotNotNull(), apply.left(), child);
         }).toRule(RuleType.UN_CORRELATED_APPLY_FILTER);
     }
 }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyProjectFilter.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyProjectFilter.java
index 4f31d672a16..ceced5b6143 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyProjectFilter.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/UnCorrelatedApplyProjectFilter.java
@@ -91,7 +91,7 @@ public class UnCorrelatedApplyProjectFilter extends 
OneRewriteRuleFactory {
                             apply.getCompareExpr(), 
apply.getTypeCoercionExpr(),
                             ExpressionUtils.optionalAnd(correlatedPredicate), 
apply.getMarkJoinSlotReference(),
                             apply.isNeedAddSubOutputToProjects(),
-                            apply.isInProject(), 
apply.isMarkJoinSlotNotNull(), apply.left(), newProject);
+                            apply.isMarkJoinSlotNotNull(), apply.left(), 
newProject);
                 }).toRule(RuleType.UN_CORRELATED_APPLY_PROJECT_FILTER);
     }
 }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/copier/LogicalPlanDeepCopier.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/copier/LogicalPlanDeepCopier.java
index e7b755a6d53..17c4bc7a514 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/copier/LogicalPlanDeepCopier.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/copier/LogicalPlanDeepCopier.java
@@ -138,7 +138,7 @@ public class LogicalPlanDeepCopier extends 
DefaultPlanRewriter<DeepCopierContext
                 .map(m -> (MarkJoinSlotReference) 
ExpressionDeepCopier.INSTANCE.deepCopy(m, context));
         return new LogicalApply<>(correlationSlot, apply.getSubqueryType(), 
apply.isNot(),
                 compareExpr, typeCoercionExpr, correlationFilter,
-                markJoinSlotReference, apply.isNeedAddSubOutputToProjects(), 
apply.isInProject(),
+                markJoinSlotReference, apply.isNeedAddSubOutputToProjects(),
                 apply.isMarkJoinSlotNotNull(), left, right);
     }
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/ScalarSubquery.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/ScalarSubquery.java
index 25a7052a4ac..af337c5b18a 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/ScalarSubquery.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/ScalarSubquery.java
@@ -31,6 +31,7 @@ import com.google.common.base.Preconditions;
 import com.google.common.collect.ImmutableList;
 import com.google.common.collect.ImmutableSet;
 
+import java.util.ArrayList;
 import java.util.List;
 import java.util.Objects;
 import java.util.Optional;
@@ -62,18 +63,15 @@ public class ScalarSubquery extends SubqueryExpr {
     }
 
     /**
-    * getTopLevelScalarAggFunction
-    */
-    public Optional<NamedExpression> getTopLevelScalarAggFunction() {
-        Plan plan = findTopLevelScalarAgg(queryPlan, 
ImmutableSet.copyOf(correlateSlots));
-        if (plan != null) {
-            LogicalAggregate aggregate = (LogicalAggregate) plan;
-            Preconditions.checkState(aggregate.getAggregateFunctions().size() 
== 1,
-                    "in scalar subquery, should only return 1 column 1 row, "
-                            + "but we found multiple columns ", 
aggregate.getOutputExpressions());
-            return Optional.of((NamedExpression) 
aggregate.getOutputExpressions().get(0));
+     * get Top Level ScalarAgg Functions
+     */
+    public static List<NamedExpression> getTopLevelScalarAggFunctions(Plan 
queryPlan,
+            List<Slot> correlateSlots) {
+        LogicalAggregate<?> aggregate = findTopLevelScalarAgg(queryPlan, 
ImmutableSet.copyOf(correlateSlots));
+        if (aggregate != null) {
+            return aggregate.getOutputExpressions();
         } else {
-            return Optional.empty();
+            return new ArrayList<>();
         }
     }
 
@@ -115,17 +113,19 @@ public class ScalarSubquery extends SubqueryExpr {
      * 1. The agg or its child contains correlated slots
      * 2. only project, sort and subquery alias node can be agg's parent
      */
-    public static Plan findTopLevelScalarAgg(Plan plan, ImmutableSet<Slot> 
slots) {
+    public static LogicalAggregate<?> findTopLevelScalarAgg(Plan plan, 
ImmutableSet<Slot> slots) {
         if (plan instanceof LogicalAggregate) {
-            if (((LogicalAggregate<?>) plan).getGroupByExpressions().isEmpty() 
&& plan.containsSlots(slots)) {
-                return plan;
+            LogicalAggregate<?> agg = (LogicalAggregate<?>) plan;
+            if (agg.getGroupByExpressions().isEmpty()
+                    && (plan.containsSlots(slots) || slots.isEmpty())) {
+                return agg;
             } else {
                 return null;
             }
         } else if (plan instanceof LogicalProject || plan instanceof 
LogicalSubQueryAlias
                 || plan instanceof LogicalSort) {
             for (Plan child : plan.children()) {
-                Plan result = findTopLevelScalarAgg(child, slots);
+                LogicalAggregate<?> result = findTopLevelScalarAgg(child, 
slots);
                 if (result != null) {
                     return result;
                 }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalApply.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalApply.java
index 0b12e225311..0a2a2adc894 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalApply.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalApply.java
@@ -67,9 +67,6 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
     // The slot replaced by the subquery in MarkJoin
     private final Optional<MarkJoinSlotReference> markJoinSlotReference;
 
-    // Whether the subquery is in logicalProject
-    private final boolean inProject;
-
     // Whether adding the subquery's output to projects
     private final boolean needAddSubOutputToProjects;
 
@@ -88,7 +85,6 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
             Optional<Expression> correlationFilter,
             Optional<MarkJoinSlotReference> markJoinSlotReference,
             boolean needAddSubOutputToProjects,
-            boolean inProject,
             boolean isMarkJoinSlotNotNull,
             LEFT_CHILD_TYPE leftChild, RIGHT_CHILD_TYPE rightChild) {
         super(PlanType.LOGICAL_APPLY, groupExpression, logicalProperties, 
leftChild, rightChild);
@@ -103,17 +99,16 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
         this.correlationFilter = correlationFilter;
         this.markJoinSlotReference = markJoinSlotReference;
         this.needAddSubOutputToProjects = needAddSubOutputToProjects;
-        this.inProject = inProject;
         this.isMarkJoinSlotNotNull = isMarkJoinSlotNotNull;
     }
 
     public LogicalApply(List<Expression> correlationSlot, SubQueryType 
subqueryType, boolean isNot,
             Optional<Expression> compareExpr, Optional<Expression> 
typeCoercionExpr,
             Optional<Expression> correlationFilter, 
Optional<MarkJoinSlotReference> markJoinSlotReference,
-            boolean needAddSubOutputToProjects, boolean inProject, boolean 
isMarkJoinSlotNotNull,
+            boolean needAddSubOutputToProjects, boolean isMarkJoinSlotNotNull,
             LEFT_CHILD_TYPE input, RIGHT_CHILD_TYPE subquery) {
         this(Optional.empty(), Optional.empty(), correlationSlot, 
subqueryType, isNot, compareExpr, typeCoercionExpr,
-                correlationFilter, markJoinSlotReference, 
needAddSubOutputToProjects, inProject, isMarkJoinSlotNotNull,
+                correlationFilter, markJoinSlotReference, 
needAddSubOutputToProjects, isMarkJoinSlotNotNull,
                 input, subquery);
     }
 
@@ -177,10 +172,6 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
         return needAddSubOutputToProjects;
     }
 
-    public boolean isInProject() {
-        return inProject;
-    }
-
     public boolean isMarkJoinSlotNotNull() {
         return isMarkJoinSlotNotNull;
     }
@@ -222,7 +213,6 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
                 && Objects.equals(correlationFilter, 
that.getCorrelationFilter())
                 && Objects.equals(markJoinSlotReference, 
that.getMarkJoinSlotReference())
                 && needAddSubOutputToProjects == 
that.needAddSubOutputToProjects
-                && inProject == that.inProject
                 && isMarkJoinSlotNotNull == that.isMarkJoinSlotNotNull
                 && isNot == that.isNot;
     }
@@ -231,7 +221,7 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
     public int hashCode() {
         return Objects.hash(
                 correlationSlot, subqueryType, compareExpr, typeCoercionExpr, 
correlationFilter,
-                markJoinSlotReference, needAddSubOutputToProjects, inProject, 
isMarkJoinSlotNotNull, isNot);
+                markJoinSlotReference, needAddSubOutputToProjects, 
isMarkJoinSlotNotNull, isNot);
     }
 
     @Override
@@ -257,7 +247,7 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
     public LogicalApply<Plan, Plan> withChildren(List<Plan> children) {
         Preconditions.checkArgument(children.size() == 2);
         return new LogicalApply<>(correlationSlot, subqueryType, isNot, 
compareExpr, typeCoercionExpr,
-                correlationFilter, markJoinSlotReference, 
needAddSubOutputToProjects, inProject, isMarkJoinSlotNotNull,
+                correlationFilter, markJoinSlotReference, 
needAddSubOutputToProjects, isMarkJoinSlotNotNull,
                 children.get(0), children.get(1));
     }
 
@@ -265,7 +255,7 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
     public Plan withGroupExpression(Optional<GroupExpression> groupExpression) 
{
         return new LogicalApply<>(groupExpression, 
Optional.of(getLogicalProperties()),
                 correlationSlot, subqueryType, isNot, compareExpr, 
typeCoercionExpr, correlationFilter,
-                markJoinSlotReference, needAddSubOutputToProjects, inProject, 
isMarkJoinSlotNotNull, left(), right());
+                markJoinSlotReference, needAddSubOutputToProjects, 
isMarkJoinSlotNotNull, left(), right());
     }
 
     @Override
@@ -274,6 +264,6 @@ public class LogicalApply<LEFT_CHILD_TYPE extends Plan, 
RIGHT_CHILD_TYPE extends
         Preconditions.checkArgument(children.size() == 2);
         return new LogicalApply<>(groupExpression, logicalProperties, 
correlationSlot, subqueryType, isNot,
                 compareExpr, typeCoercionExpr, correlationFilter, 
markJoinSlotReference,
-                needAddSubOutputToProjects, inProject, isMarkJoinSlotNotNull, 
children.get(0), children.get(1));
+                needAddSubOutputToProjects, isMarkJoinSlotNotNull, 
children.get(0), children.get(1));
     }
 }
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/analysis/SubqueryToApplyTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/analysis/SubqueryToApplyTest.java
new file mode 100644
index 00000000000..47127307995
--- /dev/null
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/analysis/SubqueryToApplyTest.java
@@ -0,0 +1,67 @@
+// 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.analysis;
+
+import org.apache.doris.nereids.trees.expressions.Alias;
+import org.apache.doris.nereids.trees.expressions.EqualTo;
+import org.apache.doris.nereids.trees.expressions.ExprId;
+import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.NamedExpression;
+import org.apache.doris.nereids.trees.expressions.ScalarSubquery;
+import org.apache.doris.nereids.trees.expressions.SlotReference;
+import org.apache.doris.nereids.trees.expressions.functions.agg.Count;
+import org.apache.doris.nereids.trees.expressions.functions.scalar.Nvl;
+import org.apache.doris.nereids.trees.expressions.literal.BigIntLiteral;
+import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.trees.plans.logical.LogicalAggregate;
+import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan;
+import org.apache.doris.nereids.trees.plans.logical.LogicalPlan;
+import org.apache.doris.nereids.types.IntegerType;
+import org.apache.doris.nereids.util.LogicalPlanBuilder;
+import org.apache.doris.nereids.util.PlanConstructor;
+
+import com.google.common.collect.ImmutableList;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.Test;
+
+import java.util.Optional;
+
+public class SubqueryToApplyTest {
+
+    @Test
+    void testAddNvlAggregate() {
+        SlotReference slotReference = new SlotReference("col1", 
IntegerType.INSTANCE);
+        NamedExpression aggregateFunction = new Alias(new ExprId(12345), new 
Count(slotReference), "count");
+        LogicalOlapScan logicalOlapScan = 
PlanConstructor.newLogicalOlapScan(0, "t1", 0);
+        LogicalAggregate<Plan> logicalAggregate = new LogicalAggregate<>(
+                ImmutableList.of(), ImmutableList.of(aggregateFunction), 
logicalOlapScan);
+        LogicalPlanBuilder planBuilder = new 
LogicalPlanBuilder(logicalAggregate);
+        LogicalPlan plan = planBuilder.projectAll().build();
+        Optional<Expression> correlatedOuterExpr = Optional
+                .of(new EqualTo(plan.getOutput().get(0), new 
BigIntLiteral(1)));
+        ScalarSubquery subquery = new ScalarSubquery(plan);
+        SubqueryToApply subqueryToApply = new SubqueryToApply();
+        SubqueryToApply.SubQueryRewriteResult rewriteResult = 
subqueryToApply.addNvlForScalarSubqueryOutput(
+                ImmutableList.of(aggregateFunction), plan.getOutput().get(0), 
subquery, correlatedOuterExpr);
+        Assertions.assertInstanceOf(EqualTo.class, 
rewriteResult.correlatedOuterExpr.get());
+        Assertions.assertInstanceOf(Alias.class, rewriteResult.subqueryOutput);
+        Assertions.assertInstanceOf(Nvl.class, 
rewriteResult.subqueryOutput.child(0));
+        Assertions.assertEquals(rewriteResult.subqueryOutput.toSlot(),
+                rewriteResult.correlatedOuterExpr.get().child(0));
+    }
+}
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/ExistsApplyToJoinTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/ExistsApplyToJoinTest.java
index 597eadf7a55..6290178754a 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/ExistsApplyToJoinTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/ExistsApplyToJoinTest.java
@@ -47,7 +47,7 @@ class ExistsApplyToJoinTest implements 
MemoPatternMatchSupported {
                 new LogicalApply<>(ImmutableList.of(leftSlots.get(0), 
rightSlots.get(0)),
                         LogicalApply.SubQueryType.EXITS_SUBQUERY, false, 
Optional.empty(), Optional.empty(),
                         Optional.of(equalTo), Optional.empty(),
-                        false, false, false, left, right);
+                        false, false, left, right);
         PlanChecker.from(MemoTestUtils.createConnectContext(), apply)
                 .applyTopDown(new ExistsApplyToJoin())
                 .matchesFromRoot(logicalJoin(
@@ -67,7 +67,7 @@ class ExistsApplyToJoinTest implements 
MemoPatternMatchSupported {
                 new LogicalApply<>(Collections.emptyList(),
                         LogicalApply.SubQueryType.EXITS_SUBQUERY, false, 
Optional.empty(), Optional.empty(),
                         Optional.of(equalTo), Optional.empty(),
-                        false, false, false, left, right);
+                        false, false, left, right);
         PlanChecker.from(MemoTestUtils.createConnectContext(), apply)
                 .applyTopDown(new ExistsApplyToJoin())
                 .matchesFromRoot(logicalJoin(
@@ -87,7 +87,7 @@ class ExistsApplyToJoinTest implements 
MemoPatternMatchSupported {
                 new LogicalApply<>(Collections.emptyList(),
                         LogicalApply.SubQueryType.EXITS_SUBQUERY, true, 
Optional.empty(), Optional.empty(),
                         Optional.of(equalTo), Optional.empty(),
-                        false, false, false, left, right);
+                        false, false, left, right);
         PlanChecker.from(MemoTestUtils.createConnectContext(), apply)
                 .applyTopDown(new ExistsApplyToJoin())
                 .matchesFromRoot(logicalFilter(logicalJoin(
@@ -108,7 +108,7 @@ class ExistsApplyToJoinTest implements 
MemoPatternMatchSupported {
                 new LogicalApply<>(ImmutableList.of(leftSlots.get(0), 
rightSlots.get(0)),
                         LogicalApply.SubQueryType.EXITS_SUBQUERY, true, 
Optional.empty(), Optional.empty(),
                         Optional.of(equalTo), Optional.empty(),
-                        false, false, false, left, right);
+                        false, false, left, right);
         PlanChecker.from(MemoTestUtils.createConnectContext(), apply)
                 .applyTopDown(new ExistsApplyToJoin())
                 .matchesFromRoot(logicalJoin(
diff --git 
a/regression-test/data/nereids_p0/subquery/correlated_scalar_subquery.out 
b/regression-test/data/nereids_p0/subquery/correlated_scalar_subquery.out
index 9414a5c9f61..f9d3c558033 100644
--- a/regression-test/data/nereids_p0/subquery/correlated_scalar_subquery.out
+++ b/regression-test/data/nereids_p0/subquery/correlated_scalar_subquery.out
@@ -106,3 +106,26 @@
 1
 1
 
+-- !select_agg_project1 --
+3
+4
+
+-- !select_agg_project2 --
+2
+2
+
+-- !select_2_aggs --
+3
+3
+4
+4
+5
+
+-- !select_3_aggs --
+1
+2
+2
+3
+4
+5
+
diff --git 
a/regression-test/suites/nereids_p0/subquery/correlated_scalar_subquery.groovy 
b/regression-test/suites/nereids_p0/subquery/correlated_scalar_subquery.groovy
index 80d9cdb4bb2..df7efda0099 100644
--- 
a/regression-test/suites/nereids_p0/subquery/correlated_scalar_subquery.groovy
+++ 
b/regression-test/suites/nereids_p0/subquery/correlated_scalar_subquery.groovy
@@ -220,4 +220,9 @@ suite("correlated_scalar_subquery") {
         """
         exception "access outer query's column before two agg nodes is not 
supported"
     }
+
+    qt_select_agg_project1 """select c2 from correlated_scalar_t1 where 
correlated_scalar_t1.c2 > (select if(count(c1) = 0, 2, 100) from 
correlated_scalar_t2 where correlated_scalar_t1.c1 = correlated_scalar_t2.c1) 
order by c2;"""
+    qt_select_agg_project2 """select c2 from correlated_scalar_t1 where 
correlated_scalar_t1.c2 = (select if(sum(c1) is null, 2, 100) from 
correlated_scalar_t2 where correlated_scalar_t1.c1 = correlated_scalar_t2.c1) 
order by c2;"""
+    qt_select_2_aggs """select c2 from correlated_scalar_t1 where 
correlated_scalar_t1.c2 > (select count(c1) - min(c1) from correlated_scalar_t2 
where correlated_scalar_t1.c1 = correlated_scalar_t2.c1) order by c2;"""
+    qt_select_3_aggs """select c2 from correlated_scalar_t1 where 
correlated_scalar_t1.c2 > (select if(sum(c1) is null, count(c1), max(c2)) from 
correlated_scalar_t2 where correlated_scalar_t1.c1 = correlated_scalar_t2.c1) 
order by c2;"""
 }
\ No newline at end of file


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]


Reply via email to