You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by ya...@apache.org on 2023/04/13 01:13:23 UTC

[doris] branch branch-1.2-lts updated: [cherry-pick-1.1](fix-expr) refractor create_tree_from_thrift to avoid stack overflow (#18514) (#18597)

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

yangzhg pushed a commit to branch branch-1.2-lts
in repository https://gitbox.apache.org/repos/asf/doris.git


The following commit(s) were added to refs/heads/branch-1.2-lts by this push:
     new 1a4a2350e8 [cherry-pick-1.1](fix-expr) refractor create_tree_from_thrift to avoid stack overflow (#18514) (#18597)
1a4a2350e8 is described below

commit 1a4a2350e80c0680cdf94c98646998414fa94594
Author: camby <zh...@baidu.com>
AuthorDate: Thu Apr 13 09:13:17 2023 +0800

    [cherry-pick-1.1](fix-expr) refractor create_tree_from_thrift to avoid stack overflow (#18514) (#18597)
    
    * cherry-pick pr18214 and also refractor create_tree_from_thrift in non-vectorized engine
---
 be/src/exprs/agg_fn_evaluator.cpp       |  3 +-
 be/src/exprs/expr.cpp                   | 61 +++++++++++++++++++++------------
 be/src/exprs/expr.h                     |  4 +--
 be/src/vec/exec/vanalytic_eval_node.cpp |  4 +--
 be/src/vec/exprs/vectorized_agg_fn.cpp  |  3 +-
 be/src/vec/exprs/vexpr.cpp              | 61 +++++++++++++++++++++------------
 be/src/vec/exprs/vexpr.h                |  6 ++--
 7 files changed, 88 insertions(+), 54 deletions(-)

diff --git a/be/src/exprs/agg_fn_evaluator.cpp b/be/src/exprs/agg_fn_evaluator.cpp
index 5998c5c79f..e43b5437f4 100644
--- a/be/src/exprs/agg_fn_evaluator.cpp
+++ b/be/src/exprs/agg_fn_evaluator.cpp
@@ -83,8 +83,7 @@ Status AggFnEvaluator::create(ObjectPool* pool, const TExpr& desc, bool is_analy
         ++node_idx;
         Expr* expr = nullptr;
         ExprContext* ctx = nullptr;
-        RETURN_IF_ERROR(
-                Expr::create_tree_from_thrift(pool, desc.nodes, nullptr, &node_idx, &expr, &ctx));
+        RETURN_IF_ERROR(Expr::create_tree_from_thrift(pool, desc.nodes, &node_idx, &expr, &ctx));
         (*result)->_input_exprs_ctxs.push_back(ctx);
     }
     return Status::OK();
diff --git a/be/src/exprs/expr.cpp b/be/src/exprs/expr.cpp
index b09bc6cfb1..53a50b798f 100644
--- a/be/src/exprs/expr.cpp
+++ b/be/src/exprs/expr.cpp
@@ -23,6 +23,7 @@
 #include <thrift/protocol/TDebugProtocol.h>
 
 #include <sstream>
+#include <stack>
 #include <vector>
 
 #include "common/object_pool.h"
@@ -255,7 +256,7 @@ Status Expr::create_expr_tree(ObjectPool* pool, const TExpr& texpr, ExprContext*
     }
     int node_idx = 0;
     Expr* e = nullptr;
-    Status status = create_tree_from_thrift(pool, texpr.nodes, nullptr, &node_idx, &e, ctx);
+    Status status = create_tree_from_thrift(pool, texpr.nodes, &node_idx, &e, ctx);
     if (status.ok() && node_idx + 1 != texpr.nodes.size()) {
         status = Status::InternalError(
                 "Expression tree only partially reconstructed. Not all thrift nodes were used.");
@@ -280,33 +281,51 @@ Status Expr::create_expr_trees(ObjectPool* pool, const std::vector<TExpr>& texpr
 }
 
 Status Expr::create_tree_from_thrift(ObjectPool* pool, const std::vector<TExprNode>& nodes,
-                                     Expr* parent, int* node_idx, Expr** root_expr,
-                                     ExprContext** ctx) {
+                                     int* node_idx, Expr** root_expr, ExprContext** ctx) {
     // propagate error case
     if (*node_idx >= nodes.size()) {
         return Status::InternalError("Failed to reconstruct expression tree from thrift.");
     }
-    int num_children = nodes[*node_idx].num_children;
-    Expr* expr = nullptr;
-    RETURN_IF_ERROR(create_expr(pool, nodes[*node_idx], &expr));
-    DCHECK(expr != nullptr);
-    if (parent != nullptr) {
-        parent->add_child(expr);
-    } else {
-        DCHECK(root_expr != nullptr);
-        DCHECK(ctx != nullptr);
-        *root_expr = expr;
-        *ctx = pool->add(new ExprContext(expr));
-    }
-    for (int i = 0; i < num_children; i++) {
-        *node_idx += 1;
-        RETURN_IF_ERROR(create_tree_from_thrift(pool, nodes, expr, node_idx, nullptr, nullptr));
-        // we are expecting a child, but have used all nodes
-        // this means we have been given a bad tree and must fail
-        if (*node_idx >= nodes.size()) {
+
+    // create root expr
+    int root_children = nodes[*node_idx].num_children;
+    Expr* root = nullptr;
+    RETURN_IF_ERROR(create_expr(pool, nodes[*node_idx], &root));
+    DCHECK(root != nullptr);
+
+    DCHECK(root_expr != nullptr);
+    DCHECK(ctx != nullptr);
+    *root_expr = root;
+    *ctx = pool->add(new ExprContext(root));
+    // short path for leaf node
+    if (root_children <= 0) {
+        return Status::OK();
+    }
+
+    // non-recursive traversal
+    std::stack<std::pair<Expr*, int>> s;
+    s.push({root, root_children});
+    while (!s.empty()) {
+        auto& parent = s.top();
+        if (parent.second > 1) {
+            parent.second -= 1;
+        } else {
+            s.pop();
+        }
+        if (++*node_idx >= nodes.size()) {
             return Status::InternalError("Failed to reconstruct expression tree from thrift.");
         }
+
+        Expr* expr = nullptr;
+        RETURN_IF_ERROR(create_expr(pool, nodes[*node_idx], &expr));
+        DCHECK(expr != nullptr);
+        parent.first->add_child(expr);
+        int num_children = nodes[*node_idx].num_children;
+        if (num_children > 0) {
+            s.push({expr, num_children});
+        }
     }
+
     return Status::OK();
 }
 
diff --git a/be/src/exprs/expr.h b/be/src/exprs/expr.h
index 448784d0f2..3a24ed5015 100644
--- a/be/src/exprs/expr.h
+++ b/be/src/exprs/expr.h
@@ -395,7 +395,6 @@ private:
     /// Creates an expr tree for the node rooted at 'node_idx' via depth-first traversal.
     /// parameters
     ///   nodes: vector of thrift expression nodes to be translated
-    ///   parent: parent of node at node_idx (or nullptr for node_idx == 0)
     ///   node_idx:
     ///     in: root of TExprNode tree
     ///     out: next node in 'nodes' that isn't part of tree
@@ -405,8 +404,7 @@ private:
     ///   status.ok() if successful
     ///   !status.ok() if tree is inconsistent or corrupt
     static Status create_tree_from_thrift(ObjectPool* pool, const std::vector<TExprNode>& nodes,
-                                          Expr* parent, int* node_idx, Expr** root_expr,
-                                          ExprContext** ctx);
+                                          int* node_idx, Expr** root_expr, ExprContext** ctx);
 
     /// Static wrappers around the virtual Get*Val() functions. Calls the appropriate
     /// Get*Val() function on expr, passing it the context and row arguments.
diff --git a/be/src/vec/exec/vanalytic_eval_node.cpp b/be/src/vec/exec/vanalytic_eval_node.cpp
index 925f493a7f..d1c4c6b10c 100644
--- a/be/src/vec/exec/vanalytic_eval_node.cpp
+++ b/be/src/vec/exec/vanalytic_eval_node.cpp
@@ -122,8 +122,8 @@ Status VAnalyticEvalNode::init(const TPlanNode& tnode, RuntimeState* state) {
             ++node_idx;
             VExpr* expr = nullptr;
             VExprContext* ctx = nullptr;
-            RETURN_IF_ERROR(VExpr::create_tree_from_thrift(_pool, desc.nodes, nullptr, &node_idx,
-                                                           &expr, &ctx));
+            RETURN_IF_ERROR(
+                    VExpr::create_tree_from_thrift(_pool, desc.nodes, &node_idx, &expr, &ctx));
             _agg_expr_ctxs[i].emplace_back(ctx);
         }
 
diff --git a/be/src/vec/exprs/vectorized_agg_fn.cpp b/be/src/vec/exprs/vectorized_agg_fn.cpp
index 213eb7bebd..6cf15fde9b 100644
--- a/be/src/vec/exprs/vectorized_agg_fn.cpp
+++ b/be/src/vec/exprs/vectorized_agg_fn.cpp
@@ -63,8 +63,7 @@ Status AggFnEvaluator::create(ObjectPool* pool, const TExpr& desc, const TSortIn
         ++node_idx;
         VExpr* expr = nullptr;
         VExprContext* ctx = nullptr;
-        RETURN_IF_ERROR(
-                VExpr::create_tree_from_thrift(pool, desc.nodes, nullptr, &node_idx, &expr, &ctx));
+        RETURN_IF_ERROR(VExpr::create_tree_from_thrift(pool, desc.nodes, &node_idx, &expr, &ctx));
         agg_fn_evaluator->_input_exprs_ctxs.push_back(ctx);
     }
 
diff --git a/be/src/vec/exprs/vexpr.cpp b/be/src/vec/exprs/vexpr.cpp
index 50e85369db..89ddc94a63 100644
--- a/be/src/vec/exprs/vexpr.cpp
+++ b/be/src/vec/exprs/vexpr.cpp
@@ -21,6 +21,7 @@
 #include <thrift/protocol/TDebugProtocol.h>
 
 #include <memory>
+#include <stack>
 
 #include "exprs/anyval_util.h"
 #include "gen_cpp/Exprs_types.h"
@@ -175,32 +176,50 @@ Status VExpr::create_expr(doris::ObjectPool* pool, const doris::TExprNode& texpr
 }
 
 Status VExpr::create_tree_from_thrift(doris::ObjectPool* pool,
-                                      const std::vector<doris::TExprNode>& nodes, VExpr* parent,
-                                      int* node_idx, VExpr** root_expr, VExprContext** ctx) {
+                                      const std::vector<doris::TExprNode>& nodes, int* node_idx,
+                                      VExpr** root_expr, VExprContext** ctx) {
     // propagate error case
     if (*node_idx >= nodes.size()) {
         return Status::InternalError("Failed to reconstruct expression tree from thrift.");
     }
-    int num_children = nodes[*node_idx].num_children;
-    VExpr* expr = nullptr;
-    RETURN_IF_ERROR(create_expr(pool, nodes[*node_idx], &expr));
-    DCHECK(expr != nullptr);
-    if (parent != nullptr) {
-        parent->add_child(expr);
-    } else {
-        DCHECK(root_expr != nullptr);
-        DCHECK(ctx != nullptr);
-        *root_expr = expr;
-        *ctx = pool->add(new VExprContext(expr));
-    }
-    for (int i = 0; i < num_children; i++) {
-        *node_idx += 1;
-        RETURN_IF_ERROR(create_tree_from_thrift(pool, nodes, expr, node_idx, nullptr, nullptr));
-        // we are expecting a child, but have used all nodes
-        // this means we have been given a bad tree and must fail
-        if (*node_idx >= nodes.size()) {
+
+    // create root expr
+    int root_children = nodes[*node_idx].num_children;
+    VExpr* root = nullptr;
+    RETURN_IF_ERROR(create_expr(pool, nodes[*node_idx], &root));
+    DCHECK(root != nullptr);
+
+    DCHECK(root_expr != nullptr);
+    DCHECK(ctx != nullptr);
+    *root_expr = root;
+    *ctx = pool->add(new VExprContext(root));
+    // short path for leaf node
+    if (root_children <= 0) {
+        return Status::OK();
+    }
+
+    // non-recursive traversal
+    std::stack<std::pair<VExpr*, int>> s;
+    s.push({root, root_children});
+    while (!s.empty()) {
+        auto& parent = s.top();
+        if (parent.second > 1) {
+            parent.second -= 1;
+        } else {
+            s.pop();
+        }
+
+        if (++*node_idx >= nodes.size()) {
             return Status::InternalError("Failed to reconstruct expression tree from thrift.");
         }
+        VExpr* expr = nullptr;
+        RETURN_IF_ERROR(create_expr(pool, nodes[*node_idx], &expr));
+        DCHECK(expr != nullptr);
+        parent.first->add_child(expr);
+        int num_children = nodes[*node_idx].num_children;
+        if (num_children > 0) {
+            s.push({expr, num_children});
+        }
     }
     return Status::OK();
 }
@@ -213,7 +232,7 @@ Status VExpr::create_expr_tree(doris::ObjectPool* pool, const doris::TExpr& texp
     }
     int node_idx = 0;
     VExpr* e = nullptr;
-    Status status = create_tree_from_thrift(pool, texpr.nodes, nullptr, &node_idx, &e, ctx);
+    Status status = create_tree_from_thrift(pool, texpr.nodes, &node_idx, &e, ctx);
     if (status.ok() && node_idx + 1 != texpr.nodes.size()) {
         status = Status::InternalError(
                 "Expression tree only partially reconstructed. Not all thrift nodes were used.");
diff --git a/be/src/vec/exprs/vexpr.h b/be/src/vec/exprs/vexpr.h
index b1f37477c4..09381477c1 100644
--- a/be/src/vec/exprs/vexpr.h
+++ b/be/src/vec/exprs/vexpr.h
@@ -124,9 +124,9 @@ public:
 
     static Status create_expr(ObjectPool* pool, const TExprNode& texpr_node, VExpr** expr);
 
-    static Status create_tree_from_thrift(ObjectPool* pool, const std::vector<TExprNode>& nodes,
-                                          VExpr* parent, int* node_idx, VExpr** root_expr,
-                                          VExprContext** ctx);
+    static Status create_tree_from_thrift(ObjectPool* pool,
+                                          const std::vector<doris::TExprNode>& nodes, int* node_idx,
+                                          VExpr** root_expr, VExprContext** ctx);
     const std::vector<VExpr*>& children() const { return _children; }
     void set_children(std::vector<VExpr*> children) { _children = children; }
     virtual std::string debug_string() const;


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