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

yiguolei 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 272a7565b8 [improvement](tracing) Remove useless span levels from be 
side tracing (#19665)
272a7565b8 is described below

commit 272a7565b81cdd09237f3c3815da42a3e852de06
Author: luozenglin <luozeng...@baidu.com>
AuthorDate: Wed May 17 19:04:52 2023 +0800

    [improvement](tracing) Remove useless span levels from be side tracing 
(#19665)
    
    1. Remove an exec node method corresponding to a span and replace it with 
an exec node corresponding to a span;
    2. Fix some problems with tracing in pipeline.
---
 be/src/common/status.h                          | 13 ---------
 be/src/exec/data_sink.h                         | 10 ++-----
 be/src/exec/exec_node.cpp                       |  4 ++-
 be/src/exec/exec_node.h                         | 11 ++-----
 be/src/pipeline/pipeline_fragment_context.cpp   |  1 -
 be/src/runtime/fragment_mgr.cpp                 | 17 ++++++-----
 be/src/runtime/plan_fragment_executor.cpp       | 20 ++++++-------
 be/src/runtime/plan_fragment_executor.h         |  2 ++
 be/src/util/runtime_profile.cpp                 | 34 ++++++++-------------
 be/src/util/runtime_profile.h                   |  9 ++----
 be/src/util/telemetry/telemetry.h               | 18 ------------
 be/src/vec/exec/join/vhash_join_node.cpp        | 32 ++++++++------------
 be/src/vec/exec/join/vjoin_node_base.cpp        |  8 +----
 be/src/vec/exec/join/vnested_loop_join_node.cpp | 32 ++++++++------------
 be/src/vec/exec/scan/vscan_node.cpp             |  5 ----
 be/src/vec/exec/vaggregation_node.cpp           | 32 ++++++++------------
 be/src/vec/exec/vanalytic_eval_node.cpp         | 18 ++++--------
 be/src/vec/exec/vassert_num_rows_node.cpp       | 17 ++++-------
 be/src/vec/exec/vempty_set_node.cpp             |  1 -
 be/src/vec/exec/vexchange_node.cpp              |  4 ---
 be/src/vec/exec/vmysql_scan_node.cpp            |  3 --
 be/src/vec/exec/vrepeat_node.cpp                | 18 ++++--------
 be/src/vec/exec/vschema_scan_node.cpp           |  8 -----
 be/src/vec/exec/vselect_node.cpp                | 18 ++++--------
 be/src/vec/exec/vset_operation_node.cpp         | 33 ++++++++-------------
 be/src/vec/exec/vsort_node.cpp                  | 17 ++++-------
 be/src/vec/exec/vtable_function_node.cpp        | 15 ++++------
 be/src/vec/exec/vtable_function_node.h          |  1 -
 be/src/vec/exec/vunion_node.cpp                 | 39 ++++++++++---------------
 be/src/vec/sink/vdata_stream_sender.cpp         |  3 --
 be/src/vec/sink/vjdbc_table_sink.cpp            |  3 --
 be/src/vec/sink/vmysql_table_sink.cpp           |  3 --
 be/src/vec/sink/vodbc_table_sink.cpp            |  3 --
 be/src/vec/sink/vresult_file_sink.cpp           |  3 --
 be/src/vec/sink/vresult_sink.cpp                |  3 --
 be/src/vec/sink/vtable_sink.cpp                 |  2 --
 be/src/vec/sink/vtablet_sink.cpp                |  3 --
 37 files changed, 143 insertions(+), 320 deletions(-)

diff --git a/be/src/common/status.h b/be/src/common/status.h
index 5ae3697a15..d60e103727 100644
--- a/be/src/common/status.h
+++ b/be/src/common/status.h
@@ -523,19 +523,6 @@ inline std::string Status::to_string() const {
 #define RETURN_ERROR_IF_NON_VEC \
     return Status::NotSupported("Non-vectorized engine is not supported since 
Doris 2.0.");
 
-// End _get_next_span after last call to get_next method
-#define RETURN_IF_ERROR_AND_CHECK_SPAN(stmt, get_next_span, done) \
-    do {                                                          \
-        Status _status_ = (stmt);                                 \
-        auto _span = (get_next_span);                             \
-        if (UNLIKELY(_span && (!_status_.ok() || done))) {        \
-            _span->End();                                         \
-        }                                                         \
-        if (UNLIKELY(!_status_.ok())) {                           \
-            return _status_;                                      \
-        }                                                         \
-    } while (false)
-
 #define RETURN_IF_STATUS_ERROR(status, stmt) \
     do {                                     \
         status = (stmt);                     \
diff --git a/be/src/exec/data_sink.h b/be/src/exec/data_sink.h
index ee847da138..542dc428cc 100644
--- a/be/src/exec/data_sink.h
+++ b/be/src/exec/data_sink.h
@@ -71,7 +71,7 @@ public:
     // It must be okay to call this multiple times. Subsequent calls should
     // be ignored.
     virtual Status close(RuntimeState* state, Status exec_status) {
-        profile()->add_to_span();
+        profile()->add_to_span(_span);
         _closed = true;
         return Status::OK();
     }
@@ -98,12 +98,6 @@ public:
         _query_statistics = statistics;
     }
 
-    void end_send_span() {
-        if (_send_span) {
-            _send_span->End();
-        }
-    }
-
 protected:
     // Set to true after close() has been called. subclasses should check and 
set this in
     // close().
@@ -113,7 +107,7 @@ protected:
     // Maybe this will be transferred to BufferControlBlock.
     std::shared_ptr<QueryStatistics> _query_statistics;
 
-    OpentelemetrySpan _send_span {};
+    OpentelemetrySpan _span {};
 };
 
 } // namespace doris
diff --git a/be/src/exec/exec_node.cpp b/be/src/exec/exec_node.cpp
index a1f29aaf8b..62a63afd98 100644
--- a/be/src/exec/exec_node.cpp
+++ b/be/src/exec/exec_node.cpp
@@ -120,6 +120,8 @@ Status ExecNode::init(const TPlanNode& tnode, RuntimeState* 
state) {
 
 Status ExecNode::prepare(RuntimeState* state) {
     DCHECK(_runtime_profile.get() != nullptr);
+    _span = state->get_tracer()->StartSpan(get_name());
+    OpentelemetryScope scope {_span};
     _rows_returned_counter = ADD_COUNTER(_runtime_profile, "RowsReturned", 
TUnit::UNIT);
     _projection_timer = ADD_TIMER(_runtime_profile, "ProjectionTime");
     _rows_returned_rate = runtime_profile()->add_derived_counter(
@@ -182,7 +184,7 @@ void ExecNode::release_resource(doris::RuntimeState* state) 
{
         }
         vectorized::VExpr::close(_projections, state);
 
-        runtime_profile()->add_to_span();
+        runtime_profile()->add_to_span(_span);
         _is_resource_released = true;
     }
 }
diff --git a/be/src/exec/exec_node.h b/be/src/exec/exec_node.h
index d9a7e5afd7..d119609ab5 100644
--- a/be/src/exec/exec_node.h
+++ b/be/src/exec/exec_node.h
@@ -235,7 +235,7 @@ public:
 
     MemTracker* mem_tracker() const { return _mem_tracker.get(); }
 
-    OpentelemetrySpan get_next_span() { return _get_next_span; }
+    OpentelemetrySpan get_next_span() { return _span; }
 
     virtual std::string get_name();
 
@@ -289,13 +289,8 @@ protected:
     RuntimeProfile::Counter* _memory_used_counter;
     RuntimeProfile::Counter* _projection_timer;
 
-    /// Since get_next is a frequent operation, it is not necessary to 
generate a span for each call
-    /// to the get_next method. Therefore, the call of the get_next method in 
the ExecNode is
-    /// merged into this _get_next_span. The _get_next_span is initialized by
-    /// INIT_AND_SCOPE_GET_NEXT_SPAN when the get_next method is called for 
the first time
-    /// (recording the start timestamp), and is ended by 
RETURN_IF_ERROR_AND_CHECK_SPAN after the
-    /// last call to the get_next method (the record is terminated timestamp).
-    OpentelemetrySpan _get_next_span;
+    //
+    OpentelemetrySpan _span;
 
     // Execution options that are determined at runtime.  This is added to the
     // runtime profile at close().  Examples for options logged here would be
diff --git a/be/src/pipeline/pipeline_fragment_context.cpp 
b/be/src/pipeline/pipeline_fragment_context.cpp
index 6384b99315..362a697fcb 100644
--- a/be/src/pipeline/pipeline_fragment_context.cpp
+++ b/be/src/pipeline/pipeline_fragment_context.cpp
@@ -199,7 +199,6 @@ Status PipelineFragmentContext::prepare(const 
doris::TPipelineFragmentParams& re
     if 
(opentelemetry::trace::Tracer::GetCurrentSpan()->GetContext().IsValid()) {
         tracer = telemetry::get_tracer(print_id(_query_id));
     }
-    START_AND_SCOPE_SPAN(tracer, span, "PipelineFragmentExecutor::prepare");
 
     LOG_INFO("PipelineFragmentContext::prepare")
             .tag("query_id", _query_id)
diff --git a/be/src/runtime/fragment_mgr.cpp b/be/src/runtime/fragment_mgr.cpp
index c8e6a8b25e..6912e58275 100644
--- a/be/src/runtime/fragment_mgr.cpp
+++ b/be/src/runtime/fragment_mgr.cpp
@@ -515,14 +515,8 @@ void 
FragmentMgr::_exec_actual(std::shared_ptr<FragmentExecState> exec_state,
                                const FinishCallback& cb) {
     std::string func_name {"PlanFragmentExecutor::_exec_actual"};
 #ifndef BE_TEST
-    auto span = 
exec_state->executor()->runtime_state()->get_tracer()->StartSpan(func_name);
     SCOPED_ATTACH_TASK(exec_state->executor()->runtime_state());
-#else
-    auto span = telemetry::get_noop_tracer()->StartSpan(func_name);
 #endif
-    auto scope = opentelemetry::trace::Scope {span};
-    span->SetAttribute("query_id", print_id(exec_state->query_id()));
-    span->SetAttribute("instance_id", 
print_id(exec_state->fragment_instance_id()));
 
     LOG_INFO(func_name)
             .tag("query_id", exec_state->query_id())
@@ -732,13 +726,16 @@ Status FragmentMgr::exec_plan_fragment(const 
TExecPlanFragmentParams& params,
                                        const FinishCallback& cb) {
     auto tracer = telemetry::is_current_span_valid() ? 
telemetry::get_tracer("tracer")
                                                      : 
telemetry::get_noop_tracer();
+    auto cur_span = opentelemetry::trace::Tracer::GetCurrentSpan();
+    cur_span->SetAttribute("query_id", print_id(params.params.query_id));
+    cur_span->SetAttribute("instance_id", 
print_id(params.params.fragment_instance_id));
+
     VLOG_ROW << "exec_plan_fragment params is "
              << apache::thrift::ThriftDebugString(params).c_str();
     // sometimes TExecPlanFragmentParams debug string is too long and glog
     // will truncate the log line, so print query options seperately for 
debuggin purpose
     VLOG_ROW << "query options is "
              << 
apache::thrift::ThriftDebugString(params.query_options).c_str();
-    START_AND_SCOPE_SPAN(tracer, span, "FragmentMgr::exec_plan_fragment");
     const TUniqueId& fragment_instance_id = params.params.fragment_instance_id;
     {
         std::lock_guard<std::mutex> lock(_lock);
@@ -809,13 +806,15 @@ Status FragmentMgr::exec_plan_fragment(const 
TPipelineFragmentParams& params,
                                        const FinishCallback& cb) {
     auto tracer = telemetry::is_current_span_valid() ? 
telemetry::get_tracer("tracer")
                                                      : 
telemetry::get_noop_tracer();
+    auto cur_span = opentelemetry::trace::Tracer::GetCurrentSpan();
+    cur_span->SetAttribute("query_id", print_id(params.query_id));
+
     VLOG_ROW << "exec_plan_fragment params is "
              << apache::thrift::ThriftDebugString(params).c_str();
     // sometimes TExecPlanFragmentParams debug string is too long and glog
     // will truncate the log line, so print query options seperately for 
debuggin purpose
     VLOG_ROW << "query options is "
              << 
apache::thrift::ThriftDebugString(params.query_options).c_str();
-    START_AND_SCOPE_SPAN(tracer, span, "FragmentMgr::exec_plan_fragment");
 
     std::shared_ptr<FragmentExecState> exec_state;
     std::shared_ptr<QueryContext> query_ctx;
@@ -833,6 +832,8 @@ Status FragmentMgr::exec_plan_fragment(const 
TPipelineFragmentParams& params,
                 continue;
             }
         }
+        START_AND_SCOPE_SPAN(tracer, span, "exec_instance");
+        span->SetAttribute("instance_id", print_id(fragment_instance_id));
 
         query_ctx->fragment_ids.push_back(fragment_instance_id);
 
diff --git a/be/src/runtime/plan_fragment_executor.cpp 
b/be/src/runtime/plan_fragment_executor.cpp
index ddad12e176..ed8aa34524 100644
--- a/be/src/runtime/plan_fragment_executor.cpp
+++ b/be/src/runtime/plan_fragment_executor.cpp
@@ -99,7 +99,8 @@ Status PlanFragmentExecutor::prepare(const 
TExecPlanFragmentParams& request,
     if 
(opentelemetry::trace::Tracer::GetCurrentSpan()->GetContext().IsValid()) {
         tracer = telemetry::get_tracer(print_id(_query_id));
     }
-    START_AND_SCOPE_SPAN(tracer, span, "PlanFragmentExecutor::prepare");
+    _span = tracer->StartSpan("Plan_fragment_executor");
+    OpentelemetryScope scope {_span};
 
     const TPlanFragmentExecParams& params = request.params;
     _query_id = params.query_id;
@@ -305,7 +306,6 @@ Status PlanFragmentExecutor::open_vectorized_internal() {
             return Status::OK();
         }
         RETURN_IF_ERROR(_sink->open(runtime_state()));
-        auto sink_send_span_guard = Defer {[this]() { 
this->_sink->end_send_span(); }};
         doris::vectorized::Block block;
         bool eos = false;
 
@@ -348,14 +348,12 @@ Status PlanFragmentExecutor::open_vectorized_internal() {
 Status 
PlanFragmentExecutor::get_vectorized_internal(::doris::vectorized::Block* 
block, bool* eos) {
     while (!_done) {
         block->clear_column_data(_plan->row_desc().num_materialized_slots());
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                _plan->get_next_after_projects(
-                        _runtime_state.get(), block, &_done,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _plan, std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                _plan->get_next_span(), _done);
+        RETURN_IF_ERROR(_plan->get_next_after_projects(
+                _runtime_state.get(), block, &_done,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _plan, std::placeholders::_1, std::placeholders::_2,
+                          std::placeholders::_3)));
 
         if (block->rows() > 0) {
             COUNTER_UPDATE(_rows_produced_counter, block->rows());
@@ -564,7 +562,7 @@ void PlanFragmentExecutor::close() {
                   << print_id(_runtime_state->fragment_instance_id());
     }
 
-    profile()->add_to_span();
+    profile()->add_to_span(_span);
     _closed = true;
 }
 
diff --git a/be/src/runtime/plan_fragment_executor.h 
b/be/src/runtime/plan_fragment_executor.h
index f74256cdcb..c95ddc75c1 100644
--- a/be/src/runtime/plan_fragment_executor.h
+++ b/be/src/runtime/plan_fragment_executor.h
@@ -206,6 +206,8 @@ private:
     PPlanFragmentCancelReason _cancel_reason;
     std::string _cancel_msg;
 
+    OpentelemetrySpan _span;
+
     ObjectPool* obj_pool() { return _runtime_state->obj_pool(); }
 
     // typedef for TPlanFragmentExecParams.per_node_scan_ranges
diff --git a/be/src/util/runtime_profile.cpp b/be/src/util/runtime_profile.cpp
index 050b8138ac..c7ccf802db 100644
--- a/be/src/util/runtime_profile.cpp
+++ b/be/src/util/runtime_profile.cpp
@@ -43,6 +43,8 @@ static const std::string THREAD_SYS_TIME = "SysTime";
 static const std::string THREAD_VOLUNTARY_CONTEXT_SWITCHES = 
"VoluntaryContextSwitches";
 static const std::string THREAD_INVOLUNTARY_CONTEXT_SWITCHES = 
"InvoluntaryContextSwitches";
 
+static const std::string SPAN_ATTRIBUTE_KEY_SEPARATOR = "-";
+
 // The root counter name for all top level counters.
 static const std::string ROOT_COUNTER;
 
@@ -546,9 +548,8 @@ void RuntimeProfile::pretty_print(std::ostream* s, const 
std::string& prefix) co
     }
 }
 
-void RuntimeProfile::add_to_span() {
-    auto span = opentelemetry::trace::Tracer::GetCurrentSpan();
-    if (!span->IsRecording() || _added_to_span) {
+void RuntimeProfile::add_to_span(OpentelemetrySpan span) {
+    if (!span || !span->IsRecording() || _added_to_span) {
         return;
     }
     _added_to_span = true;
@@ -568,7 +569,8 @@ void RuntimeProfile::add_to_span() {
     // to "VDataBufferSender"
     auto i = _name.find_first_of("(: ");
     auto short_name = _name.substr(0, i);
-    span->SetAttribute("TotalTime", print_json_counter(short_name, 
total_time->second));
+    span->SetAttribute(short_name + SPAN_ATTRIBUTE_KEY_SEPARATOR + "TotalTime",
+                       print_counter(total_time->second));
 
     {
         std::lock_guard<std::mutex> l(_info_strings_lock);
@@ -577,7 +579,8 @@ void RuntimeProfile::add_to_span() {
             if (key.compare("KeyRanges") == 0) {
                 continue;
             }
-            span->SetAttribute(key, print_json_info(short_name, 
_info_strings.find(key)->second));
+            span->SetAttribute(short_name + SPAN_ATTRIBUTE_KEY_SEPARATOR + key,
+                               _info_strings.find(key)->second);
         }
     }
 
@@ -590,9 +593,8 @@ void RuntimeProfile::add_to_span() {
         children = _children;
     }
 
-    for (int i = 0; i < children.size(); ++i) {
-        RuntimeProfile* profile = children[i].first;
-        profile->add_to_span();
+    for (auto& [profile, flag] : children) {
+        profile->add_to_span(span);
     }
 }
 
@@ -608,26 +610,14 @@ void 
RuntimeProfile::add_child_counters_to_span(OpentelemetrySpan span,
         for (const std::string& child_counter : child_counters) {
             CounterMap::const_iterator iter = counter_map.find(child_counter);
             DCHECK(iter != counter_map.end());
-            span->SetAttribute(iter->first, print_json_counter(profile_name, 
iter->second));
+            span->SetAttribute(profile_name + SPAN_ATTRIBUTE_KEY_SEPARATOR + 
iter->first,
+                               print_counter(iter->second));
             RuntimeProfile::add_child_counters_to_span(span, profile_name, 
child_counter,
                                                        counter_map, 
child_counter_map);
         }
     }
 }
 
-std::string RuntimeProfile::print_json_info(const std::string& profile_name, 
std::string value) {
-    rapidjson::StringBuffer s;
-    rapidjson::Writer<rapidjson::StringBuffer> writer(s);
-
-    writer.StartObject();
-    writer.Key("profile");
-    writer.String(profile_name.c_str());
-    writer.Key("pretty");
-    writer.String(value.c_str());
-    writer.EndObject();
-    return s.GetString();
-}
-
 void RuntimeProfile::to_thrift(TRuntimeProfileTree* tree) {
     tree->nodes.clear();
     to_thrift(&tree->nodes);
diff --git a/be/src/util/runtime_profile.h b/be/src/util/runtime_profile.h
index e0a8ec50b3..6e18f08ac9 100644
--- a/be/src/util/runtime_profile.h
+++ b/be/src/util/runtime_profile.h
@@ -321,7 +321,7 @@ public:
     // Does not hold locks when it makes any function calls.
     void pretty_print(std::ostream* s, const std::string& prefix = "") const;
 
-    void add_to_span();
+    void add_to_span(OpentelemetrySpan span);
 
     // Serializes profile to thrift.
     // Does not hold locks when it makes any function calls.
@@ -507,12 +507,9 @@ private:
                                            const CounterMap& counter_map,
                                            const ChildCounterMap& 
child_counter_map);
 
-    static std::string print_json_counter(const std::string& profile_name, 
Counter* counter) {
-        return print_json_info(profile_name,
-                               PrettyPrinter::print(counter->value(), 
counter->type()));
+    static std::string print_counter(Counter* counter) {
+        return PrettyPrinter::print(counter->value(), counter->type());
     }
-
-    static std::string print_json_info(const std::string& profile_name, 
std::string value);
 };
 
 // Utility class to update the counter at object construction and destruction.
diff --git a/be/src/util/telemetry/telemetry.h 
b/be/src/util/telemetry/telemetry.h
index c007269550..dc71f3cfe0 100644
--- a/be/src/util/telemetry/telemetry.h
+++ b/be/src/util/telemetry/telemetry.h
@@ -40,18 +40,6 @@ using OpentelemetryTracer = 
opentelemetry::nostd::shared_ptr<opentelemetry::trac
 using OpentelemetrySpan = 
opentelemetry::nostd::shared_ptr<opentelemetry::trace::Span>;
 using OpentelemetryScope = opentelemetry::trace::Scope;
 
-/// Used to initialize get_next_span and add Scope.
-#define INIT_AND_SCOPE_GET_NEXT_SPAN(tracer, get_next_span, name) \
-    do {                                                          \
-        if (UNLIKELY(!get_next_span)) {                           \
-            get_next_span = tracer->StartSpan(name);              \
-        }                                                         \
-    } while (false);                                              \
-    OpentelemetryScope scope {get_next_span};
-
-#define INIT_AND_SCOPE_SEND_SPAN(tracer, send_span, name) \
-    INIT_AND_SCOPE_GET_NEXT_SPAN(tracer, send_span, name)
-
 /// Start a span with the specified tracer, name, and variable name, and 
create a Scope for this
 /// span.
 ///
@@ -73,12 +61,6 @@ using OpentelemetryScope = opentelemetry::trace::Scope;
     auto span = tracer->StartSpan(name);         \
     OpentelemetryScope scope {span};
 
-#define START_AND_SCOPE_SPAN_IF(enable, tracer, name) \
-    OpenTelemetryScopeWrapper(enable, tracer, name)
-
-#define INIT_AND_SCOPE_REENTRANT_SPAN_IF(enable, tracer, reentrant_span, name) 
\
-    OpenTelemetryScopeWrapper(enable, tracer, reentrant_span, name)
-
 namespace telemetry {
 
 void init_tracer();
diff --git a/be/src/vec/exec/join/vhash_join_node.cpp 
b/be/src/vec/exec/join/vhash_join_node.cpp
index f099bed947..fee17c952d 100644
--- a/be/src/vec/exec/join/vhash_join_node.cpp
+++ b/be/src/vec/exec/join/vhash_join_node.cpp
@@ -641,7 +641,6 @@ Status HashJoinNode::push(RuntimeState* /*state*/, 
vectorized::Block* input_bloc
 }
 
 Status HashJoinNode::get_next(RuntimeState* state, Block* output_block, bool* 
eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"HashJoinNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
 
     if (_short_circuit_for_null_in_probe_side) {
@@ -669,14 +668,12 @@ Status HashJoinNode::get_next(RuntimeState* state, Block* 
output_block, bool* eo
     while (need_more_input_data()) {
         prepare_for_next();
         SCOPED_TIMER(_probe_next_timer);
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                child(0)->get_next_after_projects(
-                        state, &_probe_block, &_probe_eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                child(0)->get_next_span(), _probe_eos);
+        RETURN_IF_ERROR(child(0)->get_next_after_projects(
+                state, &_probe_block, &_probe_eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
 
         RETURN_IF_ERROR(push(state, &_probe_block, _probe_eos));
     }
@@ -731,7 +728,6 @@ void HashJoinNode::_prepare_probe_block() {
 }
 
 Status HashJoinNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "HashJoinNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(VJoinNodeBase::open(state));
     RETURN_IF_CANCELLED(state);
@@ -755,7 +751,6 @@ Status HashJoinNode::alloc_resource(doris::RuntimeState* 
state) {
 }
 
 void HashJoinNode::release_resource(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"HashJoinNode::release_resources");
     VExpr::close(_build_expr_ctxs, state);
     VExpr::close(_probe_expr_ctxs, state);
 
@@ -778,15 +773,12 @@ Status 
HashJoinNode::_materialize_build_side(RuntimeState* state) {
             block.clear_column_data();
             RETURN_IF_CANCELLED(state);
 
-            RETURN_IF_ERROR_AND_CHECK_SPAN(
-                    child(1)->get_next_after_projects(
-                            state, &block, &eos,
-                            std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*,
-                                                           bool*)) &
-                                              ExecNode::get_next,
-                                      _children[1], std::placeholders::_1, 
std::placeholders::_2,
-                                      std::placeholders::_3)),
-                    child(1)->get_next_span(), eos);
+            RETURN_IF_ERROR(child(1)->get_next_after_projects(
+                    state, &block, &eos,
+                    std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                      ExecNode::get_next,
+                              _children[1], std::placeholders::_1, 
std::placeholders::_2,
+                              std::placeholders::_3)));
 
             RETURN_IF_ERROR(sink(state, &block, eos));
         }
diff --git a/be/src/vec/exec/join/vjoin_node_base.cpp 
b/be/src/vec/exec/join/vjoin_node_base.cpp
index 59e24cbdcf..f6f84b2b91 100644
--- a/be/src/vec/exec/join/vjoin_node_base.cpp
+++ b/be/src/vec/exec/join/vjoin_node_base.cpp
@@ -21,7 +21,6 @@
 #include <gen_cpp/PlanNodes_types.h>
 #include <glog/logging.h>
 #include <opentelemetry/nostd/shared_ptr.h>
-#include <opentelemetry/trace/span.h>
 #include <opentelemetry/trace/tracer.h>
 #include <stddef.h>
 
@@ -110,7 +109,6 @@ Status VJoinNodeBase::close(RuntimeState* state) {
 }
 
 void VJoinNodeBase::release_resource(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VJoinNodeBase::release_resource");
     VExpr::close(_output_expr_ctxs, state);
     _join_block.clear();
     ExecNode::release_resource(state);
@@ -199,16 +197,13 @@ Status VJoinNodeBase::init(const TPlanNode& tnode, 
RuntimeState* state) {
 }
 
 Status VJoinNodeBase::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VJoinNodeBase::open");
     RETURN_IF_ERROR(ExecNode::open(state));
     RETURN_IF_CANCELLED(state);
 
     std::promise<Status> thread_status;
     try {
         state->exec_env()->join_node_thread_pool()->submit_func(
-                [this, state, thread_status_p = &thread_status,
-                 parent_span = opentelemetry::trace::Tracer::GetCurrentSpan()] 
{
-                    OpentelemetryScope scope {parent_span};
+                [this, state, thread_status_p = &thread_status] {
                     this->_probe_side_open_thread(state, thread_status_p);
                 });
     } catch (const std::system_error& e) {
@@ -243,7 +238,6 @@ void VJoinNodeBase::_reset_tuple_is_null_column() {
 }
 
 void VJoinNodeBase::_probe_side_open_thread(RuntimeState* state, 
std::promise<Status>* status) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VJoinNodeBase::_hash_table_build_thread");
     SCOPED_ATTACH_TASK(state);
     status->set_value(child(0)->open(state));
 }
diff --git a/be/src/vec/exec/join/vnested_loop_join_node.cpp 
b/be/src/vec/exec/join/vnested_loop_join_node.cpp
index d0b308264e..9d314f4160 100644
--- a/be/src/vec/exec/join/vnested_loop_join_node.cpp
+++ b/be/src/vec/exec/join/vnested_loop_join_node.cpp
@@ -166,7 +166,6 @@ Status VNestedLoopJoinNode::close(RuntimeState* state) {
     if (is_closed()) {
         return Status::OK();
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VNestedLoopJoinNode::close");
     _release_mem();
 
     return VJoinNodeBase::close(state);
@@ -181,14 +180,12 @@ Status 
VNestedLoopJoinNode::_materialize_build_side(RuntimeState* state) {
         RETURN_IF_CANCELLED(state);
 
         Block block;
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                child(1)->get_next_after_projects(
-                        state, &block, &eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[1], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                child(1)->get_next_span(), eos);
+        RETURN_IF_ERROR(child(1)->get_next_after_projects(
+                state, &block, &eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[1], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
 
         sink(state, &block, eos);
 
@@ -251,14 +248,12 @@ Status VNestedLoopJoinNode::push(doris::RuntimeState* 
state, vectorized::Block*
 Status VNestedLoopJoinNode::_fresh_left_block(doris::RuntimeState* state) {
     do {
         release_block_memory(_left_block);
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                child(0)->get_next_after_projects(
-                        state, &_left_block, &_left_side_eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                child(0)->get_next_span(), _left_side_eos);
+        RETURN_IF_ERROR(child(0)->get_next_after_projects(
+                state, &_left_block, &_left_side_eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
 
     } while (_left_block.rows() == 0 && !_left_side_eos);
 
@@ -266,8 +261,6 @@ Status 
VNestedLoopJoinNode::_fresh_left_block(doris::RuntimeState* state) {
 }
 
 Status VNestedLoopJoinNode::get_next(RuntimeState* state, Block* block, bool* 
eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span,
-                                 "VNestedLoopJoinNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     SCOPED_TIMER(_probe_timer);
     RETURN_IF_CANCELLED(state);
@@ -657,7 +650,6 @@ Status 
VNestedLoopJoinNode::alloc_resource(doris::RuntimeState* state) {
 }
 
 Status VNestedLoopJoinNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VNestedLoopJoinNode::open")
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(VJoinNodeBase::open(state));
     RETURN_IF_CANCELLED(state);
diff --git a/be/src/vec/exec/scan/vscan_node.cpp 
b/be/src/vec/exec/scan/vscan_node.cpp
index 532cebefd4..c81f44b3a3 100644
--- a/be/src/vec/exec/scan/vscan_node.cpp
+++ b/be/src/vec/exec/scan/vscan_node.cpp
@@ -154,7 +154,6 @@ Status VScanNode::prepare(RuntimeState* state) {
 }
 
 Status VScanNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VScanNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_CANCELLED(state);
     return ExecNode::open(state);
@@ -166,7 +165,6 @@ Status VScanNode::alloc_resource(RuntimeState* state) {
     }
     _input_tuple_desc = 
state->desc_tbl().get_tuple_descriptor(_input_tuple_id);
     _output_tuple_desc = 
state->desc_tbl().get_tuple_descriptor(_output_tuple_id);
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VScanNode::alloc_resource");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::alloc_resource(state));
     RETURN_IF_ERROR(_acquire_runtime_filter());
@@ -208,7 +206,6 @@ Status VScanNode::alloc_resource(RuntimeState* state) {
 }
 
 Status VScanNode::get_next(RuntimeState* state, vectorized::Block* block, 
bool* eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VScanNode::get_next");
     SCOPED_TIMER(_get_next_timer);
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     // in inverted index apply logic, in order to optimize query performance,
@@ -445,13 +442,11 @@ Status VScanNode::close(RuntimeState* state) {
     if (is_closed()) {
         return Status::OK();
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VScanNode::close");
     RETURN_IF_ERROR(ExecNode::close(state));
     return Status::OK();
 }
 
 void VScanNode::release_resource(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VScanNode::release_resource");
     if (_scanner_ctx.get()) {
         if (!state->enable_pipeline_exec() || _should_create_scanner) {
             // stop and wait the scanner scheduler to be done
diff --git a/be/src/vec/exec/vaggregation_node.cpp 
b/be/src/vec/exec/vaggregation_node.cpp
index 20bf48fe47..1ebb0721b7 100644
--- a/be/src/vec/exec/vaggregation_node.cpp
+++ b/be/src/vec/exec/vaggregation_node.cpp
@@ -511,7 +511,6 @@ Status AggregationNode::alloc_resource(doris::RuntimeState* 
state) {
 }
 
 Status AggregationNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "AggregationNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::open(state));
     RETURN_IF_ERROR(_children[0]->open(state));
@@ -523,14 +522,12 @@ Status AggregationNode::open(RuntimeState* state) {
     while (!eos) {
         RETURN_IF_CANCELLED(state);
         release_block_memory(block);
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                _children[0]->get_next_after_projects(
-                        state, &block, &eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                _children[0]->get_next_span(), eos);
+        RETURN_IF_ERROR(_children[0]->get_next_after_projects(
+                state, &block, &eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
         RETURN_IF_ERROR(sink(state, &block, eos));
     }
     _children[0]->close(state);
@@ -551,22 +548,18 @@ Status AggregationNode::do_pre_agg(vectorized::Block* 
input_block,
 }
 
 Status AggregationNode::get_next(RuntimeState* state, Block* block, bool* eos) 
{
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"AggregationNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
 
     if (_is_streaming_preagg) {
         RETURN_IF_CANCELLED(state);
         release_block_memory(_preagg_block);
         while (_preagg_block.rows() == 0 && !_child_eos) {
-            RETURN_IF_ERROR_AND_CHECK_SPAN(
-                    _children[0]->get_next_after_projects(
-                            state, &_preagg_block, &_child_eos,
-                            std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*,
-                                                           bool*)) &
-                                              ExecNode::get_next,
-                                      _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                      std::placeholders::_3)),
-                    _children[0]->get_next_span(), _child_eos);
+            RETURN_IF_ERROR(_children[0]->get_next_after_projects(
+                    state, &_preagg_block, &_child_eos,
+                    std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                      ExecNode::get_next,
+                              _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                              std::placeholders::_3)));
         };
         {
             if (_preagg_block.rows() != 0) {
@@ -626,7 +619,6 @@ void AggregationNode::release_resource(RuntimeState* state) 
{
 
 Status AggregationNode::close(RuntimeState* state) {
     if (is_closed()) return Status::OK();
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "AggregationNode::close");
     return ExecNode::close(state);
 }
 
diff --git a/be/src/vec/exec/vanalytic_eval_node.cpp 
b/be/src/vec/exec/vanalytic_eval_node.cpp
index 9ee4b424e2..528a624432 100644
--- a/be/src/vec/exec/vanalytic_eval_node.cpp
+++ b/be/src/vec/exec/vanalytic_eval_node.cpp
@@ -249,7 +249,6 @@ Status VAnalyticEvalNode::close(RuntimeState* state) {
 
 Status VAnalyticEvalNode::alloc_resource(RuntimeState* state) {
     {
-        START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VAnalyticEvalNode::open");
         SCOPED_TIMER(_runtime_profile->total_time_counter());
         RETURN_IF_ERROR(ExecNode::alloc_resource(state));
         RETURN_IF_CANCELLED(state);
@@ -297,7 +296,6 @@ void VAnalyticEvalNode::release_resource(RuntimeState* 
state) {
     if (is_closed()) {
         return;
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VAnalyticEvalNode::close");
 
     VExpr::close(_partition_by_eq_expr_ctxs, state);
     VExpr::close(_order_by_eq_expr_ctxs, state);
@@ -327,8 +325,6 @@ bool VAnalyticEvalNode::can_read() {
 }
 
 Status VAnalyticEvalNode::get_next(RuntimeState* state, vectorized::Block* 
block, bool* eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span,
-                                 "VAnalyticEvalNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_CANCELLED(state);
 
@@ -539,14 +535,12 @@ Status 
VAnalyticEvalNode::_fetch_next_block_data(RuntimeState* state) {
     Block block;
     RETURN_IF_CANCELLED(state);
     do {
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                _children[0]->get_next_after_projects(
-                        state, &block, &_input_eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                _children[0]->get_next_span(), _input_eos);
+        RETURN_IF_ERROR(_children[0]->get_next_after_projects(
+                state, &block, &_input_eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
     } while (!_input_eos && block.rows() == 0);
 
     RETURN_IF_ERROR(sink(state, &block, _input_eos));
diff --git a/be/src/vec/exec/vassert_num_rows_node.cpp 
b/be/src/vec/exec/vassert_num_rows_node.cpp
index 4b57ed0a05..84421180dd 100644
--- a/be/src/vec/exec/vassert_num_rows_node.cpp
+++ b/be/src/vec/exec/vassert_num_rows_node.cpp
@@ -53,7 +53,6 @@ VAssertNumRowsNode::VAssertNumRowsNode(ObjectPool* pool, 
const TPlanNode& tnode,
 }
 
 Status VAssertNumRowsNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VAssertNumRowsNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::open(state));
     // ISSUE-3435
@@ -108,17 +107,13 @@ Status VAssertNumRowsNode::pull(doris::RuntimeState* 
state, vectorized::Block* b
 }
 
 Status VAssertNumRowsNode::get_next(RuntimeState* state, Block* block, bool* 
eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span,
-                                 "VAssertNumRowsNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
-    RETURN_IF_ERROR_AND_CHECK_SPAN(
-            child(0)->get_next_after_projects(
-                    state, block, eos,
-                    std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                      ExecNode::get_next,
-                              _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                              std::placeholders::_3)),
-            child(0)->get_next_span(), *eos);
+    RETURN_IF_ERROR(child(0)->get_next_after_projects(
+            state, block, eos,
+            std::bind((Status(ExecNode::*)(RuntimeState*, vectorized::Block*, 
bool*)) &
+                              ExecNode::get_next,
+                      _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                      std::placeholders::_3)));
 
     return pull(state, block, eos);
 }
diff --git a/be/src/vec/exec/vempty_set_node.cpp 
b/be/src/vec/exec/vempty_set_node.cpp
index 1bee0aa5a5..787bb0990f 100644
--- a/be/src/vec/exec/vempty_set_node.cpp
+++ b/be/src/vec/exec/vempty_set_node.cpp
@@ -34,7 +34,6 @@ VEmptySetNode::VEmptySetNode(ObjectPool* pool, const 
TPlanNode& tnode, const Des
         : ExecNode(pool, tnode, descs) {}
 
 Status VEmptySetNode::get_next(RuntimeState* state, Block* block, bool* eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VEmptySetNode::get_next");
     *eos = true;
     return Status::OK();
 }
diff --git a/be/src/vec/exec/vexchange_node.cpp 
b/be/src/vec/exec/vexchange_node.cpp
index 8535f4a261..61fb57e06b 100644
--- a/be/src/vec/exec/vexchange_node.cpp
+++ b/be/src/vec/exec/vexchange_node.cpp
@@ -88,7 +88,6 @@ Status VExchangeNode::alloc_resource(RuntimeState* state) {
 }
 
 Status VExchangeNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VExchangeNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::open(state));
 
@@ -96,7 +95,6 @@ Status VExchangeNode::open(RuntimeState* state) {
 }
 
 Status VExchangeNode::get_next(RuntimeState* state, Block* block, bool* eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VExchangeNode::get_next");
     SCOPED_TIMER(runtime_profile()->total_time_counter());
     if (_is_merging && state->enable_pipeline_exec() && !_is_ready) {
         
RETURN_IF_ERROR(_stream_recvr->create_merger(_vsort_exec_exprs.lhs_ordering_expr_ctxs(),
@@ -152,8 +150,6 @@ Status VExchangeNode::close(RuntimeState* state) {
     if (is_closed()) {
         return Status::OK();
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VExchangeNode::close");
-
     return ExecNode::close(state);
 }
 
diff --git a/be/src/vec/exec/vmysql_scan_node.cpp 
b/be/src/vec/exec/vmysql_scan_node.cpp
index 36251e24d7..8673861f66 100644
--- a/be/src/vec/exec/vmysql_scan_node.cpp
+++ b/be/src/vec/exec/vmysql_scan_node.cpp
@@ -93,7 +93,6 @@ Status VMysqlScanNode::open(RuntimeState* state) {
     if (nullptr == state) {
         return Status::InternalError("input pointer is nullptr.");
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VMysqlScanNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::open(state));
     VLOG_CRITICAL << "MysqlScanNode::Open";
@@ -126,7 +125,6 @@ Status VMysqlScanNode::get_next(RuntimeState* state, 
vectorized::Block* block, b
     if (state == nullptr || block == nullptr || eos == nullptr) {
         return Status::InternalError("input is nullptr");
     }
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VMysqlScanNode::get_next");
     VLOG_CRITICAL << "VMysqlScanNode::GetNext";
 
     if (!_is_init) {
@@ -219,7 +217,6 @@ Status VMysqlScanNode::close(RuntimeState* state) {
     if (is_closed()) {
         return Status::OK();
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VMysqlScanNode::close");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
 
     return ExecNode::close(state);
diff --git a/be/src/vec/exec/vrepeat_node.cpp b/be/src/vec/exec/vrepeat_node.cpp
index 8cae519037..cb6a2278e6 100644
--- a/be/src/vec/exec/vrepeat_node.cpp
+++ b/be/src/vec/exec/vrepeat_node.cpp
@@ -81,7 +81,6 @@ Status VRepeatNode::prepare(RuntimeState* state) {
 }
 
 Status VRepeatNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VRepeatNode::open");
     VLOG_CRITICAL << "VRepeatNode::open";
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::open(state));
@@ -90,7 +89,6 @@ Status VRepeatNode::open(RuntimeState* state) {
 }
 
 Status VRepeatNode::alloc_resource(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VRepeatNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::alloc_resource(state));
     RETURN_IF_ERROR(VExpr::open(_expr_ctxs, state));
@@ -249,7 +247,6 @@ Status VRepeatNode::get_next(RuntimeState* state, Block* 
block, bool* eos) {
     if (state == nullptr || block == nullptr || eos == nullptr) {
         return Status::InternalError("input is nullptr");
     }
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VRepeatNode::get_next");
     VLOG_CRITICAL << "VRepeatNode::get_next";
     SCOPED_TIMER(_runtime_profile->total_time_counter());
 
@@ -260,14 +257,12 @@ Status VRepeatNode::get_next(RuntimeState* state, Block* 
block, bool* eos) {
     }
     DCHECK(block->rows() == 0);
     while (need_more_input_data()) {
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                child(0)->get_next_after_projects(
-                        state, &_child_block, &_child_eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                child(0)->get_next_span(), _child_eos);
+        RETURN_IF_ERROR(child(0)->get_next_after_projects(
+                state, &_child_block, &_child_eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
 
         push(state, &_child_block, _child_eos);
     }
@@ -284,7 +279,6 @@ Status VRepeatNode::close(RuntimeState* state) {
 }
 
 void VRepeatNode::release_resource(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VSortNode::close");
     VExpr::close(_expr_ctxs, state);
     ExecNode::release_resource(state);
 }
diff --git a/be/src/vec/exec/vschema_scan_node.cpp 
b/be/src/vec/exec/vschema_scan_node.cpp
index f79c77bc44..fd06f48ba6 100644
--- a/be/src/vec/exec/vschema_scan_node.cpp
+++ b/be/src/vec/exec/vschema_scan_node.cpp
@@ -21,8 +21,6 @@
 #include <gen_cpp/PlanNodes_types.h>
 #include <gen_cpp/Types_types.h>
 #include <opentelemetry/nostd/shared_ptr.h>
-#include <opentelemetry/trace/span.h>
-#include <opentelemetry/trace/span_metadata.h>
 
 #include <boost/algorithm/string/predicate.hpp>
 #include <ostream>
@@ -111,14 +109,11 @@ Status VSchemaScanNode::open(RuntimeState* state) {
         return Status::InternalError("input pointer is nullptr.");
     }
 
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VSchemaScanNode::open");
     if (!_is_init) {
-        span->SetStatus(opentelemetry::trace::StatusCode::kError, "Open before 
Init.");
         return Status::InternalError("Open before Init.");
     }
 
     if (nullptr == state) {
-        span->SetStatus(opentelemetry::trace::StatusCode::kError, "input 
pointer is nullptr.");
         return Status::InternalError("input pointer is nullptr.");
     }
 
@@ -145,7 +140,6 @@ Status VSchemaScanNode::prepare(RuntimeState* state) {
     if (nullptr == state) {
         return Status::InternalError("state pointer is nullptr.");
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VSchemaScanNode::prepare");
     RETURN_IF_ERROR(ScanNode::prepare(state));
 
     // get dest tuple desc
@@ -219,7 +213,6 @@ Status VSchemaScanNode::get_next(RuntimeState* state, 
vectorized::Block* block,
     if (state == nullptr || block == nullptr || eos == nullptr) {
         return Status::InternalError("input is NULL pointer");
     }
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VSchemaScanNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
 
     VLOG_CRITICAL << "VSchemaScanNode::GetNext";
@@ -290,7 +283,6 @@ Status VSchemaScanNode::close(RuntimeState* state) {
     if (is_closed()) {
         return Status::OK();
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VSchemaScanNode::close");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     return ExecNode::close(state);
 }
diff --git a/be/src/vec/exec/vselect_node.cpp b/be/src/vec/exec/vselect_node.cpp
index accc99e94c..c8b61ec94a 100644
--- a/be/src/vec/exec/vselect_node.cpp
+++ b/be/src/vec/exec/vselect_node.cpp
@@ -48,26 +48,22 @@ Status VSelectNode::prepare(RuntimeState* state) {
 }
 
 Status VSelectNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VSelectNode::open");
     RETURN_IF_ERROR(ExecNode::open(state));
     RETURN_IF_ERROR(child(0)->open(state));
     return Status::OK();
 }
 
 Status VSelectNode::get_next(RuntimeState* state, vectorized::Block* block, 
bool* eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VSelectNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_CANCELLED(state);
     do {
         RETURN_IF_CANCELLED(state);
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                _children[0]->get_next_after_projects(
-                        state, block, &_child_eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                _children[0]->get_next_span(), _child_eos);
+        RETURN_IF_ERROR(_children[0]->get_next_after_projects(
+                state, block, &_child_eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
         if (_child_eos) {
             *eos = true;
             break;
@@ -78,7 +74,6 @@ Status VSelectNode::get_next(RuntimeState* state, 
vectorized::Block* block, bool
 }
 
 Status VSelectNode::pull(RuntimeState* state, vectorized::Block* output_block, 
bool* eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VSelectNode::pull");
     RETURN_IF_CANCELLED(state);
     RETURN_IF_ERROR(
             VExprContext::filter_block(_vconjunct_ctx_ptr, output_block, 
output_block->columns()));
@@ -91,7 +86,6 @@ Status VSelectNode::close(RuntimeState* state) {
     if (is_closed()) {
         return Status::OK();
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VSelectNode::close");
     return ExecNode::close(state);
 }
 
diff --git a/be/src/vec/exec/vset_operation_node.cpp 
b/be/src/vec/exec/vset_operation_node.cpp
index a949f33d6e..4e83ffb7e8 100644
--- a/be/src/vec/exec/vset_operation_node.cpp
+++ b/be/src/vec/exec/vset_operation_node.cpp
@@ -181,7 +181,6 @@ Status VSetOperationNode<is_intersect>::close(RuntimeState* 
state) {
     if (is_closed()) {
         return Status::OK();
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VSetOperationNode<is_intersect>::close");
     return ExecNode::close(state);
 }
 
@@ -222,7 +221,6 @@ Status 
VSetOperationNode<is_intersect>::alloc_resource(RuntimeState* state) {
 
 template <bool is_intersect>
 Status VSetOperationNode<is_intersect>::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VSetOperationNode<is_intersect>::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::open(state));
 
@@ -237,16 +235,12 @@ Status 
VSetOperationNode<is_intersect>::open(RuntimeState* state) {
         while (!eos) {
             release_block_memory(_probe_block, i);
             RETURN_IF_CANCELLED(state);
-            RETURN_IF_ERROR_AND_CHECK_SPAN(
-                    child(i)->get_next_after_projects(
-                            state, &_probe_block, &eos,
-                            std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*,
-                                                           bool*)) &
-                                              ExecNode::get_next,
-                                      _children[i], std::placeholders::_1, 
std::placeholders::_2,
-                                      std::placeholders::_3)),
-                    child(i)->get_next_span(), eos);
-
+            RETURN_IF_ERROR(child(i)->get_next_after_projects(
+                    state, &_probe_block, &eos,
+                    std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                      ExecNode::get_next,
+                              _children[i], std::placeholders::_1, 
std::placeholders::_2,
+                              std::placeholders::_3)));
             RETURN_IF_ERROR(sink_probe(state, i, &_probe_block, eos));
         }
     }
@@ -257,7 +251,6 @@ template <bool is_intersect>
 Status VSetOperationNode<is_intersect>::get_next(RuntimeState* state, Block* 
output_block,
                                                  bool* eos) {
     SCOPED_TIMER(_runtime_profile->total_time_counter());
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VExceptNode::get_next");
     return pull(state, output_block, eos);
 }
 
@@ -447,14 +440,12 @@ Status 
VSetOperationNode<is_intersect>::hash_table_build(RuntimeState* state) {
     while (!eos) {
         block.clear_column_data();
         RETURN_IF_CANCELLED(state);
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                child(0)->get_next_after_projects(
-                        state, &block, &eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                child(0)->get_next_span(), eos);
+        RETURN_IF_ERROR(child(0)->get_next_after_projects(
+                state, &block, &eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
         if (eos) {
             child(0)->close(state);
         }
diff --git a/be/src/vec/exec/vsort_node.cpp b/be/src/vec/exec/vsort_node.cpp
index 252d9ec6d7..4c7bcb40d3 100644
--- a/be/src/vec/exec/vsort_node.cpp
+++ b/be/src/vec/exec/vsort_node.cpp
@@ -164,7 +164,6 @@ Status VSortNode::sink(RuntimeState* state, 
vectorized::Block* input_block, bool
 }
 
 Status VSortNode::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VSortNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_ERROR(ExecNode::open(state));
     RETURN_IF_ERROR(child(0)->open(state));
@@ -174,14 +173,12 @@ Status VSortNode::open(RuntimeState* state) {
     bool eos = false;
     std::unique_ptr<Block> upstream_block = Block::create_unique();
     do {
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                child(0)->get_next_after_projects(
-                        state, upstream_block.get(), &eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                child(0)->get_next_span(), eos);
+        RETURN_IF_ERROR(child(0)->get_next_after_projects(
+                state, upstream_block.get(), &eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*, bool*)) &
+                                  ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
         RETURN_IF_ERROR(sink(state, upstream_block.get(), eos));
     } while (!eos);
 
@@ -203,7 +200,6 @@ Status VSortNode::pull(doris::RuntimeState* state, 
vectorized::Block* output_blo
 }
 
 Status VSortNode::get_next(RuntimeState* state, Block* block, bool* eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VSortNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
 
     return pull(state, block, eos);
@@ -214,7 +210,6 @@ Status VSortNode::reset(RuntimeState* state) {
 }
 
 void VSortNode::release_resource(doris::RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VSortNode::close");
     _vsort_exec_exprs.close(state);
     _sorter = nullptr;
     ExecNode::release_resource(state);
diff --git a/be/src/vec/exec/vtable_function_node.cpp 
b/be/src/vec/exec/vtable_function_node.cpp
index 9f3fbccb28..ce80dbb931 100644
--- a/be/src/vec/exec/vtable_function_node.cpp
+++ b/be/src/vec/exec/vtable_function_node.cpp
@@ -133,22 +133,17 @@ Status VTableFunctionNode::prepare(RuntimeState* state) {
 }
 
 Status VTableFunctionNode::get_next(RuntimeState* state, Block* block, bool* 
eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span,
-                                 "VTableFunctionNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
 
     RETURN_IF_CANCELLED(state);
 
     // if child_block is empty, get data from child.
     while (need_more_input_data()) {
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                child(0)->get_next_after_projects(
-                        state, &_child_block, &_child_eos,
-                        std::bind((Status(ExecNode::*)(RuntimeState*, Block*, 
bool*)) &
-                                          ExecNode::get_next,
-                                  _children[0], std::placeholders::_1, 
std::placeholders::_2,
-                                  std::placeholders::_3)),
-                child(0)->get_next_span(), _child_eos);
+        RETURN_IF_ERROR(child(0)->get_next_after_projects(
+                state, &_child_block, &_child_eos,
+                std::bind((Status(ExecNode::*)(RuntimeState*, Block*, bool*)) 
& ExecNode::get_next,
+                          _children[0], std::placeholders::_1, 
std::placeholders::_2,
+                          std::placeholders::_3)));
 
         RETURN_IF_ERROR(push(state, &_child_block, _child_eos));
     }
diff --git a/be/src/vec/exec/vtable_function_node.h 
b/be/src/vec/exec/vtable_function_node.h
index acfd3e4e72..1a4f1438c3 100644
--- a/be/src/vec/exec/vtable_function_node.h
+++ b/be/src/vec/exec/vtable_function_node.h
@@ -55,7 +55,6 @@ public:
     Status init(const TPlanNode& tnode, RuntimeState* state = nullptr) 
override;
     Status prepare(RuntimeState* state) override;
     Status open(RuntimeState* state) override {
-        START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"TableFunctionNode::open");
         RETURN_IF_ERROR(alloc_resource(state));
         return _children[0]->open(state);
     }
diff --git a/be/src/vec/exec/vunion_node.cpp b/be/src/vec/exec/vunion_node.cpp
index d3b73a7ae1..6bee8ec6a5 100644
--- a/be/src/vec/exec/vunion_node.cpp
+++ b/be/src/vec/exec/vunion_node.cpp
@@ -105,7 +105,6 @@ Status VUnionNode::open(RuntimeState* state) {
 }
 
 Status VUnionNode::alloc_resource(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VUnionNode::open");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     // open const expr lists.
     for (const std::vector<VExprContext*>& exprs : _const_expr_lists) {
@@ -128,16 +127,14 @@ Status VUnionNode::get_next_pass_through(RuntimeState* 
state, Block* block) {
         _child_eos = false;
     }
     DCHECK_EQ(block->rows(), 0);
-    RETURN_IF_ERROR_AND_CHECK_SPAN(
-            child(_child_idx)
-                    ->get_next_after_projects(
-                            state, block, &_child_eos,
-                            std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*,
-                                                           bool*)) &
-                                              ExecNode::get_next,
-                                      _children[_child_idx], 
std::placeholders::_1,
-                                      std::placeholders::_2, 
std::placeholders::_3)),
-            child(_child_idx)->get_next_span(), _child_eos);
+    RETURN_IF_ERROR(child(_child_idx)
+                            ->get_next_after_projects(
+                                    state, block, &_child_eos,
+                                    
std::bind((Status(ExecNode::*)(RuntimeState*,
+                                                                   
vectorized::Block*, bool*)) &
+                                                      ExecNode::get_next,
+                                              _children[_child_idx], 
std::placeholders::_1,
+                                              std::placeholders::_2, 
std::placeholders::_3)));
     if (_child_eos) {
         // Even though the child is at eos, it's not OK to close() it here. 
Once we close
         // the child, the row batches that it produced are invalid. Marking 
the batch as
@@ -177,16 +174,14 @@ Status VUnionNode::get_next_materialized(RuntimeState* 
state, Block* block) {
         // Here need materialize block of child block, so here so not mem_reuse
         child_block.clear();
         // The first batch from each child is always fetched here.
-        RETURN_IF_ERROR_AND_CHECK_SPAN(
-                child(_child_idx)
-                        ->get_next_after_projects(
-                                state, &child_block, &_child_eos,
-                                std::bind((Status(ExecNode::*)(RuntimeState*, 
vectorized::Block*,
-                                                               bool*)) &
-                                                  ExecNode::get_next,
-                                          _children[_child_idx], 
std::placeholders::_1,
-                                          std::placeholders::_2, 
std::placeholders::_3)),
-                child(_child_idx)->get_next_span(), _child_eos);
+        RETURN_IF_ERROR(child(_child_idx)
+                                ->get_next_after_projects(
+                                        state, &child_block, &_child_eos,
+                                        
std::bind((Status(ExecNode::*)(RuntimeState*,
+                                                                       
vectorized::Block*, bool*)) &
+                                                          ExecNode::get_next,
+                                                  _children[_child_idx], 
std::placeholders::_1,
+                                                  std::placeholders::_2, 
std::placeholders::_3)));
         SCOPED_TIMER(_materialize_exprs_evaluate_timer);
         if (child_block.rows() > 0) {
             Block res;
@@ -280,7 +275,6 @@ Status VUnionNode::materialize_child_block(RuntimeState* 
state, int child_id,
 }
 
 Status VUnionNode::get_next(RuntimeState* state, Block* block, bool* eos) {
-    INIT_AND_SCOPE_GET_NEXT_SPAN(state->get_tracer(), _get_next_span, 
"VUnionNode::get_next");
     SCOPED_TIMER(_runtime_profile->total_time_counter());
     RETURN_IF_CANCELLED(state);
 
@@ -323,7 +317,6 @@ void VUnionNode::release_resource(RuntimeState* state) {
     if (is_closed()) {
         return;
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VUnionNode::close");
     for (auto& exprs : _const_expr_lists) {
         VExpr::close(exprs, state);
     }
diff --git a/be/src/vec/sink/vdata_stream_sender.cpp 
b/be/src/vec/sink/vdata_stream_sender.cpp
index b1d27d787b..48edab1911 100644
--- a/be/src/vec/sink/vdata_stream_sender.cpp
+++ b/be/src/vec/sink/vdata_stream_sender.cpp
@@ -480,7 +480,6 @@ Status VDataStreamSender::prepare(RuntimeState* state) {
 }
 
 Status VDataStreamSender::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VDataStreamSender::open");
     DCHECK(state != nullptr);
     int local_size = 0;
     for (int i = 0; i < _channels.size(); ++i) {
@@ -498,7 +497,6 @@ Status VDataStreamSender::open(RuntimeState* state) {
 }
 
 Status VDataStreamSender::send(RuntimeState* state, Block* block, bool eos) {
-    INIT_AND_SCOPE_SEND_SPAN(state->get_tracer(), _send_span, 
"VDataStreamSender::send")
     SCOPED_TIMER(_profile->total_time_counter());
     if (_part_type == TPartitionType::UNPARTITIONED || _channels.size() == 1) {
         // 1. serialize depends on it is not local exchange
@@ -635,7 +633,6 @@ Status VDataStreamSender::close(RuntimeState* state, Status 
exec_status) {
         return Status::OK();
     }
 
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, 
"VDataStreamSender::close");
     Status final_st = Status::OK();
     for (int i = 0; i < _channels.size(); ++i) {
         Status st = _channels[i]->close(state);
diff --git a/be/src/vec/sink/vjdbc_table_sink.cpp 
b/be/src/vec/sink/vjdbc_table_sink.cpp
index 38526e1e5e..ce7bc3f43d 100644
--- a/be/src/vec/sink/vjdbc_table_sink.cpp
+++ b/be/src/vec/sink/vjdbc_table_sink.cpp
@@ -62,7 +62,6 @@ Status VJdbcTableSink::init(const TDataSink& t_sink) {
 }
 
 Status VJdbcTableSink::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VJdbcTableSink::open");
     RETURN_IF_ERROR(VTableSink::open(state));
 
     // create writer
@@ -77,7 +76,6 @@ Status VJdbcTableSink::open(RuntimeState* state) {
 }
 
 Status VJdbcTableSink::send(RuntimeState* state, Block* block, bool eos) {
-    INIT_AND_SCOPE_SEND_SPAN(state->get_tracer(), _send_span, 
"VJdbcTableSink::send");
     Status status = Status::OK();
     if (block == nullptr || block->rows() == 0) {
         return status;
@@ -100,7 +98,6 @@ Status VJdbcTableSink::send(RuntimeState* state, Block* 
block, bool eos) {
 }
 
 Status VJdbcTableSink::close(RuntimeState* state, Status exec_status) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VJdbcTableSink::close");
     RETURN_IF_ERROR(VTableSink::close(state, exec_status));
     if (exec_status.ok() && _use_transaction) {
         RETURN_IF_ERROR(_writer->finish_trans());
diff --git a/be/src/vec/sink/vmysql_table_sink.cpp 
b/be/src/vec/sink/vmysql_table_sink.cpp
index e2ade983db..ee1c015c54 100644
--- a/be/src/vec/sink/vmysql_table_sink.cpp
+++ b/be/src/vec/sink/vmysql_table_sink.cpp
@@ -52,7 +52,6 @@ Status VMysqlTableSink::init(const TDataSink& t_sink) {
 }
 
 Status VMysqlTableSink::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VMysqlTableSink::open");
     // Prepare the exprs to run.
     RETURN_IF_ERROR(VTableSink::open(state));
     // create writer
@@ -62,12 +61,10 @@ Status VMysqlTableSink::open(RuntimeState* state) {
 }
 
 Status VMysqlTableSink::send(RuntimeState* state, Block* block, bool eos) {
-    INIT_AND_SCOPE_SEND_SPAN(state->get_tracer(), _send_span, 
"VMysqlTableSink::send");
     return _writer->append(block);
 }
 
 Status VMysqlTableSink::close(RuntimeState* state, Status exec_status) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VMysqlTableSink::close");
     RETURN_IF_ERROR(VTableSink::close(state, exec_status));
     return Status::OK();
 }
diff --git a/be/src/vec/sink/vodbc_table_sink.cpp 
b/be/src/vec/sink/vodbc_table_sink.cpp
index 2b8a1e8806..9bd445538f 100644
--- a/be/src/vec/sink/vodbc_table_sink.cpp
+++ b/be/src/vec/sink/vodbc_table_sink.cpp
@@ -52,7 +52,6 @@ Status VOdbcTableSink::init(const TDataSink& t_sink) {
 }
 
 Status VOdbcTableSink::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VOdbcTableSink::open");
     RETURN_IF_ERROR(VTableSink::open(state));
 
     // create writer
@@ -66,7 +65,6 @@ Status VOdbcTableSink::open(RuntimeState* state) {
 }
 
 Status VOdbcTableSink::send(RuntimeState* state, Block* block, bool eos) {
-    INIT_AND_SCOPE_SEND_SPAN(state->get_tracer(), _send_span, 
"VOdbcTableSink::send");
     Status status = Status::OK();
     if (block == nullptr || block->rows() == 0) {
         return status;
@@ -89,7 +87,6 @@ Status VOdbcTableSink::send(RuntimeState* state, Block* 
block, bool eos) {
 }
 
 Status VOdbcTableSink::close(RuntimeState* state, Status exec_status) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VOdbcTableSink::close");
     RETURN_IF_ERROR(VTableSink::close(state, exec_status));
     if (exec_status.ok() && _use_transaction) {
         RETURN_IF_ERROR(_writer->finish_trans());
diff --git a/be/src/vec/sink/vresult_file_sink.cpp 
b/be/src/vec/sink/vresult_file_sink.cpp
index 785daaaf01..a4b15f032a 100644
--- a/be/src/vec/sink/vresult_file_sink.cpp
+++ b/be/src/vec/sink/vresult_file_sink.cpp
@@ -142,7 +142,6 @@ Status VResultFileSink::prepare(RuntimeState* state) {
 }
 
 Status VResultFileSink::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VResultFileSink::open");
     if (!_is_top_sink) {
         RETURN_IF_ERROR(_stream_sender->open(state));
     }
@@ -150,7 +149,6 @@ Status VResultFileSink::open(RuntimeState* state) {
 }
 
 Status VResultFileSink::send(RuntimeState* state, Block* block, bool eos) {
-    INIT_AND_SCOPE_SEND_SPAN(state->get_tracer(), _send_span, 
"VResultFileSink::send");
     RETURN_IF_ERROR(_writer->append_block(*block));
     return Status::OK();
 }
@@ -160,7 +158,6 @@ Status VResultFileSink::close(RuntimeState* state, Status 
exec_status) {
         return Status::OK();
     }
 
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VResultFileSink::close");
     Status final_status = exec_status;
     // close the writer
     if (_writer) {
diff --git a/be/src/vec/sink/vresult_sink.cpp b/be/src/vec/sink/vresult_sink.cpp
index d61dc13875..297f201790 100644
--- a/be/src/vec/sink/vresult_sink.cpp
+++ b/be/src/vec/sink/vresult_sink.cpp
@@ -107,7 +107,6 @@ Status VResultSink::prepare(RuntimeState* state) {
 }
 
 Status VResultSink::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VResultSink::open");
     return VExpr::open(_output_vexpr_ctxs, state);
 }
 
@@ -126,7 +125,6 @@ Status VResultSink::second_phase_fetch_data(RuntimeState* 
state, Block* final_bl
 }
 
 Status VResultSink::send(RuntimeState* state, Block* block, bool eos) {
-    INIT_AND_SCOPE_SEND_SPAN(state->get_tracer(), _send_span, 
"VResultSink::send");
     if (_fetch_option.use_two_phase_fetch && block->rows() > 0) {
         RETURN_IF_ERROR(second_phase_fetch_data(state, block));
     }
@@ -144,7 +142,6 @@ Status VResultSink::close(RuntimeState* state, Status 
exec_status) {
         return Status::OK();
     }
 
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VResultSink::close");
     Status final_status = exec_status;
 
     if (_writer) {
diff --git a/be/src/vec/sink/vtable_sink.cpp b/be/src/vec/sink/vtable_sink.cpp
index 1a34c20e91..02259f0cfa 100644
--- a/be/src/vec/sink/vtable_sink.cpp
+++ b/be/src/vec/sink/vtable_sink.cpp
@@ -57,14 +57,12 @@ Status VTableSink::prepare(RuntimeState* state) {
 }
 
 Status VTableSink::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VTableSink::open");
     // Prepare the exprs to run.
     RETURN_IF_ERROR(VExpr::open(_output_vexpr_ctxs, state));
     return Status::OK();
 }
 
 Status VTableSink::send(RuntimeState* state, Block* block, bool eos) {
-    INIT_AND_SCOPE_SEND_SPAN(state->get_tracer(), _send_span, 
"VTableSink::send");
     return Status::OK();
 }
 Status VTableSink::close(RuntimeState* state, Status exec_status) {
diff --git a/be/src/vec/sink/vtablet_sink.cpp b/be/src/vec/sink/vtablet_sink.cpp
index a80758833e..f74c4f4e9a 100644
--- a/be/src/vec/sink/vtablet_sink.cpp
+++ b/be/src/vec/sink/vtablet_sink.cpp
@@ -1098,7 +1098,6 @@ Status VOlapTableSink::prepare(RuntimeState* state) {
 }
 
 Status VOlapTableSink::open(RuntimeState* state) {
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VOlapTableSink::open");
     // Prepare the exprs to run.
     RETURN_IF_ERROR(vectorized::VExpr::open(_output_vexpr_ctxs, state));
     SCOPED_TIMER(_profile->total_time_counter());
@@ -1255,7 +1254,6 @@ void VOlapTableSink::_generate_row_distribution_payload(
 }
 
 Status VOlapTableSink::send(RuntimeState* state, vectorized::Block* 
input_block, bool eos) {
-    INIT_AND_SCOPE_SEND_SPAN(state->get_tracer(), _send_span, 
"VOlapTableSink::send");
     SCOPED_CONSUME_MEM_TRACKER(_mem_tracker.get());
     Status status = Status::OK();
 
@@ -1373,7 +1371,6 @@ Status VOlapTableSink::close(RuntimeState* state, Status 
exec_status) {
     if (_closed) {
         return _close_status;
     }
-    START_AND_SCOPE_SPAN(state->get_tracer(), span, "VOlapTableSink::close");
     vectorized::VExpr::close(_output_vexpr_ctxs, state);
     Status status = exec_status;
     if (status.ok()) {


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

Reply via email to