Coverage Report

Created: 2026-05-19 20:37

next uncovered line (L), next uncovered region (R), next uncovered branch (B)
be/src/exec/pipeline/pipeline_fragment_context.cpp
Line
Count
Source
1
// Licensed to the Apache Software Foundation (ASF) under one
2
// or more contributor license agreements.  See the NOTICE file
3
// distributed with this work for additional information
4
// regarding copyright ownership.  The ASF licenses this file
5
// to you under the Apache License, Version 2.0 (the
6
// "License"); you may not use this file except in compliance
7
// with the License.  You may obtain a copy of the License at
8
//
9
//   http://www.apache.org/licenses/LICENSE-2.0
10
//
11
// Unless required by applicable law or agreed to in writing,
12
// software distributed under the License is distributed on an
13
// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
14
// KIND, either express or implied.  See the License for the
15
// specific language governing permissions and limitations
16
// under the License.
17
18
#include "exec/pipeline/pipeline_fragment_context.h"
19
20
#include <gen_cpp/DataSinks_types.h>
21
#include <gen_cpp/FrontendService.h>
22
#include <gen_cpp/FrontendService_types.h>
23
#include <gen_cpp/PaloInternalService_types.h>
24
#include <gen_cpp/PlanNodes_types.h>
25
#include <pthread.h>
26
27
#include <algorithm>
28
#include <cstdlib>
29
// IWYU pragma: no_include <bits/chrono.h>
30
#include <fmt/format.h>
31
#include <thrift/Thrift.h>
32
#include <thrift/protocol/TDebugProtocol.h>
33
#include <thrift/transport/TTransportException.h>
34
35
#include <chrono> // IWYU pragma: keep
36
#include <map>
37
#include <memory>
38
#include <ostream>
39
#include <utility>
40
41
#include "cloud/config.h"
42
#include "common/cast_set.h"
43
#include "common/config.h"
44
#include "common/exception.h"
45
#include "common/logging.h"
46
#include "common/status.h"
47
#include "exec/exchange/local_exchange_sink_operator.h"
48
#include "exec/exchange/local_exchange_source_operator.h"
49
#include "exec/exchange/local_exchanger.h"
50
#include "exec/exchange/vdata_stream_mgr.h"
51
#include "exec/operator/aggregation_sink_operator.h"
52
#include "exec/operator/aggregation_source_operator.h"
53
#include "exec/operator/analytic_sink_operator.h"
54
#include "exec/operator/analytic_source_operator.h"
55
#include "exec/operator/assert_num_rows_operator.h"
56
#include "exec/operator/blackhole_sink_operator.h"
57
#include "exec/operator/bucketed_aggregation_sink_operator.h"
58
#include "exec/operator/bucketed_aggregation_source_operator.h"
59
#include "exec/operator/cache_sink_operator.h"
60
#include "exec/operator/cache_source_operator.h"
61
#include "exec/operator/datagen_operator.h"
62
#include "exec/operator/dict_sink_operator.h"
63
#include "exec/operator/distinct_streaming_aggregation_operator.h"
64
#include "exec/operator/empty_set_operator.h"
65
#include "exec/operator/exchange_sink_operator.h"
66
#include "exec/operator/exchange_source_operator.h"
67
#include "exec/operator/file_scan_operator.h"
68
#include "exec/operator/group_commit_block_sink_operator.h"
69
#include "exec/operator/group_commit_scan_operator.h"
70
#include "exec/operator/hashjoin_build_sink.h"
71
#include "exec/operator/hashjoin_probe_operator.h"
72
#include "exec/operator/hive_table_sink_operator.h"
73
#include "exec/operator/iceberg_delete_sink_operator.h"
74
#include "exec/operator/iceberg_merge_sink_operator.h"
75
#include "exec/operator/iceberg_table_sink_operator.h"
76
#include "exec/operator/jdbc_scan_operator.h"
77
#include "exec/operator/jdbc_table_sink_operator.h"
78
#include "exec/operator/local_merge_sort_source_operator.h"
79
#include "exec/operator/materialization_opertor.h"
80
#include "exec/operator/maxcompute_table_sink_operator.h"
81
#include "exec/operator/memory_scratch_sink_operator.h"
82
#include "exec/operator/meta_scan_operator.h"
83
#include "exec/operator/multi_cast_data_stream_sink.h"
84
#include "exec/operator/multi_cast_data_stream_source.h"
85
#include "exec/operator/nested_loop_join_build_operator.h"
86
#include "exec/operator/nested_loop_join_probe_operator.h"
87
#include "exec/operator/olap_scan_operator.h"
88
#include "exec/operator/olap_table_sink_operator.h"
89
#include "exec/operator/olap_table_sink_v2_operator.h"
90
#include "exec/operator/partition_sort_sink_operator.h"
91
#include "exec/operator/partition_sort_source_operator.h"
92
#include "exec/operator/partitioned_aggregation_sink_operator.h"
93
#include "exec/operator/partitioned_aggregation_source_operator.h"
94
#include "exec/operator/partitioned_hash_join_probe_operator.h"
95
#include "exec/operator/partitioned_hash_join_sink_operator.h"
96
#include "exec/operator/rec_cte_anchor_sink_operator.h"
97
#include "exec/operator/rec_cte_scan_operator.h"
98
#include "exec/operator/rec_cte_sink_operator.h"
99
#include "exec/operator/rec_cte_source_operator.h"
100
#include "exec/operator/repeat_operator.h"
101
#include "exec/operator/result_file_sink_operator.h"
102
#include "exec/operator/result_sink_operator.h"
103
#include "exec/operator/schema_scan_operator.h"
104
#include "exec/operator/select_operator.h"
105
#include "exec/operator/set_probe_sink_operator.h"
106
#include "exec/operator/set_sink_operator.h"
107
#include "exec/operator/set_source_operator.h"
108
#include "exec/operator/sort_sink_operator.h"
109
#include "exec/operator/sort_source_operator.h"
110
#include "exec/operator/spill_iceberg_table_sink_operator.h"
111
#include "exec/operator/spill_sort_sink_operator.h"
112
#include "exec/operator/spill_sort_source_operator.h"
113
#include "exec/operator/streaming_aggregation_operator.h"
114
#include "exec/operator/table_function_operator.h"
115
#include "exec/operator/tvf_table_sink_operator.h"
116
#include "exec/operator/union_sink_operator.h"
117
#include "exec/operator/union_source_operator.h"
118
#include "exec/pipeline/dependency.h"
119
#include "exec/pipeline/pipeline_task.h"
120
#include "exec/pipeline/task_scheduler.h"
121
#include "exec/runtime_filter/runtime_filter_mgr.h"
122
#include "exec/sort/topn_sorter.h"
123
#include "exec/spill/spill_file.h"
124
#include "io/fs/stream_load_pipe.h"
125
#include "load/stream_load/new_load_stream_mgr.h"
126
#include "runtime/exec_env.h"
127
#include "runtime/fragment_mgr.h"
128
#include "runtime/result_buffer_mgr.h"
129
#include "runtime/runtime_state.h"
130
#include "runtime/thread_context.h"
131
#include "service/backend_options.h"
132
#include "util/client_cache.h"
133
#include "util/countdown_latch.h"
134
#include "util/debug_util.h"
135
#include "util/network_util.h"
136
#include "util/uid_util.h"
137
138
namespace doris {
139
PipelineFragmentContext::PipelineFragmentContext(
140
        TUniqueId query_id, const TPipelineFragmentParams& request,
141
        std::shared_ptr<QueryContext> query_ctx, ExecEnv* exec_env,
142
        const std::function<void(RuntimeState*, Status*)>& call_back)
143
27
        : _query_id(std::move(query_id)),
144
27
          _fragment_id(request.fragment_id),
145
27
          _exec_env(exec_env),
146
27
          _query_ctx(std::move(query_ctx)),
147
27
          _call_back(call_back),
148
27
          _is_report_on_cancel(true),
149
27
          _params(request),
150
27
          _parallel_instances(_params.__isset.parallel_instances ? _params.parallel_instances : 0),
151
27
          _need_notify_close(request.__isset.need_notify_close ? request.need_notify_close
152
27
                                                               : false) {
153
27
    _fragment_watcher.start();
154
27
}
155
156
27
PipelineFragmentContext::~PipelineFragmentContext() {
157
27
    LOG_INFO("PipelineFragmentContext::~PipelineFragmentContext")
158
27
            .tag("query_id", print_id(_query_id))
159
27
            .tag("fragment_id", _fragment_id);
160
27
    _release_resource();
161
27
    {
162
        // The memory released by the query end is recorded in the query mem tracker.
163
27
        SCOPED_SWITCH_THREAD_MEM_TRACKER_LIMITER(_query_ctx->query_mem_tracker());
164
27
        _runtime_state.reset();
165
27
        _query_ctx.reset();
166
27
    }
167
27
}
168
169
0
bool PipelineFragmentContext::is_timeout(timespec now) const {
170
0
    if (_timeout <= 0) {
171
0
        return false;
172
0
    }
173
0
    return _fragment_watcher.elapsed_time_seconds(now) > _timeout;
174
0
}
175
176
// notify_close() transitions the PFC from "waiting for external close notification" to
177
// "self-managed close". For recursive CTE fragments, the old PFC is kept alive until
178
// the rerun_fragment(wait_for_destroy) RPC calls this to trigger shutdown.
179
// Returns true if all tasks have already closed (i.e., the PFC can be safely destroyed).
180
0
bool PipelineFragmentContext::notify_close() {
181
0
    bool all_closed = false;
182
0
    bool need_remove = false;
183
0
    {
184
0
        std::lock_guard<std::mutex> l(_task_mutex);
185
0
        if (_closed_tasks >= _total_tasks) {
186
0
            if (_need_notify_close) {
187
                // Fragment was cancelled and waiting for notify to close.
188
                // Record that we need to remove from fragment mgr, but do it
189
                // after releasing _task_mutex to avoid ABBA deadlock with
190
                // dump_pipeline_tasks() (which acquires _pipeline_map lock
191
                // first, then _task_mutex via debug_string()).
192
0
                need_remove = true;
193
0
            }
194
0
            all_closed = true;
195
0
        }
196
        // make fragment release by self after cancel
197
0
        _need_notify_close = false;
198
0
    }
199
0
    if (need_remove) {
200
0
        _exec_env->fragment_mgr()->remove_pipeline_context({_query_id, _fragment_id});
201
0
    }
202
0
    return all_closed;
203
0
}
204
205
// Must not add lock in this method. Because it will call query ctx cancel. And
206
// QueryCtx cancel will call fragment ctx cancel. And Also Fragment ctx's running
207
// Method like exchange sink buffer will call query ctx cancel. If we add lock here
208
// There maybe dead lock.
209
0
void PipelineFragmentContext::cancel(const Status reason) {
210
0
    LOG_INFO("PipelineFragmentContext::cancel")
211
0
            .tag("query_id", print_id(_query_id))
212
0
            .tag("fragment_id", _fragment_id)
213
0
            .tag("reason", reason.to_string());
214
0
    if (notify_close()) {
215
0
        return;
216
0
    }
217
    // Timeout is a special error code, we need print current stack to debug timeout issue.
218
0
    if (reason.is<ErrorCode::TIMEOUT>()) {
219
0
        auto dbg_str = fmt::format("PipelineFragmentContext is cancelled due to timeout:\n{}",
220
0
                                   debug_string());
221
0
        LOG_LONG_STRING(WARNING, dbg_str);
222
0
    }
223
224
    // `ILLEGAL_STATE` means queries this fragment belongs to was not found in FE (maybe finished)
225
0
    if (reason.is<ErrorCode::ILLEGAL_STATE>()) {
226
0
        LOG_WARNING("PipelineFragmentContext is cancelled due to illegal state : {}",
227
0
                    debug_string());
228
0
    }
229
230
0
    if (reason.is<ErrorCode::MEM_LIMIT_EXCEEDED>() || reason.is<ErrorCode::MEM_ALLOC_FAILED>()) {
231
0
        print_profile("cancel pipeline, reason: " + reason.to_string());
232
0
    }
233
234
0
    if (auto error_url = get_load_error_url(); !error_url.empty()) {
235
0
        _query_ctx->set_load_error_url(error_url);
236
0
    }
237
238
0
    if (auto first_error_msg = get_first_error_msg(); !first_error_msg.empty()) {
239
0
        _query_ctx->set_first_error_msg(first_error_msg);
240
0
    }
241
242
0
    _query_ctx->cancel(reason, _fragment_id);
243
0
    if (reason.is<ErrorCode::LIMIT_REACH>()) {
244
0
        _is_report_on_cancel = false;
245
0
    } else {
246
0
        for (auto& id : _fragment_instance_ids) {
247
0
            LOG(WARNING) << "PipelineFragmentContext cancel instance: " << print_id(id);
248
0
        }
249
0
    }
250
    // Get pipe from new load stream manager and send cancel to it or the fragment may hang to wait read from pipe
251
    // For stream load the fragment's query_id == load id, it is set in FE.
252
0
    auto stream_load_ctx = _exec_env->new_load_stream_mgr()->get(_query_id);
253
0
    if (stream_load_ctx != nullptr) {
254
0
        stream_load_ctx->pipe->cancel(reason.to_string());
255
        // Set error URL here because after pipe is cancelled, stream load execution may return early.
256
        // We need to set the error URL at this point to ensure error information is properly
257
        // propagated to the client.
258
0
        stream_load_ctx->error_url = get_load_error_url();
259
0
        stream_load_ctx->first_error_msg = get_first_error_msg();
260
0
    }
261
262
0
    for (auto& tasks : _tasks) {
263
0
        for (auto& task : tasks) {
264
0
            task.first->unblock_all_dependencies();
265
0
        }
266
0
    }
267
0
}
268
269
0
PipelinePtr PipelineFragmentContext::add_pipeline(PipelinePtr parent, int idx) {
270
0
    PipelineId id = _next_pipeline_id++;
271
0
    auto pipeline = std::make_shared<Pipeline>(
272
0
            id, parent ? std::min(parent->num_tasks(), _num_instances) : _num_instances,
273
0
            parent ? parent->num_tasks() : _num_instances);
274
0
    if (idx >= 0) {
275
0
        _pipelines.insert(_pipelines.begin() + idx, pipeline);
276
0
    } else {
277
0
        _pipelines.emplace_back(pipeline);
278
0
    }
279
0
    if (parent) {
280
0
        parent->set_children(pipeline);
281
0
    }
282
0
    return pipeline;
283
0
}
284
285
0
Status PipelineFragmentContext::_build_and_prepare_full_pipeline(ThreadPool* thread_pool) {
286
0
    {
287
0
        SCOPED_TIMER(_build_pipelines_timer);
288
        // 2. Build pipelines with operators in this fragment.
289
0
        auto root_pipeline = add_pipeline();
290
0
        RETURN_IF_ERROR(_build_pipelines(_runtime_state->obj_pool(), *_query_ctx->desc_tbl,
291
0
                                         &_root_op, root_pipeline));
292
293
        // Propagate _num_instances from LOCAL_EXCHANGE pipelines to ancestor pipelines
294
        // that inherited reduced num_tasks from a serial operator.
295
0
        _propagate_local_exchange_num_tasks();
296
297
        // Create deferred local exchangers now that all pipelines have final num_tasks.
298
0
        RETURN_IF_ERROR(_create_deferred_local_exchangers());
299
300
        // Raise num_tasks for pipelines whose serial non-scan operators (e.g.,
301
        // UNPARTITIONED Exchange) reduced num_tasks below _num_instances.
302
        // Without this, fragment instances 1+ have no task for these pipelines
303
        // and downstream operators fail with "must set shared state".
304
        //
305
        // This applies to ALL pipelines (not just deferred exchanger upstreams):
306
        // fragments with UNION/INTERSECT/EXCEPT + serial Exchange in child
307
        // pipelines also need the raise, even without FE-planned local exchange.
308
        //
309
        // Exception: serial scan sources (pooling scan) keep num_tasks=1 — the
310
        // PassthroughExchanger(1, N) handles the fan-out correctly.
311
        // NOTE: Do NOT raise pipelines whose source is a serial operator
312
        // (Exchange or scan) — they legitimately have 1 task, and raising
313
        // them causes crashes (e.g., 4 Exchange tasks but only 1 receives
314
        // data).  The correct fix for shared state injection across
315
        // instances is handled by the FE: it inserts local exchange nodes
316
        // between serial operators and their downstream consumers, creating
317
        // proper pipeline boundaries with _num_instances tasks.
318
319
        // 3. Create sink operator
320
0
        if (!_params.fragment.__isset.output_sink) {
321
0
            return Status::InternalError("No output sink in this fragment!");
322
0
        }
323
0
        RETURN_IF_ERROR(_create_data_sink(_runtime_state->obj_pool(), _params.fragment.output_sink,
324
0
                                          _params.fragment.output_exprs, _params,
325
0
                                          root_pipeline->output_row_desc(), _runtime_state.get(),
326
0
                                          *_desc_tbl, root_pipeline->id()));
327
0
        RETURN_IF_ERROR(_sink->init(_params.fragment.output_sink));
328
0
        RETURN_IF_ERROR(root_pipeline->set_sink(_sink));
329
330
0
        for (PipelinePtr& pipeline : _pipelines) {
331
0
            DCHECK(pipeline->sink() != nullptr) << pipeline->operators().size();
332
0
            RETURN_IF_ERROR(pipeline->sink()->set_child(pipeline->operators().back()));
333
0
        }
334
0
    }
335
    // 4. Build local exchanger
336
0
    if (_runtime_state->plan_local_shuffle()) {
337
0
        SCOPED_TIMER(_plan_local_exchanger_timer);
338
0
        RETURN_IF_ERROR(_plan_local_exchange(_params.num_buckets,
339
0
                                             _params.bucket_seq_to_instance_idx,
340
0
                                             _params.shuffle_idx_to_instance_idx));
341
0
    }
342
343
    // 5. Initialize global states in pipelines.
344
0
    for (PipelinePtr& pipeline : _pipelines) {
345
0
        SCOPED_TIMER(_prepare_all_pipelines_timer);
346
0
        pipeline->children().clear();
347
0
        RETURN_IF_ERROR(pipeline->prepare(_runtime_state.get()));
348
0
    }
349
350
0
    {
351
0
        SCOPED_TIMER(_build_tasks_timer);
352
        // 6. Build pipeline tasks and initialize local state.
353
0
        RETURN_IF_ERROR(_build_pipeline_tasks(thread_pool));
354
0
    }
355
356
0
    return Status::OK();
357
0
}
358
359
0
Status PipelineFragmentContext::prepare(ThreadPool* thread_pool) {
360
0
    if (_prepared) {
361
0
        return Status::InternalError("Already prepared");
362
0
    }
363
0
    if (_params.__isset.query_options && _params.query_options.__isset.execution_timeout) {
364
0
        _timeout = _params.query_options.execution_timeout;
365
0
    }
366
367
0
    _fragment_level_profile = std::make_unique<RuntimeProfile>("PipelineContext");
368
0
    _prepare_timer = ADD_TIMER(_fragment_level_profile, "PrepareTime");
369
0
    SCOPED_TIMER(_prepare_timer);
370
0
    _build_pipelines_timer = ADD_TIMER(_fragment_level_profile, "BuildPipelinesTime");
371
0
    _init_context_timer = ADD_TIMER(_fragment_level_profile, "InitContextTime");
372
0
    _plan_local_exchanger_timer = ADD_TIMER(_fragment_level_profile, "PlanLocalLocalExchangerTime");
373
0
    _build_tasks_timer = ADD_TIMER(_fragment_level_profile, "BuildTasksTime");
374
0
    _prepare_all_pipelines_timer = ADD_TIMER(_fragment_level_profile, "PrepareAllPipelinesTime");
375
0
    {
376
0
        SCOPED_TIMER(_init_context_timer);
377
0
        cast_set(_num_instances, _params.local_params.size());
378
0
        _total_instances =
379
0
                _params.__isset.total_instances ? _params.total_instances : _num_instances;
380
381
0
        auto* fragment_context = this;
382
383
0
        if (_params.query_options.__isset.is_report_success) {
384
0
            fragment_context->set_is_report_success(_params.query_options.is_report_success);
385
0
        }
386
387
        // 1. Set up the global runtime state.
388
0
        _runtime_state = RuntimeState::create_unique(
389
0
                _params.query_id, _params.fragment_id, _params.query_options,
390
0
                _query_ctx->query_globals, _exec_env, _query_ctx.get());
391
0
        _runtime_state->set_task_execution_context(shared_from_this());
392
0
        SCOPED_SWITCH_THREAD_MEM_TRACKER_LIMITER(_runtime_state->query_mem_tracker());
393
0
        if (_params.__isset.backend_id) {
394
0
            _runtime_state->set_backend_id(_params.backend_id);
395
0
        }
396
0
        if (_params.__isset.import_label) {
397
0
            _runtime_state->set_import_label(_params.import_label);
398
0
        }
399
0
        if (_params.__isset.db_name) {
400
0
            _runtime_state->set_db_name(_params.db_name);
401
0
        }
402
0
        if (_params.__isset.load_job_id) {
403
0
            _runtime_state->set_load_job_id(_params.load_job_id);
404
0
        }
405
406
0
        if (_params.is_simplified_param) {
407
0
            _desc_tbl = _query_ctx->desc_tbl;
408
0
        } else {
409
0
            DCHECK(_params.__isset.desc_tbl);
410
0
            RETURN_IF_ERROR(DescriptorTbl::create(_runtime_state->obj_pool(), _params.desc_tbl,
411
0
                                                  &_desc_tbl));
412
0
        }
413
0
        _runtime_state->set_desc_tbl(_desc_tbl);
414
0
        _runtime_state->set_num_per_fragment_instances(_params.num_senders);
415
0
        _runtime_state->set_load_stream_per_node(_params.load_stream_per_node);
416
0
        _runtime_state->set_total_load_streams(_params.total_load_streams);
417
0
        _runtime_state->set_num_local_sink(_params.num_local_sink);
418
419
        // init fragment_instance_ids
420
0
        const auto target_size = _params.local_params.size();
421
0
        _fragment_instance_ids.resize(target_size);
422
0
        for (size_t i = 0; i < _params.local_params.size(); i++) {
423
0
            auto fragment_instance_id = _params.local_params[i].fragment_instance_id;
424
0
            _fragment_instance_ids[i] = fragment_instance_id;
425
0
        }
426
0
    }
427
428
0
    RETURN_IF_ERROR(_build_and_prepare_full_pipeline(thread_pool));
429
430
0
    _init_next_report_time();
431
432
0
    _prepared = true;
433
0
    return Status::OK();
434
0
}
435
436
Status PipelineFragmentContext::_build_pipeline_tasks_for_instance(
437
        int instance_idx,
438
0
        const std::vector<std::shared_ptr<RuntimeProfile>>& pipeline_id_to_profile) {
439
0
    const auto& local_params = _params.local_params[instance_idx];
440
0
    auto fragment_instance_id = local_params.fragment_instance_id;
441
0
    auto runtime_filter_mgr = std::make_unique<RuntimeFilterMgr>(false);
442
0
    std::map<PipelineId, PipelineTask*> pipeline_id_to_task;
443
0
    auto get_shared_state = [&](PipelinePtr pipeline)
444
0
            -> std::map<int, std::pair<std::shared_ptr<BasicSharedState>,
445
0
                                       std::vector<std::shared_ptr<Dependency>>>> {
446
0
        std::map<int, std::pair<std::shared_ptr<BasicSharedState>,
447
0
                                std::vector<std::shared_ptr<Dependency>>>>
448
0
                shared_state_map;
449
0
        for (auto& op : pipeline->operators()) {
450
0
            auto source_id = op->operator_id();
451
0
            if (auto iter = _op_id_to_shared_state.find(source_id);
452
0
                iter != _op_id_to_shared_state.end()) {
453
0
                shared_state_map.insert({source_id, iter->second});
454
0
            }
455
0
        }
456
0
        for (auto sink_to_source_id : pipeline->sink()->dests_id()) {
457
0
            if (auto iter = _op_id_to_shared_state.find(sink_to_source_id);
458
0
                iter != _op_id_to_shared_state.end()) {
459
0
                shared_state_map.insert({sink_to_source_id, iter->second});
460
0
            }
461
0
        }
462
0
        return shared_state_map;
463
0
    };
464
465
0
    for (size_t pip_idx = 0; pip_idx < _pipelines.size(); pip_idx++) {
466
0
        auto& pipeline = _pipelines[pip_idx];
467
0
        if (pipeline->num_tasks() > 1 || instance_idx == 0) {
468
0
            auto task_runtime_state = RuntimeState::create_unique(
469
0
                    local_params.fragment_instance_id, _params.query_id, _params.fragment_id,
470
0
                    _params.query_options, _query_ctx->query_globals, _exec_env, _query_ctx.get());
471
0
            {
472
                // Initialize runtime state for this task
473
0
                task_runtime_state->set_query_mem_tracker(_query_ctx->query_mem_tracker());
474
475
0
                task_runtime_state->set_task_execution_context(shared_from_this());
476
0
                task_runtime_state->set_be_number(local_params.backend_num);
477
478
0
                if (_params.__isset.backend_id) {
479
0
                    task_runtime_state->set_backend_id(_params.backend_id);
480
0
                }
481
0
                if (_params.__isset.import_label) {
482
0
                    task_runtime_state->set_import_label(_params.import_label);
483
0
                }
484
0
                if (_params.__isset.db_name) {
485
0
                    task_runtime_state->set_db_name(_params.db_name);
486
0
                }
487
0
                if (_params.__isset.load_job_id) {
488
0
                    task_runtime_state->set_load_job_id(_params.load_job_id);
489
0
                }
490
0
                if (_params.__isset.wal_id) {
491
0
                    task_runtime_state->set_wal_id(_params.wal_id);
492
0
                }
493
0
                if (_params.__isset.content_length) {
494
0
                    task_runtime_state->set_content_length(_params.content_length);
495
0
                }
496
497
0
                task_runtime_state->set_desc_tbl(_desc_tbl);
498
0
                task_runtime_state->set_per_fragment_instance_idx(local_params.sender_id);
499
0
                task_runtime_state->set_num_per_fragment_instances(_params.num_senders);
500
0
                task_runtime_state->resize_op_id_to_local_state(max_operator_id());
501
0
                task_runtime_state->set_max_operator_id(max_operator_id());
502
0
                task_runtime_state->set_load_stream_per_node(_params.load_stream_per_node);
503
0
                task_runtime_state->set_total_load_streams(_params.total_load_streams);
504
0
                task_runtime_state->set_num_local_sink(_params.num_local_sink);
505
506
0
                task_runtime_state->set_runtime_filter_mgr(runtime_filter_mgr.get());
507
0
            }
508
0
            auto cur_task_id = _total_tasks++;
509
0
            task_runtime_state->set_task_id(cur_task_id);
510
0
            task_runtime_state->set_task_num(pipeline->num_tasks());
511
0
            auto task = std::make_shared<PipelineTask>(
512
0
                    pipeline, cur_task_id, task_runtime_state.get(),
513
0
                    std::dynamic_pointer_cast<PipelineFragmentContext>(shared_from_this()),
514
0
                    pipeline_id_to_profile[pip_idx].get(), get_shared_state(pipeline),
515
0
                    instance_idx);
516
0
            pipeline->incr_created_tasks(instance_idx, task.get());
517
0
            pipeline_id_to_task.insert({pipeline->id(), task.get()});
518
0
            _tasks[instance_idx].emplace_back(
519
0
                    std::pair<std::shared_ptr<PipelineTask>, std::unique_ptr<RuntimeState>> {
520
0
                            std::move(task), std::move(task_runtime_state)});
521
0
        }
522
0
    }
523
524
    /**
525
         * Build DAG for pipeline tasks.
526
         * For example, we have
527
         *
528
         *   ExchangeSink (Pipeline1)     JoinBuildSink (Pipeline2)
529
         *            \                      /
530
         *          JoinProbeOperator1 (Pipeline1)    JoinBuildSink (Pipeline3)
531
         *                 \                          /
532
         *               JoinProbeOperator2 (Pipeline1)
533
         *
534
         * In this fragment, we have three pipelines and pipeline 1 depends on pipeline 2 and pipeline 3.
535
         * To build this DAG, `_dag` manage dependencies between pipelines by pipeline ID and
536
         * `pipeline_id_to_task` is used to find the task by a unique pipeline ID.
537
         *
538
         * Finally, we have two upstream dependencies in Pipeline1 corresponding to JoinProbeOperator1
539
         * and JoinProbeOperator2.
540
         */
541
0
    for (auto& _pipeline : _pipelines) {
542
0
        if (pipeline_id_to_task.contains(_pipeline->id())) {
543
0
            auto* task = pipeline_id_to_task[_pipeline->id()];
544
0
            DCHECK(task != nullptr);
545
546
            // If this task has upstream dependency, then inject it into this task.
547
0
            if (_dag.contains(_pipeline->id())) {
548
0
                auto& deps = _dag[_pipeline->id()];
549
0
                for (auto& dep : deps) {
550
0
                    if (pipeline_id_to_task.contains(dep)) {
551
0
                        auto ss = pipeline_id_to_task[dep]->get_sink_shared_state();
552
0
                        if (ss) {
553
0
                            task->inject_shared_state(ss);
554
0
                        } else {
555
0
                            pipeline_id_to_task[dep]->inject_shared_state(
556
0
                                    task->get_source_shared_state());
557
0
                        }
558
0
                    }
559
0
                }
560
0
            }
561
0
        }
562
0
    }
563
0
    for (size_t pip_idx = 0; pip_idx < _pipelines.size(); pip_idx++) {
564
0
        if (pipeline_id_to_task.contains(_pipelines[pip_idx]->id())) {
565
0
            auto* task = pipeline_id_to_task[_pipelines[pip_idx]->id()];
566
0
            DCHECK(pipeline_id_to_profile[pip_idx]);
567
0
            std::vector<TScanRangeParams> scan_ranges;
568
0
            auto node_id = _pipelines[pip_idx]->operators().front()->node_id();
569
0
            if (local_params.per_node_scan_ranges.contains(node_id)) {
570
0
                scan_ranges = local_params.per_node_scan_ranges.find(node_id)->second;
571
0
            }
572
0
            RETURN_IF_ERROR_OR_CATCH_EXCEPTION(task->prepare(scan_ranges, local_params.sender_id,
573
0
                                                             _params.fragment.output_sink));
574
0
        }
575
0
    }
576
0
    {
577
0
        std::lock_guard<std::mutex> l(_state_map_lock);
578
0
        _runtime_filter_mgr_map[instance_idx] = std::move(runtime_filter_mgr);
579
0
    }
580
0
    return Status::OK();
581
0
}
582
583
0
Status PipelineFragmentContext::_build_pipeline_tasks(ThreadPool* thread_pool) {
584
0
    _total_tasks = 0;
585
0
    _closed_tasks = 0;
586
0
    const auto target_size = _params.local_params.size();
587
0
    _tasks.resize(target_size);
588
0
    _runtime_filter_mgr_map.resize(target_size);
589
0
    for (size_t pip_idx = 0; pip_idx < _pipelines.size(); pip_idx++) {
590
0
        _pip_id_to_pipeline[_pipelines[pip_idx]->id()] = _pipelines[pip_idx].get();
591
0
    }
592
0
    auto pipeline_id_to_profile = _runtime_state->build_pipeline_profile(_pipelines.size());
593
594
0
    if (target_size > 1 &&
595
0
        (_runtime_state->query_options().__isset.parallel_prepare_threshold &&
596
0
         target_size > _runtime_state->query_options().parallel_prepare_threshold)) {
597
        // If instances parallelism is big enough ( > parallel_prepare_threshold), we will prepare all tasks by multi-threads
598
0
        std::vector<Status> prepare_status(target_size);
599
0
        int submitted_tasks = 0;
600
0
        Status submit_status;
601
0
        CountDownLatch latch((int)target_size);
602
0
        for (int i = 0; i < target_size; i++) {
603
0
            submit_status = thread_pool->submit_func([&, i]() {
604
0
                SCOPED_ATTACH_TASK(_query_ctx.get());
605
0
                prepare_status[i] = _build_pipeline_tasks_for_instance(i, pipeline_id_to_profile);
606
0
                latch.count_down();
607
0
            });
608
0
            if (LIKELY(submit_status.ok())) {
609
0
                submitted_tasks++;
610
0
            } else {
611
0
                break;
612
0
            }
613
0
        }
614
0
        latch.arrive_and_wait(target_size - submitted_tasks);
615
0
        if (UNLIKELY(!submit_status.ok())) {
616
0
            return submit_status;
617
0
        }
618
0
        for (int i = 0; i < submitted_tasks; i++) {
619
0
            if (!prepare_status[i].ok()) {
620
0
                return prepare_status[i];
621
0
            }
622
0
        }
623
0
    } else {
624
0
        for (int i = 0; i < target_size; i++) {
625
0
            RETURN_IF_ERROR(_build_pipeline_tasks_for_instance(i, pipeline_id_to_profile));
626
0
        }
627
0
    }
628
0
    _pipeline_parent_map.clear();
629
0
    _op_id_to_shared_state.clear();
630
    // Record task cardinality once when this fragment context finishes task initialization.
631
0
    _query_ctx->add_total_task_num(_total_tasks.load(std::memory_order_relaxed));
632
633
0
    return Status::OK();
634
0
}
635
636
0
void PipelineFragmentContext::_init_next_report_time() {
637
0
    auto interval_s = config::pipeline_status_report_interval;
638
0
    if (_is_report_success && interval_s > 0 && _timeout > interval_s) {
639
0
        VLOG_FILE << "enable period report: fragment id=" << _fragment_id;
640
0
        uint64_t report_fragment_offset = (uint64_t)(rand() % interval_s) * NANOS_PER_SEC;
641
        // We don't want to wait longer than it takes to run the entire fragment.
642
0
        _previous_report_time =
643
0
                MonotonicNanos() + report_fragment_offset - (uint64_t)(interval_s)*NANOS_PER_SEC;
644
0
        _disable_period_report = false;
645
0
    }
646
0
}
647
648
0
void PipelineFragmentContext::refresh_next_report_time() {
649
0
    auto disable = _disable_period_report.load(std::memory_order_acquire);
650
0
    DCHECK(disable == true);
651
0
    _previous_report_time.store(MonotonicNanos(), std::memory_order_release);
652
0
    _disable_period_report.compare_exchange_strong(disable, false);
653
0
}
654
655
0
void PipelineFragmentContext::trigger_report_if_necessary() {
656
0
    if (!_is_report_success) {
657
0
        return;
658
0
    }
659
0
    auto disable = _disable_period_report.load(std::memory_order_acquire);
660
0
    if (disable) {
661
0
        return;
662
0
    }
663
0
    int32_t interval_s = config::pipeline_status_report_interval;
664
0
    if (interval_s <= 0) {
665
0
        LOG(WARNING) << "config::status_report_interval is equal to or less than zero, do not "
666
0
                        "trigger "
667
0
                        "report.";
668
0
    }
669
0
    uint64_t next_report_time = _previous_report_time.load(std::memory_order_acquire) +
670
0
                                (uint64_t)(interval_s)*NANOS_PER_SEC;
671
0
    if (MonotonicNanos() > next_report_time) {
672
0
        if (!_disable_period_report.compare_exchange_strong(disable, true,
673
0
                                                            std::memory_order_acq_rel)) {
674
0
            return;
675
0
        }
676
0
        if (VLOG_FILE_IS_ON) {
677
0
            VLOG_FILE << "Reporting "
678
0
                      << "profile for query_id " << print_id(_query_id)
679
0
                      << ", fragment id: " << _fragment_id;
680
681
0
            std::stringstream ss;
682
0
            _runtime_state->runtime_profile()->compute_time_in_profile();
683
0
            _runtime_state->runtime_profile()->pretty_print(&ss);
684
0
            if (_runtime_state->load_channel_profile()) {
685
0
                _runtime_state->load_channel_profile()->pretty_print(&ss);
686
0
            }
687
688
0
            VLOG_FILE << "Query " << print_id(get_query_id()) << " fragment " << get_fragment_id()
689
0
                      << " profile:\n"
690
0
                      << ss.str();
691
0
        }
692
0
        auto st = send_report(false);
693
0
        if (!st.ok()) {
694
0
            disable = true;
695
0
            _disable_period_report.compare_exchange_strong(disable, false,
696
0
                                                           std::memory_order_acq_rel);
697
0
        }
698
0
    }
699
0
}
700
701
Status PipelineFragmentContext::_build_pipelines(ObjectPool* pool, const DescriptorTbl& descs,
702
0
                                                 OperatorPtr* root, PipelinePtr cur_pipe) {
703
0
    if (_params.fragment.plan.nodes.empty()) {
704
0
        throw Exception(ErrorCode::INTERNAL_ERROR, "Invalid plan which has no plan node!");
705
0
    }
706
707
0
    int node_idx = 0;
708
709
0
    RETURN_IF_ERROR(_create_tree_helper(pool, _params.fragment.plan.nodes, descs, nullptr,
710
0
                                        &node_idx, root, cur_pipe, 0, false, false));
711
712
0
    if (node_idx + 1 != _params.fragment.plan.nodes.size()) {
713
0
        return Status::InternalError(
714
0
                "Plan tree only partially reconstructed. Not all thrift nodes were used.");
715
0
    }
716
0
    return Status::OK();
717
0
}
718
719
0
Status PipelineFragmentContext::_create_deferred_local_exchangers() {
720
0
    for (auto& info : _deferred_exchangers) {
721
0
        const int sender_count = info.upstream_pipe->num_tasks();
722
0
        switch (info.partition_type) {
723
0
        case TLocalPartitionType::LOCAL_EXECUTION_HASH_SHUFFLE:
724
0
        case TLocalPartitionType::GLOBAL_EXECUTION_HASH_SHUFFLE:
725
0
            info.shared_state->exchanger = ShuffleExchanger::create_unique(
726
0
                    sender_count, _num_instances, info.num_partitions, info.free_blocks_limit,
727
0
                    info.partition_type);
728
0
            break;
729
0
        case TLocalPartitionType::BUCKET_HASH_SHUFFLE:
730
0
            info.shared_state->exchanger = BucketShuffleExchanger::create_unique(
731
0
                    sender_count, _num_instances, info.num_partitions, info.free_blocks_limit);
732
0
            break;
733
0
        case TLocalPartitionType::PASSTHROUGH:
734
0
            info.shared_state->exchanger = PassthroughExchanger::create_unique(
735
0
                    sender_count, _num_instances, info.free_blocks_limit);
736
0
            break;
737
0
        case TLocalPartitionType::BROADCAST:
738
0
            info.shared_state->exchanger = BroadcastExchanger::create_unique(
739
0
                    sender_count, _num_instances, info.free_blocks_limit);
740
0
            break;
741
0
        case TLocalPartitionType::PASS_TO_ONE:
742
0
            if (_runtime_state->enable_share_hash_table_for_broadcast_join()) {
743
0
                info.shared_state->exchanger = PassToOneExchanger::create_unique(
744
0
                        sender_count, _num_instances, info.free_blocks_limit);
745
0
            } else {
746
0
                info.shared_state->exchanger = BroadcastExchanger::create_unique(
747
0
                        sender_count, _num_instances, info.free_blocks_limit);
748
0
            }
749
0
            break;
750
0
        case TLocalPartitionType::ADAPTIVE_PASSTHROUGH:
751
0
            info.shared_state->exchanger = AdaptivePassthroughExchanger::create_unique(
752
0
                    sender_count, _num_instances, info.free_blocks_limit);
753
0
            break;
754
0
        case TLocalPartitionType::NOOP:
755
0
        case TLocalPartitionType::LOCAL_MERGE_SORT:
756
            // FE-planned LocalExchangeNode currently never emits NOOP or LOCAL_MERGE_SORT
757
            // through the deferred-exchanger path.  NOOP means "no exchange needed" and
758
            // is filtered out before reaching here; LOCAL_MERGE_SORT is planned by the
759
            // legacy BE path only.  Crash in debug to surface the protocol violation if
760
            // that ever changes; return an error in release to avoid silently corrupting
761
            // execution.
762
0
            DCHECK(false) << "FE-planned local exchange should not emit partition_type="
763
0
                          << static_cast<int>(info.partition_type);
764
0
            return Status::InternalError("FE-planned local exchange emitted unsupported type: " +
765
0
                                         std::to_string(static_cast<int>(info.partition_type)));
766
0
        default:
767
            // New TLocalPartitionType added on FE side without a BE handler here.
768
0
            DCHECK(false) << "Unhandled TLocalPartitionType in deferred exchangers: "
769
0
                          << static_cast<int>(info.partition_type);
770
0
            return Status::InternalError("Unsupported FE-planned local exchange type: " +
771
0
                                         std::to_string(static_cast<int>(info.partition_type)));
772
0
        }
773
0
    }
774
0
    _deferred_exchangers.clear();
775
0
    return Status::OK();
776
0
}
777
778
0
void PipelineFragmentContext::_propagate_local_exchange_num_tasks() {
779
    // Only runs when FE has planned local exchanges and BE deferred their construction.
780
    // In legacy mode (enable_local_shuffle_planner=false) BE plans LE itself via
781
    // _plan_local_exchange and _deferred_exchangers stays empty — the legacy path
782
    // already gets its num_tasks right at construction time, so the propagate passes
783
    // would be no-ops and are skipped.  This is a transitional design: once the FE
784
    // planner is the only planner, the propagation logic itself should degrade into
785
    // a pure assertion that the FE plan already wired the right num_tasks everywhere.
786
0
    if (_deferred_exchangers.empty()) {
787
0
        return;
788
0
    }
789
    // Fix num_tasks mismatches between paired pipelines created by pipeline-splitting
790
    // operators (AGG, SORT, JOIN).  These pairs share state via inject_shared_state and
791
    // must have consistent num_tasks; otherwise instance 1+ tasks access null shared_state.
792
    //
793
    // Pass 1 (upward): when FE inserts LOCAL_EXCHANGE_NODE below a splitting operator,
794
    // the LE pipeline gets _num_instances but ancestor pipelines still have the reduced
795
    // count.  Raise them to _num_instances (skip serial source operators).
796
    //
797
    // Pass 2 (downward): when a serial Exchange feeds a splitting operator (e.g., scalar
798
    // merge-finalize AGG), the sink pipeline has 1 task but the source pipeline may have
799
    // been raised (by LE or by inheriting _num_instances from add_pipeline).  Lower it
800
    // to match, unless it contains LocalExchangeSource (deliberately raised by LE).
801
0
    std::unordered_map<PipelineId, PipelinePtr> id_to_pipe;
802
0
    for (auto& p : _pipelines) {
803
0
        id_to_pipe[p->id()] = p;
804
0
    }
805
806
    // Pass 1 (upward): raise downstream pipelines to _num_instances when any upstream
807
    // dependency was already raised by LOCAL_EXCHANGE.
808
0
    bool changed = true;
809
0
    while (changed) {
810
0
        changed = false;
811
0
        for (const auto& [downstream_id, upstream_ids] : _dag) {
812
0
            auto dit = id_to_pipe.find(downstream_id);
813
0
            if (dit == id_to_pipe.end()) {
814
0
                continue;
815
0
            }
816
0
            auto& downstream = dit->second;
817
0
            if (downstream->num_tasks() >= _num_instances) {
818
0
                continue;
819
0
            }
820
0
            if (!downstream->operators().empty() &&
821
0
                downstream->operators().front()->is_serial_operator()) {
822
0
                continue;
823
0
            }
824
0
            for (auto upstream_id : upstream_ids) {
825
0
                auto uit = id_to_pipe.find(upstream_id);
826
0
                if (uit != id_to_pipe.end() && uit->second->num_tasks() >= _num_instances) {
827
0
                    downstream->set_num_tasks(_num_instances);
828
0
                    changed = true;
829
0
                    break;
830
0
                }
831
0
            }
832
0
        }
833
0
    }
834
835
    // Pass 2 (downward): when a pipeline-splitting operator (AGG, SORT, JOIN) creates
836
    // paired pipelines (source + sink), they share state via inject_shared_state and must
837
    // have consistent num_tasks.  If the sink pipeline has a serial source (Exchange) with
838
    // num_tasks < _num_instances, but the source pipeline was raised (by inheriting from
839
    // an LE-fixed parent during add_pipeline), lower the source pipeline to match.
840
    // Skip pipelines containing LocalExchangeSource — they were deliberately raised.
841
    //
842
    // Example: serial Exchange → AGG(merge finalize) → LOCAL_EXCHANGE → NLJ
843
    //   P_sink  = [Exchange, AGGSink]   num_tasks=1  (serial)
844
    //   P_source = [AGGSource]          num_tasks=_num_instances (inherited)
845
    //   → lower P_source to 1; the LE above fans out the 1 result to N NLJ tasks.
846
0
    changed = true;
847
0
    while (changed) {
848
0
        changed = false;
849
0
        for (const auto& [downstream_id, upstream_ids] : _dag) {
850
0
            auto dit = id_to_pipe.find(downstream_id);
851
0
            if (dit == id_to_pipe.end()) {
852
0
                continue;
853
0
            }
854
0
            auto& downstream = dit->second;
855
0
            if (downstream->num_tasks() <= 1) {
856
0
                continue;
857
0
            }
858
            // Don't lower pipelines that contain LocalExchangeSource — they were
859
            // deliberately raised to _num_instances by LOCAL_EXCHANGE processing.
860
0
            bool has_le_source = false;
861
0
            for (auto& op : downstream->operators()) {
862
0
                if (dynamic_cast<LocalExchangeSourceOperatorX*>(op.get())) {
863
0
                    has_le_source = true;
864
0
                    break;
865
0
                }
866
0
            }
867
0
            if (has_le_source) {
868
0
                continue;
869
0
            }
870
0
            for (auto upstream_id : upstream_ids) {
871
0
                auto uit = id_to_pipe.find(upstream_id);
872
0
                if (uit != id_to_pipe.end() && uit->second->num_tasks() < downstream->num_tasks() &&
873
0
                    !uit->second->operators().empty() &&
874
0
                    uit->second->operators().front()->is_serial_operator()) {
875
0
                    downstream->set_num_tasks(uit->second->num_tasks());
876
0
                    changed = true;
877
0
                    break;
878
0
                }
879
0
            }
880
0
        }
881
0
    }
882
0
}
883
884
Status PipelineFragmentContext::_create_tree_helper(
885
        ObjectPool* pool, const std::vector<TPlanNode>& tnodes, const DescriptorTbl& descs,
886
        OperatorPtr parent, int* node_idx, OperatorPtr* root, PipelinePtr& cur_pipe, int child_idx,
887
0
        const bool followed_by_shuffled_operator, const bool require_bucket_distribution) {
888
    // propagate error case
889
0
    if (*node_idx >= tnodes.size()) {
890
0
        return Status::InternalError(
891
0
                "Failed to reconstruct plan tree from thrift. Node id: {}, number of nodes: {}",
892
0
                *node_idx, tnodes.size());
893
0
    }
894
0
    const TPlanNode& tnode = tnodes[*node_idx];
895
896
0
    int num_children = tnodes[*node_idx].num_children;
897
0
    bool current_followed_by_shuffled_operator = followed_by_shuffled_operator;
898
0
    bool current_require_bucket_distribution = require_bucket_distribution;
899
    // TODO: Create CacheOperator is confused now
900
0
    OperatorPtr op = nullptr;
901
0
    OperatorPtr cache_op = nullptr;
902
0
    RETURN_IF_ERROR(_create_operator(pool, tnodes[*node_idx], descs, op, cur_pipe,
903
0
                                     parent == nullptr ? -1 : parent->node_id(), child_idx,
904
0
                                     followed_by_shuffled_operator,
905
0
                                     current_require_bucket_distribution, cache_op));
906
    // Initialization must be done here. For example, group by expressions in agg will be used to
907
    // decide if a local shuffle should be planed, so it must be initialized here.
908
0
    RETURN_IF_ERROR(op->init(tnode, _runtime_state.get()));
909
    // assert(parent != nullptr || (node_idx == 0 && root_expr != nullptr));
910
0
    if (parent != nullptr) {
911
        // add to parent's child(s)
912
0
        RETURN_IF_ERROR(parent->set_child(cache_op ? cache_op : op));
913
0
    } else {
914
0
        *root = op;
915
0
    }
916
    /**
917
     * `TLocalPartitionType::HASH_SHUFFLE` should be used if an operator is followed by a shuffled operator (shuffled hash join, union operator followed by co-located operators).
918
     *
919
     * For plan:
920
     * LocalExchange(id=0) -> Aggregation(id=1) -> ShuffledHashJoin(id=2)
921
     *                           Exchange(id=3) -> ShuffledHashJoinBuild(id=2)
922
     * We must ensure data distribution of `LocalExchange(id=0)` is same as Exchange(id=3).
923
     *
924
     * If an operator's is followed by a local exchange without shuffle (e.g. passthrough), a
925
     * shuffled local exchanger will be used before join so it is not followed by shuffle join.
926
     */
927
0
    auto required_data_distribution =
928
0
            cur_pipe->operators().empty()
929
0
                    ? cur_pipe->sink()->required_data_distribution(_runtime_state.get())
930
0
                    : op->required_data_distribution(_runtime_state.get());
931
0
    current_followed_by_shuffled_operator =
932
0
            ((followed_by_shuffled_operator ||
933
0
              (cur_pipe->operators().empty() ? cur_pipe->sink()->is_shuffled_operator()
934
0
                                             : op->is_shuffled_operator())) &&
935
0
             Pipeline::is_hash_exchange(required_data_distribution.distribution_type)) ||
936
0
            (followed_by_shuffled_operator &&
937
0
             required_data_distribution.distribution_type == TLocalPartitionType::NOOP);
938
939
0
    current_require_bucket_distribution =
940
0
            ((require_bucket_distribution ||
941
0
              (cur_pipe->operators().empty() ? cur_pipe->sink()->is_colocated_operator()
942
0
                                             : op->is_colocated_operator())) &&
943
0
             Pipeline::is_hash_exchange(required_data_distribution.distribution_type)) ||
944
0
            (require_bucket_distribution &&
945
0
             required_data_distribution.distribution_type == TLocalPartitionType::NOOP);
946
947
0
    if (num_children == 0) {
948
0
        _use_serial_source = op->is_serial_operator();
949
0
    }
950
    // rely on that tnodes is preorder of the plan
951
0
    for (int i = 0; i < num_children; i++) {
952
0
        ++*node_idx;
953
0
        RETURN_IF_ERROR(_create_tree_helper(pool, tnodes, descs, op, node_idx, nullptr, cur_pipe, i,
954
0
                                            current_followed_by_shuffled_operator,
955
0
                                            current_require_bucket_distribution));
956
957
        // we are expecting a child, but have used all nodes
958
        // this means we have been given a bad tree and must fail
959
0
        if (*node_idx >= tnodes.size()) {
960
0
            return Status::InternalError(
961
0
                    "Failed to reconstruct plan tree from thrift. Node id: {}, number of "
962
0
                    "nodes: {}",
963
0
                    *node_idx, tnodes.size());
964
0
        }
965
0
    }
966
967
0
    return Status::OK();
968
0
}
969
970
void PipelineFragmentContext::_inherit_pipeline_properties(
971
        const DataDistribution& data_distribution, PipelinePtr pipe_with_source,
972
0
        PipelinePtr pipe_with_sink) {
973
0
    pipe_with_sink->set_num_tasks(pipe_with_source->num_tasks());
974
0
    pipe_with_source->set_num_tasks(_num_instances);
975
0
    pipe_with_source->set_data_distribution(data_distribution);
976
0
}
977
978
Status PipelineFragmentContext::_add_local_exchange_impl(
979
        int idx, ObjectPool* pool, PipelinePtr cur_pipe, PipelinePtr new_pip,
980
        DataDistribution data_distribution, bool* do_local_exchange, int num_buckets,
981
        const std::map<int, int>& bucket_seq_to_instance_idx,
982
0
        const std::map<int, int>& shuffle_idx_to_instance_idx) {
983
0
    auto& operators = cur_pipe->operators();
984
0
    const auto downstream_pipeline_id = cur_pipe->id();
985
0
    auto local_exchange_id = next_operator_id();
986
    // 1. Create a new pipeline with local exchange sink.
987
0
    DataSinkOperatorPtr sink;
988
0
    auto sink_id = next_sink_operator_id();
989
990
    /**
991
     * `bucket_seq_to_instance_idx` is empty if no scan operator is contained in this fragment.
992
     * So co-located operators(e.g. Agg, Analytic) should use `HASH_SHUFFLE` instead of `BUCKET_HASH_SHUFFLE`.
993
     */
994
0
    const bool followed_by_shuffled_operator =
995
0
            operators.size() > idx ? operators[idx]->followed_by_shuffled_operator()
996
0
                                   : cur_pipe->sink()->followed_by_shuffled_operator();
997
0
    const bool use_global_hash_shuffle = bucket_seq_to_instance_idx.empty() &&
998
0
                                         !shuffle_idx_to_instance_idx.contains(-1) &&
999
0
                                         followed_by_shuffled_operator && !_use_serial_source;
1000
0
    sink = std::make_shared<LocalExchangeSinkOperatorX>(
1001
0
            sink_id, local_exchange_id, use_global_hash_shuffle ? _total_instances : _num_instances,
1002
0
            data_distribution.partition_exprs, bucket_seq_to_instance_idx);
1003
0
    if (bucket_seq_to_instance_idx.empty() &&
1004
0
        data_distribution.distribution_type == TLocalPartitionType::BUCKET_HASH_SHUFFLE) {
1005
0
        data_distribution.distribution_type =
1006
0
                use_global_hash_shuffle ? TLocalPartitionType::GLOBAL_EXECUTION_HASH_SHUFFLE
1007
0
                                        : TLocalPartitionType::LOCAL_EXECUTION_HASH_SHUFFLE;
1008
0
    }
1009
0
    if (!use_global_hash_shuffle &&
1010
0
        data_distribution.distribution_type == TLocalPartitionType::GLOBAL_EXECUTION_HASH_SHUFFLE) {
1011
0
        data_distribution.distribution_type = TLocalPartitionType::LOCAL_EXECUTION_HASH_SHUFFLE;
1012
0
    }
1013
0
    RETURN_IF_ERROR(new_pip->set_sink(sink));
1014
0
    RETURN_IF_ERROR(new_pip->sink()->init(_runtime_state.get(), data_distribution.distribution_type,
1015
0
                                          num_buckets, shuffle_idx_to_instance_idx));
1016
1017
    // 2. Create and initialize LocalExchangeSharedState.
1018
0
    std::shared_ptr<LocalExchangeSharedState> shared_state =
1019
0
            LocalExchangeSharedState::create_shared(_num_instances);
1020
0
    switch (data_distribution.distribution_type) {
1021
0
    case TLocalPartitionType::LOCAL_EXECUTION_HASH_SHUFFLE:
1022
0
    case TLocalPartitionType::GLOBAL_EXECUTION_HASH_SHUFFLE:
1023
0
        shared_state->exchanger = ShuffleExchanger::create_unique(
1024
0
                std::max(cur_pipe->num_tasks(), _num_instances), _num_instances,
1025
0
                use_global_hash_shuffle ? _total_instances : _num_instances,
1026
0
                _runtime_state->query_options().__isset.local_exchange_free_blocks_limit
1027
0
                        ? cast_set<int>(
1028
0
                                  _runtime_state->query_options().local_exchange_free_blocks_limit)
1029
0
                        : 0,
1030
0
                data_distribution.distribution_type);
1031
0
        break;
1032
0
    case TLocalPartitionType::BUCKET_HASH_SHUFFLE:
1033
0
        shared_state->exchanger = BucketShuffleExchanger::create_unique(
1034
0
                std::max(cur_pipe->num_tasks(), _num_instances), _num_instances, num_buckets,
1035
0
                _runtime_state->query_options().__isset.local_exchange_free_blocks_limit
1036
0
                        ? cast_set<int>(
1037
0
                                  _runtime_state->query_options().local_exchange_free_blocks_limit)
1038
0
                        : 0);
1039
0
        break;
1040
0
    case TLocalPartitionType::PASSTHROUGH:
1041
0
        shared_state->exchanger = PassthroughExchanger::create_unique(
1042
0
                cur_pipe->num_tasks(), _num_instances,
1043
0
                _runtime_state->query_options().__isset.local_exchange_free_blocks_limit
1044
0
                        ? cast_set<int>(
1045
0
                                  _runtime_state->query_options().local_exchange_free_blocks_limit)
1046
0
                        : 0);
1047
0
        break;
1048
0
    case TLocalPartitionType::BROADCAST:
1049
0
        shared_state->exchanger = BroadcastExchanger::create_unique(
1050
0
                cur_pipe->num_tasks(), _num_instances,
1051
0
                _runtime_state->query_options().__isset.local_exchange_free_blocks_limit
1052
0
                        ? cast_set<int>(
1053
0
                                  _runtime_state->query_options().local_exchange_free_blocks_limit)
1054
0
                        : 0);
1055
0
        break;
1056
0
    case TLocalPartitionType::PASS_TO_ONE:
1057
0
        if (_runtime_state->enable_share_hash_table_for_broadcast_join()) {
1058
            // If shared hash table is enabled for BJ, hash table will be built by only one task
1059
0
            shared_state->exchanger = PassToOneExchanger::create_unique(
1060
0
                    cur_pipe->num_tasks(), _num_instances,
1061
0
                    _runtime_state->query_options().__isset.local_exchange_free_blocks_limit
1062
0
                            ? cast_set<int>(_runtime_state->query_options()
1063
0
                                                    .local_exchange_free_blocks_limit)
1064
0
                            : 0);
1065
0
        } else {
1066
0
            shared_state->exchanger = BroadcastExchanger::create_unique(
1067
0
                    cur_pipe->num_tasks(), _num_instances,
1068
0
                    _runtime_state->query_options().__isset.local_exchange_free_blocks_limit
1069
0
                            ? cast_set<int>(_runtime_state->query_options()
1070
0
                                                    .local_exchange_free_blocks_limit)
1071
0
                            : 0);
1072
0
        }
1073
0
        break;
1074
0
    case TLocalPartitionType::ADAPTIVE_PASSTHROUGH:
1075
0
        shared_state->exchanger = AdaptivePassthroughExchanger::create_unique(
1076
0
                std::max(cur_pipe->num_tasks(), _num_instances), _num_instances,
1077
0
                _runtime_state->query_options().__isset.local_exchange_free_blocks_limit
1078
0
                        ? cast_set<int>(
1079
0
                                  _runtime_state->query_options().local_exchange_free_blocks_limit)
1080
0
                        : 0);
1081
0
        break;
1082
0
    default:
1083
0
        return Status::InternalError("Unsupported local exchange type : " +
1084
0
                                     std::to_string((int)data_distribution.distribution_type));
1085
0
    }
1086
0
    shared_state->create_source_dependencies(_num_instances, local_exchange_id, local_exchange_id,
1087
0
                                             "LOCAL_EXCHANGE_OPERATOR");
1088
0
    shared_state->create_sink_dependency(sink_id, local_exchange_id, "LOCAL_EXCHANGE_SINK");
1089
0
    _op_id_to_shared_state.insert({local_exchange_id, {shared_state, shared_state->sink_deps}});
1090
1091
    // 3. Set two pipelines' operator list. For example, split pipeline [Scan - AggSink] to
1092
    // pipeline1 [Scan - LocalExchangeSink] and pipeline2 [LocalExchangeSource - AggSink].
1093
1094
    // 3.1 Initialize new pipeline's operator list.
1095
0
    std::copy(operators.begin(), operators.begin() + idx,
1096
0
              std::inserter(new_pip->operators(), new_pip->operators().end()));
1097
1098
    // 3.2 Erase unused operators in previous pipeline.
1099
0
    operators.erase(operators.begin(), operators.begin() + idx);
1100
1101
    // 4. Initialize LocalExchangeSource and insert it into this pipeline.
1102
0
    OperatorPtr source_op;
1103
0
    source_op = std::make_shared<LocalExchangeSourceOperatorX>(pool, local_exchange_id);
1104
0
    RETURN_IF_ERROR(source_op->set_child(new_pip->operators().back()));
1105
0
    RETURN_IF_ERROR(source_op->init(data_distribution.distribution_type));
1106
0
    if (!operators.empty()) {
1107
0
        RETURN_IF_ERROR(operators.front()->set_child(nullptr));
1108
0
        RETURN_IF_ERROR(operators.front()->set_child(source_op));
1109
0
    }
1110
0
    operators.insert(operators.begin(), source_op);
1111
1112
    // 5. Set children for two pipelines separately.
1113
0
    std::vector<std::shared_ptr<Pipeline>> new_children;
1114
0
    std::vector<PipelineId> edges_with_source;
1115
0
    for (auto child : cur_pipe->children()) {
1116
0
        bool found = false;
1117
0
        for (auto op : new_pip->operators()) {
1118
0
            if (child->sink()->node_id() == op->node_id()) {
1119
0
                new_pip->set_children(child);
1120
0
                found = true;
1121
0
            };
1122
0
        }
1123
0
        if (!found) {
1124
0
            new_children.push_back(child);
1125
0
            edges_with_source.push_back(child->id());
1126
0
        }
1127
0
    }
1128
0
    new_children.push_back(new_pip);
1129
0
    edges_with_source.push_back(new_pip->id());
1130
1131
    // 6. Set DAG for new pipelines.
1132
0
    if (!new_pip->children().empty()) {
1133
0
        std::vector<PipelineId> edges_with_sink;
1134
0
        for (auto child : new_pip->children()) {
1135
0
            edges_with_sink.push_back(child->id());
1136
0
        }
1137
0
        _dag.insert({new_pip->id(), edges_with_sink});
1138
0
    }
1139
0
    cur_pipe->set_children(new_children);
1140
0
    _dag[downstream_pipeline_id] = edges_with_source;
1141
0
    RETURN_IF_ERROR(new_pip->sink()->set_child(new_pip->operators().back()));
1142
0
    RETURN_IF_ERROR(cur_pipe->sink()->set_child(nullptr));
1143
0
    RETURN_IF_ERROR(cur_pipe->sink()->set_child(cur_pipe->operators().back()));
1144
1145
    // 7. Inherit properties from current pipeline.
1146
0
    _inherit_pipeline_properties(data_distribution, cur_pipe, new_pip);
1147
0
    return Status::OK();
1148
0
}
1149
1150
Status PipelineFragmentContext::_add_local_exchange(
1151
        int pip_idx, int idx, int node_id, ObjectPool* pool, PipelinePtr cur_pipe,
1152
        DataDistribution data_distribution, bool* do_local_exchange, int num_buckets,
1153
        const std::map<int, int>& bucket_seq_to_instance_idx,
1154
0
        const std::map<int, int>& shuffle_idx_to_instance_idx) {
1155
0
    if (_num_instances <= 1 || cur_pipe->num_tasks_of_parent() <= 1) {
1156
0
        return Status::OK();
1157
0
    }
1158
1159
0
    if (!cur_pipe->need_to_local_exchange(data_distribution, idx)) {
1160
0
        return Status::OK();
1161
0
    }
1162
0
    *do_local_exchange = true;
1163
1164
0
    auto& operators = cur_pipe->operators();
1165
0
    auto total_op_num = operators.size();
1166
0
    auto new_pip = add_pipeline(cur_pipe, pip_idx + 1);
1167
0
    RETURN_IF_ERROR(_add_local_exchange_impl(
1168
0
            idx, pool, cur_pipe, new_pip, data_distribution, do_local_exchange, num_buckets,
1169
0
            bucket_seq_to_instance_idx, shuffle_idx_to_instance_idx));
1170
1171
0
    CHECK(total_op_num + 1 == cur_pipe->operators().size() + new_pip->operators().size())
1172
0
            << "total_op_num: " << total_op_num
1173
0
            << " cur_pipe->operators().size(): " << cur_pipe->operators().size()
1174
0
            << " new_pip->operators().size(): " << new_pip->operators().size();
1175
1176
    // There are some local shuffles with relatively heavy operations on the sink.
1177
    // If the local sink concurrency is 1 and the local source concurrency is n, the sink becomes a bottleneck.
1178
    // Therefore, local passthrough is used to increase the concurrency of the sink.
1179
    // op -> local sink(1) -> local source (n)
1180
    // op -> local passthrough(1) -> local passthrough(n) ->  local sink(n) -> local source (n)
1181
0
    if (cur_pipe->num_tasks() > 1 && new_pip->num_tasks() == 1 &&
1182
0
        Pipeline::heavy_operations_on_the_sink(data_distribution.distribution_type)) {
1183
0
        RETURN_IF_ERROR(_add_local_exchange_impl(
1184
0
                cast_set<int>(new_pip->operators().size()), pool, new_pip,
1185
0
                add_pipeline(new_pip, pip_idx + 2),
1186
0
                DataDistribution(TLocalPartitionType::PASSTHROUGH), do_local_exchange, num_buckets,
1187
0
                bucket_seq_to_instance_idx, shuffle_idx_to_instance_idx));
1188
0
    }
1189
0
    return Status::OK();
1190
0
}
1191
1192
Status PipelineFragmentContext::_plan_local_exchange(
1193
        int num_buckets, const std::map<int, int>& bucket_seq_to_instance_idx,
1194
0
        const std::map<int, int>& shuffle_idx_to_instance_idx) {
1195
0
    for (int pip_idx = cast_set<int>(_pipelines.size()) - 1; pip_idx >= 0; pip_idx--) {
1196
0
        _pipelines[pip_idx]->init_data_distribution(_runtime_state.get());
1197
        // Set property if child pipeline is not join operator's child.
1198
0
        if (!_pipelines[pip_idx]->children().empty()) {
1199
0
            for (auto& child : _pipelines[pip_idx]->children()) {
1200
0
                if (child->sink()->node_id() ==
1201
0
                    _pipelines[pip_idx]->operators().front()->node_id()) {
1202
0
                    _pipelines[pip_idx]->set_data_distribution(child->data_distribution());
1203
0
                }
1204
0
            }
1205
0
        }
1206
1207
        // if 'num_buckets == 0' means the fragment is colocated by exchange node not the
1208
        // scan node. so here use `_num_instance` to replace the `num_buckets` to prevent dividing 0
1209
        // still keep colocate plan after local shuffle
1210
0
        RETURN_IF_ERROR(_plan_local_exchange(num_buckets, pip_idx, _pipelines[pip_idx],
1211
0
                                             bucket_seq_to_instance_idx,
1212
0
                                             shuffle_idx_to_instance_idx));
1213
0
    }
1214
0
    return Status::OK();
1215
0
}
1216
1217
Status PipelineFragmentContext::_plan_local_exchange(
1218
        int num_buckets, int pip_idx, PipelinePtr pip,
1219
        const std::map<int, int>& bucket_seq_to_instance_idx,
1220
0
        const std::map<int, int>& shuffle_idx_to_instance_idx) {
1221
0
    int idx = 1;
1222
0
    bool do_local_exchange = false;
1223
0
    do {
1224
0
        auto& ops = pip->operators();
1225
0
        do_local_exchange = false;
1226
        // Plan local exchange for each operator.
1227
0
        for (; idx < ops.size();) {
1228
0
            auto _le_req = ops[idx]->required_data_distribution(_runtime_state.get());
1229
0
            if (_le_req.need_local_exchange()) {
1230
0
                RETURN_IF_ERROR(_add_local_exchange(
1231
0
                        pip_idx, idx, ops[idx]->node_id(), _runtime_state->obj_pool(), pip, _le_req,
1232
0
                        &do_local_exchange, num_buckets, bucket_seq_to_instance_idx,
1233
0
                        shuffle_idx_to_instance_idx));
1234
0
            }
1235
0
            if (do_local_exchange) {
1236
                // If local exchange is needed for current operator, we will split this pipeline to
1237
                // two pipelines by local exchange sink/source. And then we need to process remaining
1238
                // operators in this pipeline so we set idx to 2 (0 is local exchange source and 1
1239
                // is current operator was already processed) and continue to plan local exchange.
1240
0
                idx = 2;
1241
0
                break;
1242
0
            }
1243
0
            idx++;
1244
0
        }
1245
0
    } while (do_local_exchange);
1246
0
    if (pip->sink()->required_data_distribution(_runtime_state.get()).need_local_exchange()) {
1247
0
        RETURN_IF_ERROR(_add_local_exchange(
1248
0
                pip_idx, idx, pip->sink()->node_id(), _runtime_state->obj_pool(), pip,
1249
0
                pip->sink()->required_data_distribution(_runtime_state.get()), &do_local_exchange,
1250
0
                num_buckets, bucket_seq_to_instance_idx, shuffle_idx_to_instance_idx));
1251
0
    }
1252
0
    return Status::OK();
1253
0
}
1254
1255
Status PipelineFragmentContext::_create_data_sink(ObjectPool* pool, const TDataSink& thrift_sink,
1256
                                                  const std::vector<TExpr>& output_exprs,
1257
                                                  const TPipelineFragmentParams& params,
1258
                                                  const RowDescriptor& row_desc,
1259
                                                  RuntimeState* state, DescriptorTbl& desc_tbl,
1260
0
                                                  PipelineId cur_pipeline_id) {
1261
0
    switch (thrift_sink.type) {
1262
0
    case TDataSinkType::DATA_STREAM_SINK: {
1263
0
        if (!thrift_sink.__isset.stream_sink) {
1264
0
            return Status::InternalError("Missing data stream sink.");
1265
0
        }
1266
0
        _sink = std::make_shared<ExchangeSinkOperatorX>(
1267
0
                state, row_desc, next_sink_operator_id(), thrift_sink.stream_sink,
1268
0
                params.destinations, _fragment_instance_ids);
1269
0
        break;
1270
0
    }
1271
0
    case TDataSinkType::RESULT_SINK: {
1272
0
        if (!thrift_sink.__isset.result_sink) {
1273
0
            return Status::InternalError("Missing data buffer sink.");
1274
0
        }
1275
1276
0
        auto& pipeline = _pipelines[cur_pipeline_id];
1277
0
        int child_node_id = pipeline->operators().back()->node_id();
1278
0
        _sink = std::make_shared<ResultSinkOperatorX>(next_sink_operator_id(), child_node_id + 1,
1279
0
                                                      row_desc, output_exprs,
1280
0
                                                      thrift_sink.result_sink);
1281
0
        break;
1282
0
    }
1283
0
    case TDataSinkType::DICTIONARY_SINK: {
1284
0
        if (!thrift_sink.__isset.dictionary_sink) {
1285
0
            return Status::InternalError("Missing dict sink.");
1286
0
        }
1287
1288
0
        _sink = std::make_shared<DictSinkOperatorX>(next_sink_operator_id(), row_desc, output_exprs,
1289
0
                                                    thrift_sink.dictionary_sink);
1290
0
        break;
1291
0
    }
1292
0
    case TDataSinkType::GROUP_COMMIT_OLAP_TABLE_SINK:
1293
0
    case TDataSinkType::OLAP_TABLE_SINK: {
1294
0
        auto& pipeline = _pipelines[cur_pipeline_id];
1295
0
        int child_node_id = pipeline->operators().back()->node_id();
1296
0
        if (state->query_options().enable_memtable_on_sink_node &&
1297
0
            !_has_inverted_index_v1_or_partial_update(thrift_sink.olap_table_sink) &&
1298
0
            !_has_row_binlog(thrift_sink.olap_table_sink) && !config::is_cloud_mode()) {
1299
0
            _sink = std::make_shared<OlapTableSinkV2OperatorX>(
1300
0
                    pool, next_sink_operator_id(), child_node_id + 1, row_desc, output_exprs);
1301
0
        } else {
1302
0
            _sink = std::make_shared<OlapTableSinkOperatorX>(
1303
0
                    pool, next_sink_operator_id(), child_node_id + 1, row_desc, output_exprs);
1304
0
        }
1305
0
        break;
1306
0
    }
1307
0
    case TDataSinkType::GROUP_COMMIT_BLOCK_SINK: {
1308
0
        DCHECK(thrift_sink.__isset.olap_table_sink);
1309
0
        DCHECK(state->get_query_ctx() != nullptr);
1310
0
        state->get_query_ctx()->query_mem_tracker()->is_group_commit_load = true;
1311
0
        _sink = std::make_shared<GroupCommitBlockSinkOperatorX>(next_sink_operator_id(), row_desc,
1312
0
                                                                output_exprs);
1313
0
        break;
1314
0
    }
1315
0
    case TDataSinkType::HIVE_TABLE_SINK: {
1316
0
        if (!thrift_sink.__isset.hive_table_sink) {
1317
0
            return Status::InternalError("Missing hive table sink.");
1318
0
        }
1319
0
        _sink = std::make_shared<HiveTableSinkOperatorX>(pool, next_sink_operator_id(), row_desc,
1320
0
                                                         output_exprs);
1321
0
        break;
1322
0
    }
1323
0
    case TDataSinkType::ICEBERG_TABLE_SINK: {
1324
0
        if (!thrift_sink.__isset.iceberg_table_sink) {
1325
0
            return Status::InternalError("Missing iceberg table sink.");
1326
0
        }
1327
0
        if (thrift_sink.iceberg_table_sink.__isset.sort_info) {
1328
0
            _sink = std::make_shared<SpillIcebergTableSinkOperatorX>(pool, next_sink_operator_id(),
1329
0
                                                                     row_desc, output_exprs);
1330
0
        } else {
1331
0
            _sink = std::make_shared<IcebergTableSinkOperatorX>(pool, next_sink_operator_id(),
1332
0
                                                                row_desc, output_exprs);
1333
0
        }
1334
0
        break;
1335
0
    }
1336
0
    case TDataSinkType::ICEBERG_DELETE_SINK: {
1337
0
        if (!thrift_sink.__isset.iceberg_delete_sink) {
1338
0
            return Status::InternalError("Missing iceberg delete sink.");
1339
0
        }
1340
0
        _sink = std::make_shared<IcebergDeleteSinkOperatorX>(pool, next_sink_operator_id(),
1341
0
                                                             row_desc, output_exprs);
1342
0
        break;
1343
0
    }
1344
0
    case TDataSinkType::ICEBERG_MERGE_SINK: {
1345
0
        if (!thrift_sink.__isset.iceberg_merge_sink) {
1346
0
            return Status::InternalError("Missing iceberg merge sink.");
1347
0
        }
1348
0
        _sink = std::make_shared<IcebergMergeSinkOperatorX>(pool, next_sink_operator_id(), row_desc,
1349
0
                                                            output_exprs);
1350
0
        break;
1351
0
    }
1352
0
    case TDataSinkType::MAXCOMPUTE_TABLE_SINK: {
1353
0
        if (!thrift_sink.__isset.max_compute_table_sink) {
1354
0
            return Status::InternalError("Missing max compute table sink.");
1355
0
        }
1356
0
        _sink = std::make_shared<MCTableSinkOperatorX>(pool, next_sink_operator_id(), row_desc,
1357
0
                                                       output_exprs);
1358
0
        break;
1359
0
    }
1360
0
    case TDataSinkType::JDBC_TABLE_SINK: {
1361
0
        if (!thrift_sink.__isset.jdbc_table_sink) {
1362
0
            return Status::InternalError("Missing data jdbc sink.");
1363
0
        }
1364
0
        if (config::enable_java_support) {
1365
0
            _sink = std::make_shared<JdbcTableSinkOperatorX>(row_desc, next_sink_operator_id(),
1366
0
                                                             output_exprs);
1367
0
        } else {
1368
0
            return Status::InternalError(
1369
0
                    "Jdbc table sink is not enabled, you can change be config "
1370
0
                    "enable_java_support to true and restart be.");
1371
0
        }
1372
0
        break;
1373
0
    }
1374
0
    case TDataSinkType::MEMORY_SCRATCH_SINK: {
1375
0
        if (!thrift_sink.__isset.memory_scratch_sink) {
1376
0
            return Status::InternalError("Missing data buffer sink.");
1377
0
        }
1378
1379
0
        _sink = std::make_shared<MemoryScratchSinkOperatorX>(row_desc, next_sink_operator_id(),
1380
0
                                                             output_exprs);
1381
0
        break;
1382
0
    }
1383
0
    case TDataSinkType::RESULT_FILE_SINK: {
1384
0
        if (!thrift_sink.__isset.result_file_sink) {
1385
0
            return Status::InternalError("Missing result file sink.");
1386
0
        }
1387
1388
        // Result file sink is not the top sink
1389
0
        if (params.__isset.destinations && !params.destinations.empty()) {
1390
0
            _sink = std::make_shared<ResultFileSinkOperatorX>(
1391
0
                    next_sink_operator_id(), row_desc, thrift_sink.result_file_sink,
1392
0
                    params.destinations, output_exprs, desc_tbl);
1393
0
        } else {
1394
0
            _sink = std::make_shared<ResultFileSinkOperatorX>(next_sink_operator_id(), row_desc,
1395
0
                                                              output_exprs);
1396
0
        }
1397
0
        break;
1398
0
    }
1399
0
    case TDataSinkType::MULTI_CAST_DATA_STREAM_SINK: {
1400
0
        DCHECK(thrift_sink.__isset.multi_cast_stream_sink);
1401
0
        DCHECK_GT(thrift_sink.multi_cast_stream_sink.sinks.size(), 0);
1402
0
        auto sink_id = next_sink_operator_id();
1403
0
        const int multi_cast_node_id = sink_id;
1404
0
        auto sender_size = thrift_sink.multi_cast_stream_sink.sinks.size();
1405
        // one sink has multiple sources.
1406
0
        std::vector<int> sources;
1407
0
        for (int i = 0; i < sender_size; ++i) {
1408
0
            auto source_id = next_operator_id();
1409
0
            sources.push_back(source_id);
1410
0
        }
1411
1412
0
        _sink = std::make_shared<MultiCastDataStreamSinkOperatorX>(
1413
0
                sink_id, multi_cast_node_id, sources, pool, thrift_sink.multi_cast_stream_sink);
1414
0
        for (int i = 0; i < sender_size; ++i) {
1415
0
            auto new_pipeline = add_pipeline();
1416
            // use to exchange sink
1417
0
            RowDescriptor* exchange_row_desc = nullptr;
1418
0
            {
1419
0
                const auto& tmp_row_desc =
1420
0
                        !thrift_sink.multi_cast_stream_sink.sinks[i].output_exprs.empty()
1421
0
                                ? RowDescriptor(state->desc_tbl(),
1422
0
                                                {thrift_sink.multi_cast_stream_sink.sinks[i]
1423
0
                                                         .output_tuple_id})
1424
0
                                : row_desc;
1425
0
                exchange_row_desc = pool->add(new RowDescriptor(tmp_row_desc));
1426
0
            }
1427
0
            auto source_id = sources[i];
1428
0
            OperatorPtr source_op;
1429
            // 1. create and set the source operator of multi_cast_data_stream_source for new pipeline
1430
0
            source_op = std::make_shared<MultiCastDataStreamerSourceOperatorX>(
1431
0
                    /*node_id*/ source_id, /*consumer_id*/ i, pool,
1432
0
                    thrift_sink.multi_cast_stream_sink.sinks[i], row_desc,
1433
0
                    /*operator_id=*/source_id);
1434
0
            RETURN_IF_ERROR(new_pipeline->add_operator(
1435
0
                    source_op, params.__isset.parallel_instances ? params.parallel_instances : 0));
1436
            // 2. create and set sink operator of data stream sender for new pipeline
1437
1438
0
            DataSinkOperatorPtr sink_op;
1439
0
            sink_op = std::make_shared<ExchangeSinkOperatorX>(
1440
0
                    state, *exchange_row_desc, next_sink_operator_id(),
1441
0
                    thrift_sink.multi_cast_stream_sink.sinks[i],
1442
0
                    thrift_sink.multi_cast_stream_sink.destinations[i], _fragment_instance_ids);
1443
1444
0
            RETURN_IF_ERROR(new_pipeline->set_sink(sink_op));
1445
0
            {
1446
0
                TDataSink* t = pool->add(new TDataSink());
1447
0
                t->stream_sink = thrift_sink.multi_cast_stream_sink.sinks[i];
1448
0
                RETURN_IF_ERROR(sink_op->init(*t));
1449
0
            }
1450
1451
            // 3. set dependency dag
1452
0
            _dag[new_pipeline->id()].push_back(cur_pipeline_id);
1453
0
        }
1454
0
        if (sources.empty()) {
1455
0
            return Status::InternalError("size of sources must be greater than 0");
1456
0
        }
1457
0
        break;
1458
0
    }
1459
0
    case TDataSinkType::BLACKHOLE_SINK: {
1460
0
        if (!thrift_sink.__isset.blackhole_sink) {
1461
0
            return Status::InternalError("Missing blackhole sink.");
1462
0
        }
1463
1464
0
        _sink.reset(new BlackholeSinkOperatorX(next_sink_operator_id()));
1465
0
        break;
1466
0
    }
1467
0
    case TDataSinkType::TVF_TABLE_SINK: {
1468
0
        if (!thrift_sink.__isset.tvf_table_sink) {
1469
0
            return Status::InternalError("Missing TVF table sink.");
1470
0
        }
1471
0
        _sink = std::make_shared<TVFTableSinkOperatorX>(pool, next_sink_operator_id(), row_desc,
1472
0
                                                        output_exprs);
1473
0
        break;
1474
0
    }
1475
0
    default:
1476
0
        return Status::InternalError("Unsuported sink type in pipeline: {}", thrift_sink.type);
1477
0
    }
1478
0
    return Status::OK();
1479
0
}
1480
1481
// NOLINTBEGIN(readability-function-size)
1482
// NOLINTBEGIN(readability-function-cognitive-complexity)
1483
Status PipelineFragmentContext::_create_operator(ObjectPool* pool, const TPlanNode& tnode,
1484
                                                 const DescriptorTbl& descs, OperatorPtr& op,
1485
                                                 PipelinePtr& cur_pipe, int parent_idx,
1486
                                                 int child_idx,
1487
                                                 const bool followed_by_shuffled_operator,
1488
                                                 const bool require_bucket_distribution,
1489
0
                                                 OperatorPtr& cache_op) {
1490
0
    std::vector<DataSinkOperatorPtr> sink_ops;
1491
0
    Defer defer = Defer([&]() {
1492
0
        if (op) {
1493
0
            op->update_operator(tnode, followed_by_shuffled_operator, require_bucket_distribution);
1494
0
        }
1495
0
        for (auto& s : sink_ops) {
1496
0
            s->update_operator(tnode, followed_by_shuffled_operator, require_bucket_distribution);
1497
0
        }
1498
0
    });
1499
    // We directly construct the operator from Thrift because the given array is in the order of preorder traversal.
1500
    // Therefore, here we need to use a stack-like structure.
1501
0
    _pipeline_parent_map.pop(cur_pipe, parent_idx, child_idx);
1502
0
    std::stringstream error_msg;
1503
0
    bool enable_query_cache = _params.fragment.__isset.query_cache_param;
1504
1505
0
    bool fe_with_old_version = false;
1506
0
    switch (tnode.node_type) {
1507
0
    case TPlanNodeType::OLAP_SCAN_NODE: {
1508
0
        op = std::make_shared<OlapScanOperatorX>(
1509
0
                pool, tnode, next_operator_id(), descs, _num_instances,
1510
0
                enable_query_cache ? _params.fragment.query_cache_param : TQueryCacheParam {});
1511
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1512
0
        fe_with_old_version = !tnode.__isset.is_serial_operator;
1513
0
        break;
1514
0
    }
1515
0
    case TPlanNodeType::GROUP_COMMIT_SCAN_NODE: {
1516
0
        DCHECK(_query_ctx != nullptr);
1517
0
        _query_ctx->query_mem_tracker()->is_group_commit_load = true;
1518
0
        op = std::make_shared<GroupCommitOperatorX>(pool, tnode, next_operator_id(), descs,
1519
0
                                                    _num_instances);
1520
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1521
0
        fe_with_old_version = !tnode.__isset.is_serial_operator;
1522
0
        break;
1523
0
    }
1524
0
    case TPlanNodeType::JDBC_SCAN_NODE: {
1525
0
        if (config::enable_java_support) {
1526
0
            op = std::make_shared<JDBCScanOperatorX>(pool, tnode, next_operator_id(), descs,
1527
0
                                                     _num_instances);
1528
0
            RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1529
0
        } else {
1530
0
            return Status::InternalError(
1531
0
                    "Jdbc scan node is disabled, you can change be config enable_java_support "
1532
0
                    "to true and restart be.");
1533
0
        }
1534
0
        fe_with_old_version = !tnode.__isset.is_serial_operator;
1535
0
        break;
1536
0
    }
1537
0
    case TPlanNodeType::FILE_SCAN_NODE: {
1538
0
        op = std::make_shared<FileScanOperatorX>(pool, tnode, next_operator_id(), descs,
1539
0
                                                 _num_instances);
1540
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1541
0
        fe_with_old_version = !tnode.__isset.is_serial_operator;
1542
0
        break;
1543
0
    }
1544
0
    case TPlanNodeType::EXCHANGE_NODE: {
1545
0
        int num_senders = _params.per_exch_num_senders.contains(tnode.node_id)
1546
0
                                  ? _params.per_exch_num_senders.find(tnode.node_id)->second
1547
0
                                  : 0;
1548
0
        DCHECK_GT(num_senders, 0);
1549
0
        op = std::make_shared<ExchangeSourceOperatorX>(pool, tnode, next_operator_id(), descs,
1550
0
                                                       num_senders);
1551
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1552
0
        fe_with_old_version = !tnode.__isset.is_serial_operator;
1553
0
        break;
1554
0
    }
1555
0
    case TPlanNodeType::AGGREGATION_NODE: {
1556
0
        if (tnode.agg_node.grouping_exprs.empty() &&
1557
0
            descs.get_tuple_descriptor(tnode.agg_node.output_tuple_id)->slots().empty()) {
1558
0
            return Status::InternalError("Illegal aggregate node " + std::to_string(tnode.node_id) +
1559
0
                                         ": group by and output is empty");
1560
0
        }
1561
0
        bool need_create_cache_op =
1562
0
                enable_query_cache && tnode.node_id == _params.fragment.query_cache_param.node_id;
1563
0
        auto create_query_cache_operator = [&](PipelinePtr& new_pipe) {
1564
0
            auto cache_node_id = _params.local_params[0].per_node_scan_ranges.begin()->first;
1565
0
            auto cache_source_id = next_operator_id();
1566
0
            op = std::make_shared<CacheSourceOperatorX>(pool, cache_node_id, cache_source_id,
1567
0
                                                        _params.fragment.query_cache_param);
1568
0
            RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1569
1570
0
            const auto downstream_pipeline_id = cur_pipe->id();
1571
0
            if (!_dag.contains(downstream_pipeline_id)) {
1572
0
                _dag.insert({downstream_pipeline_id, {}});
1573
0
            }
1574
0
            new_pipe = add_pipeline(cur_pipe);
1575
0
            _dag[downstream_pipeline_id].push_back(new_pipe->id());
1576
1577
0
            DataSinkOperatorPtr cache_sink(new CacheSinkOperatorX(
1578
0
                    next_sink_operator_id(), op->node_id(), op->operator_id()));
1579
0
            RETURN_IF_ERROR(new_pipe->set_sink(cache_sink));
1580
0
            return Status::OK();
1581
0
        };
1582
0
        const bool group_by_limit_opt =
1583
0
                tnode.agg_node.__isset.agg_sort_info_by_group_key && tnode.limit > 0;
1584
1585
        /// PartitionedAggSourceOperatorX does not support "group by limit opt(#29641)" yet.
1586
        /// If `group_by_limit_opt` is true, then it might not need to spill at all.
1587
0
        const bool enable_spill = _runtime_state->enable_spill() &&
1588
0
                                  !tnode.agg_node.grouping_exprs.empty() && !group_by_limit_opt;
1589
0
        const bool is_streaming_agg = tnode.agg_node.__isset.use_streaming_preaggregation &&
1590
0
                                      tnode.agg_node.use_streaming_preaggregation &&
1591
0
                                      !tnode.agg_node.grouping_exprs.empty();
1592
        // TODO: distinct streaming agg does not support spill.
1593
0
        const bool can_use_distinct_streaming_agg =
1594
0
                (!enable_spill || is_streaming_agg) && tnode.agg_node.aggregate_functions.empty() &&
1595
0
                !tnode.agg_node.__isset.agg_sort_info_by_group_key &&
1596
0
                _params.query_options.__isset.enable_distinct_streaming_aggregation &&
1597
0
                _params.query_options.enable_distinct_streaming_aggregation;
1598
1599
0
        if (can_use_distinct_streaming_agg) {
1600
0
            if (need_create_cache_op) {
1601
0
                PipelinePtr new_pipe;
1602
0
                RETURN_IF_ERROR(create_query_cache_operator(new_pipe));
1603
1604
0
                cache_op = op;
1605
0
                op = std::make_shared<DistinctStreamingAggOperatorX>(pool, next_operator_id(),
1606
0
                                                                     tnode, descs);
1607
0
                RETURN_IF_ERROR(new_pipe->add_operator(op, _parallel_instances));
1608
0
                RETURN_IF_ERROR(cur_pipe->operators().front()->set_child(op));
1609
0
                cur_pipe = new_pipe;
1610
0
            } else {
1611
0
                op = std::make_shared<DistinctStreamingAggOperatorX>(pool, next_operator_id(),
1612
0
                                                                     tnode, descs);
1613
0
                RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1614
0
            }
1615
0
        } else if (is_streaming_agg) {
1616
0
            if (need_create_cache_op) {
1617
0
                PipelinePtr new_pipe;
1618
0
                RETURN_IF_ERROR(create_query_cache_operator(new_pipe));
1619
0
                cache_op = op;
1620
0
                op = std::make_shared<StreamingAggOperatorX>(pool, next_operator_id(), tnode,
1621
0
                                                             descs);
1622
0
                RETURN_IF_ERROR(cur_pipe->operators().front()->set_child(op));
1623
0
                RETURN_IF_ERROR(new_pipe->add_operator(op, _parallel_instances));
1624
0
                cur_pipe = new_pipe;
1625
0
            } else {
1626
0
                op = std::make_shared<StreamingAggOperatorX>(pool, next_operator_id(), tnode,
1627
0
                                                             descs);
1628
0
                RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1629
0
            }
1630
0
        } else {
1631
            // create new pipeline to add query cache operator
1632
0
            PipelinePtr new_pipe;
1633
0
            if (need_create_cache_op) {
1634
0
                RETURN_IF_ERROR(create_query_cache_operator(new_pipe));
1635
0
                cache_op = op;
1636
0
            }
1637
1638
0
            if (enable_spill) {
1639
0
                op = std::make_shared<PartitionedAggSourceOperatorX>(pool, tnode,
1640
0
                                                                     next_operator_id(), descs);
1641
0
            } else {
1642
0
                op = std::make_shared<AggSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1643
0
            }
1644
0
            if (need_create_cache_op) {
1645
0
                RETURN_IF_ERROR(cur_pipe->operators().front()->set_child(op));
1646
0
                RETURN_IF_ERROR(new_pipe->add_operator(op, _parallel_instances));
1647
0
                cur_pipe = new_pipe;
1648
0
            } else {
1649
0
                RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1650
0
            }
1651
1652
0
            const auto downstream_pipeline_id = cur_pipe->id();
1653
0
            if (!_dag.contains(downstream_pipeline_id)) {
1654
0
                _dag.insert({downstream_pipeline_id, {}});
1655
0
            }
1656
0
            cur_pipe = add_pipeline(cur_pipe);
1657
0
            _dag[downstream_pipeline_id].push_back(cur_pipe->id());
1658
1659
0
            if (enable_spill) {
1660
0
                sink_ops.push_back(std::make_shared<PartitionedAggSinkOperatorX>(
1661
0
                        pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1662
0
            } else {
1663
0
                sink_ops.push_back(std::make_shared<AggSinkOperatorX>(
1664
0
                        pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1665
0
            }
1666
0
            RETURN_IF_ERROR(cur_pipe->set_sink(sink_ops.back()));
1667
0
            RETURN_IF_ERROR(cur_pipe->sink()->init(tnode, _runtime_state.get()));
1668
0
        }
1669
0
        break;
1670
0
    }
1671
0
    case TPlanNodeType::BUCKETED_AGGREGATION_NODE: {
1672
0
        if (tnode.bucketed_agg_node.grouping_exprs.empty()) {
1673
0
            return Status::InternalError(
1674
0
                    "Bucketed aggregation node {} should not be used without group by keys",
1675
0
                    tnode.node_id);
1676
0
        }
1677
1678
        // Create source operator (goes on the current / downstream pipeline).
1679
0
        op = std::make_shared<BucketedAggSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1680
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1681
1682
        // Create a new pipeline for the sink side.
1683
0
        const auto downstream_pipeline_id = cur_pipe->id();
1684
0
        if (!_dag.contains(downstream_pipeline_id)) {
1685
0
            _dag.insert({downstream_pipeline_id, {}});
1686
0
        }
1687
0
        cur_pipe = add_pipeline(cur_pipe);
1688
0
        _dag[downstream_pipeline_id].push_back(cur_pipe->id());
1689
1690
        // Create sink operator.
1691
0
        sink_ops.push_back(std::make_shared<BucketedAggSinkOperatorX>(
1692
0
                pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1693
0
        RETURN_IF_ERROR(cur_pipe->set_sink(sink_ops.back()));
1694
0
        RETURN_IF_ERROR(cur_pipe->sink()->init(tnode, _runtime_state.get()));
1695
1696
        // Pre-register a single shared state for ALL instances so that every
1697
        // sink instance writes its per-instance hash table into the same
1698
        // BucketedAggSharedState and every source instance can merge across
1699
        // all of them.
1700
0
        {
1701
0
            auto shared_state = BucketedAggSharedState::create_shared();
1702
0
            shared_state->id = op->operator_id();
1703
0
            shared_state->related_op_ids.insert(op->operator_id());
1704
1705
0
            for (int i = 0; i < _num_instances; i++) {
1706
0
                auto sink_dep = std::make_shared<Dependency>(op->operator_id(), op->node_id(),
1707
0
                                                             "BUCKETED_AGG_SINK_DEPENDENCY");
1708
0
                sink_dep->set_shared_state(shared_state.get());
1709
0
                shared_state->sink_deps.push_back(sink_dep);
1710
0
            }
1711
0
            shared_state->create_source_dependencies(_num_instances, op->operator_id(),
1712
0
                                                     op->node_id(), "BUCKETED_AGG_SOURCE");
1713
0
            _op_id_to_shared_state.insert(
1714
0
                    {op->operator_id(), {shared_state, shared_state->sink_deps}});
1715
0
        }
1716
0
        break;
1717
0
    }
1718
0
    case TPlanNodeType::HASH_JOIN_NODE: {
1719
0
        const auto is_broadcast_join = tnode.hash_join_node.__isset.is_broadcast_join &&
1720
0
                                       tnode.hash_join_node.is_broadcast_join;
1721
0
        const auto enable_spill = _runtime_state->enable_spill();
1722
0
        if (enable_spill && !is_broadcast_join) {
1723
0
            auto tnode_ = tnode;
1724
0
            tnode_.runtime_filters.clear();
1725
0
            auto inner_probe_operator =
1726
0
                    std::make_shared<HashJoinProbeOperatorX>(pool, tnode_, 0, descs);
1727
1728
            // probe side inner sink operator is used to build hash table on probe side when data is spilled.
1729
            // So here use `tnode_` which has no runtime filters.
1730
0
            auto probe_side_inner_sink_operator =
1731
0
                    std::make_shared<HashJoinBuildSinkOperatorX>(pool, 0, 0, tnode_, descs);
1732
1733
0
            RETURN_IF_ERROR(inner_probe_operator->init(tnode_, _runtime_state.get()));
1734
0
            RETURN_IF_ERROR(probe_side_inner_sink_operator->init(tnode_, _runtime_state.get()));
1735
1736
0
            auto probe_operator = std::make_shared<PartitionedHashJoinProbeOperatorX>(
1737
0
                    pool, tnode_, next_operator_id(), descs);
1738
0
            probe_operator->set_inner_operators(probe_side_inner_sink_operator,
1739
0
                                                inner_probe_operator);
1740
0
            op = std::move(probe_operator);
1741
0
            RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1742
1743
0
            const auto downstream_pipeline_id = cur_pipe->id();
1744
0
            if (!_dag.contains(downstream_pipeline_id)) {
1745
0
                _dag.insert({downstream_pipeline_id, {}});
1746
0
            }
1747
0
            PipelinePtr build_side_pipe = add_pipeline(cur_pipe);
1748
0
            _dag[downstream_pipeline_id].push_back(build_side_pipe->id());
1749
1750
0
            auto inner_sink_operator =
1751
0
                    std::make_shared<HashJoinBuildSinkOperatorX>(pool, 0, 0, tnode, descs);
1752
0
            auto sink_operator = std::make_shared<PartitionedHashJoinSinkOperatorX>(
1753
0
                    pool, next_sink_operator_id(), op->operator_id(), tnode_, descs);
1754
0
            RETURN_IF_ERROR(inner_sink_operator->init(tnode, _runtime_state.get()));
1755
1756
0
            sink_operator->set_inner_operators(inner_sink_operator, inner_probe_operator);
1757
0
            sink_ops.push_back(std::move(sink_operator));
1758
0
            RETURN_IF_ERROR(build_side_pipe->set_sink(sink_ops.back()));
1759
0
            RETURN_IF_ERROR(build_side_pipe->sink()->init(tnode_, _runtime_state.get()));
1760
1761
0
            _pipeline_parent_map.push(op->node_id(), cur_pipe);
1762
0
            _pipeline_parent_map.push(op->node_id(), build_side_pipe);
1763
0
        } else {
1764
0
            op = std::make_shared<HashJoinProbeOperatorX>(pool, tnode, next_operator_id(), descs);
1765
0
            RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1766
1767
0
            const auto downstream_pipeline_id = cur_pipe->id();
1768
0
            if (!_dag.contains(downstream_pipeline_id)) {
1769
0
                _dag.insert({downstream_pipeline_id, {}});
1770
0
            }
1771
0
            PipelinePtr build_side_pipe = add_pipeline(cur_pipe);
1772
0
            _dag[downstream_pipeline_id].push_back(build_side_pipe->id());
1773
1774
0
            sink_ops.push_back(std::make_shared<HashJoinBuildSinkOperatorX>(
1775
0
                    pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1776
0
            RETURN_IF_ERROR(build_side_pipe->set_sink(sink_ops.back()));
1777
0
            RETURN_IF_ERROR(build_side_pipe->sink()->init(tnode, _runtime_state.get()));
1778
1779
0
            _pipeline_parent_map.push(op->node_id(), cur_pipe);
1780
0
            _pipeline_parent_map.push(op->node_id(), build_side_pipe);
1781
0
        }
1782
0
        if (is_broadcast_join && _runtime_state->enable_share_hash_table_for_broadcast_join()) {
1783
0
            std::shared_ptr<HashJoinSharedState> shared_state =
1784
0
                    HashJoinSharedState::create_shared(_num_instances);
1785
0
            for (int i = 0; i < _num_instances; i++) {
1786
0
                auto sink_dep = std::make_shared<Dependency>(op->operator_id(), op->node_id(),
1787
0
                                                             "HASH_JOIN_BUILD_DEPENDENCY");
1788
0
                sink_dep->set_shared_state(shared_state.get());
1789
0
                shared_state->sink_deps.push_back(sink_dep);
1790
0
            }
1791
0
            shared_state->create_source_dependencies(_num_instances, op->operator_id(),
1792
0
                                                     op->node_id(), "HASH_JOIN_PROBE");
1793
0
            _op_id_to_shared_state.insert(
1794
0
                    {op->operator_id(), {shared_state, shared_state->sink_deps}});
1795
0
        }
1796
0
        break;
1797
0
    }
1798
0
    case TPlanNodeType::CROSS_JOIN_NODE: {
1799
0
        op = std::make_shared<NestedLoopJoinProbeOperatorX>(pool, tnode, next_operator_id(), descs);
1800
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1801
1802
0
        const auto downstream_pipeline_id = cur_pipe->id();
1803
0
        if (!_dag.contains(downstream_pipeline_id)) {
1804
0
            _dag.insert({downstream_pipeline_id, {}});
1805
0
        }
1806
0
        PipelinePtr build_side_pipe = add_pipeline(cur_pipe);
1807
0
        _dag[downstream_pipeline_id].push_back(build_side_pipe->id());
1808
1809
0
        sink_ops.push_back(std::make_shared<NestedLoopJoinBuildSinkOperatorX>(
1810
0
                pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1811
0
        RETURN_IF_ERROR(build_side_pipe->set_sink(sink_ops.back()));
1812
0
        RETURN_IF_ERROR(build_side_pipe->sink()->init(tnode, _runtime_state.get()));
1813
0
        _pipeline_parent_map.push(op->node_id(), cur_pipe);
1814
0
        _pipeline_parent_map.push(op->node_id(), build_side_pipe);
1815
0
        break;
1816
0
    }
1817
0
    case TPlanNodeType::UNION_NODE: {
1818
0
        int child_count = tnode.num_children;
1819
0
        op = std::make_shared<UnionSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1820
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1821
1822
0
        const auto downstream_pipeline_id = cur_pipe->id();
1823
0
        if (!_dag.contains(downstream_pipeline_id)) {
1824
0
            _dag.insert({downstream_pipeline_id, {}});
1825
0
        }
1826
0
        for (int i = 0; i < child_count; i++) {
1827
0
            PipelinePtr build_side_pipe = add_pipeline(cur_pipe);
1828
0
            _dag[downstream_pipeline_id].push_back(build_side_pipe->id());
1829
0
            sink_ops.push_back(std::make_shared<UnionSinkOperatorX>(
1830
0
                    i, next_sink_operator_id(), op->operator_id(), pool, tnode, descs));
1831
0
            RETURN_IF_ERROR(build_side_pipe->set_sink(sink_ops.back()));
1832
0
            RETURN_IF_ERROR(build_side_pipe->sink()->init(tnode, _runtime_state.get()));
1833
            // preset children pipelines. if any pipeline found this as its father, will use the prepared pipeline to build.
1834
0
            _pipeline_parent_map.push(op->node_id(), build_side_pipe);
1835
0
        }
1836
0
        break;
1837
0
    }
1838
0
    case TPlanNodeType::SORT_NODE: {
1839
0
        const auto should_spill = _runtime_state->enable_spill() &&
1840
0
                                  tnode.sort_node.algorithm == TSortAlgorithm::FULL_SORT;
1841
0
        const bool use_local_merge =
1842
0
                tnode.sort_node.__isset.use_local_merge && tnode.sort_node.use_local_merge;
1843
0
        if (should_spill) {
1844
0
            op = std::make_shared<SpillSortSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1845
0
        } else if (use_local_merge) {
1846
0
            op = std::make_shared<LocalMergeSortSourceOperatorX>(pool, tnode, next_operator_id(),
1847
0
                                                                 descs);
1848
0
        } else {
1849
0
            op = std::make_shared<SortSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1850
0
        }
1851
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1852
1853
0
        const auto downstream_pipeline_id = cur_pipe->id();
1854
0
        if (!_dag.contains(downstream_pipeline_id)) {
1855
0
            _dag.insert({downstream_pipeline_id, {}});
1856
0
        }
1857
0
        cur_pipe = add_pipeline(cur_pipe);
1858
0
        _dag[downstream_pipeline_id].push_back(cur_pipe->id());
1859
1860
0
        if (should_spill) {
1861
0
            sink_ops.push_back(std::make_shared<SpillSortSinkOperatorX>(
1862
0
                    pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1863
0
        } else {
1864
0
            sink_ops.push_back(std::make_shared<SortSinkOperatorX>(
1865
0
                    pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1866
0
        }
1867
0
        RETURN_IF_ERROR(cur_pipe->set_sink(sink_ops.back()));
1868
0
        RETURN_IF_ERROR(cur_pipe->sink()->init(tnode, _runtime_state.get()));
1869
0
        break;
1870
0
    }
1871
0
    case TPlanNodeType::PARTITION_SORT_NODE: {
1872
0
        op = std::make_shared<PartitionSortSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1873
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1874
1875
0
        const auto downstream_pipeline_id = cur_pipe->id();
1876
0
        if (!_dag.contains(downstream_pipeline_id)) {
1877
0
            _dag.insert({downstream_pipeline_id, {}});
1878
0
        }
1879
0
        cur_pipe = add_pipeline(cur_pipe);
1880
0
        _dag[downstream_pipeline_id].push_back(cur_pipe->id());
1881
1882
0
        sink_ops.push_back(std::make_shared<PartitionSortSinkOperatorX>(
1883
0
                pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1884
0
        RETURN_IF_ERROR(cur_pipe->set_sink(sink_ops.back()));
1885
0
        RETURN_IF_ERROR(cur_pipe->sink()->init(tnode, _runtime_state.get()));
1886
0
        break;
1887
0
    }
1888
0
    case TPlanNodeType::ANALYTIC_EVAL_NODE: {
1889
0
        op = std::make_shared<AnalyticSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1890
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1891
1892
0
        const auto downstream_pipeline_id = cur_pipe->id();
1893
0
        if (!_dag.contains(downstream_pipeline_id)) {
1894
0
            _dag.insert({downstream_pipeline_id, {}});
1895
0
        }
1896
0
        cur_pipe = add_pipeline(cur_pipe);
1897
0
        _dag[downstream_pipeline_id].push_back(cur_pipe->id());
1898
1899
0
        sink_ops.push_back(std::make_shared<AnalyticSinkOperatorX>(
1900
0
                pool, next_sink_operator_id(), op->operator_id(), tnode, descs));
1901
0
        RETURN_IF_ERROR(cur_pipe->set_sink(sink_ops.back()));
1902
0
        RETURN_IF_ERROR(cur_pipe->sink()->init(tnode, _runtime_state.get()));
1903
0
        break;
1904
0
    }
1905
0
    case TPlanNodeType::MATERIALIZATION_NODE: {
1906
0
        op = std::make_shared<MaterializationOperator>(pool, tnode, next_operator_id(), descs);
1907
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1908
0
        break;
1909
0
    }
1910
0
    case TPlanNodeType::INTERSECT_NODE: {
1911
0
        RETURN_IF_ERROR(_build_operators_for_set_operation_node<true>(pool, tnode, descs, op,
1912
0
                                                                      cur_pipe, sink_ops));
1913
0
        break;
1914
0
    }
1915
0
    case TPlanNodeType::EXCEPT_NODE: {
1916
0
        RETURN_IF_ERROR(_build_operators_for_set_operation_node<false>(pool, tnode, descs, op,
1917
0
                                                                       cur_pipe, sink_ops));
1918
0
        break;
1919
0
    }
1920
0
    case TPlanNodeType::REPEAT_NODE: {
1921
0
        op = std::make_shared<RepeatOperatorX>(pool, tnode, next_operator_id(), descs);
1922
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1923
0
        break;
1924
0
    }
1925
0
    case TPlanNodeType::TABLE_FUNCTION_NODE: {
1926
0
        op = std::make_shared<TableFunctionOperatorX>(pool, tnode, next_operator_id(), descs);
1927
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1928
0
        break;
1929
0
    }
1930
0
    case TPlanNodeType::ASSERT_NUM_ROWS_NODE: {
1931
0
        op = std::make_shared<AssertNumRowsOperatorX>(pool, tnode, next_operator_id(), descs);
1932
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1933
0
        break;
1934
0
    }
1935
0
    case TPlanNodeType::EMPTY_SET_NODE: {
1936
0
        op = std::make_shared<EmptySetSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1937
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1938
0
        break;
1939
0
    }
1940
0
    case TPlanNodeType::DATA_GEN_SCAN_NODE: {
1941
0
        op = std::make_shared<DataGenSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1942
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1943
0
        fe_with_old_version = !tnode.__isset.is_serial_operator;
1944
0
        break;
1945
0
    }
1946
0
    case TPlanNodeType::SCHEMA_SCAN_NODE: {
1947
0
        op = std::make_shared<SchemaScanOperatorX>(pool, tnode, next_operator_id(), descs);
1948
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1949
0
        break;
1950
0
    }
1951
0
    case TPlanNodeType::META_SCAN_NODE: {
1952
0
        op = std::make_shared<MetaScanOperatorX>(pool, tnode, next_operator_id(), descs);
1953
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1954
0
        break;
1955
0
    }
1956
0
    case TPlanNodeType::SELECT_NODE: {
1957
0
        op = std::make_shared<SelectOperatorX>(pool, tnode, next_operator_id(), descs);
1958
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1959
0
        break;
1960
0
    }
1961
0
    case TPlanNodeType::REC_CTE_NODE: {
1962
0
        op = std::make_shared<RecCTESourceOperatorX>(pool, tnode, next_operator_id(), descs);
1963
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1964
1965
0
        const auto downstream_pipeline_id = cur_pipe->id();
1966
0
        if (!_dag.contains(downstream_pipeline_id)) {
1967
0
            _dag.insert({downstream_pipeline_id, {}});
1968
0
        }
1969
1970
0
        PipelinePtr anchor_side_pipe = add_pipeline(cur_pipe);
1971
0
        _dag[downstream_pipeline_id].push_back(anchor_side_pipe->id());
1972
1973
0
        DataSinkOperatorPtr anchor_sink;
1974
0
        anchor_sink = std::make_shared<RecCTEAnchorSinkOperatorX>(next_sink_operator_id(),
1975
0
                                                                  op->operator_id(), tnode, descs);
1976
0
        RETURN_IF_ERROR(anchor_side_pipe->set_sink(anchor_sink));
1977
0
        RETURN_IF_ERROR(anchor_side_pipe->sink()->init(tnode, _runtime_state.get()));
1978
0
        _pipeline_parent_map.push(op->node_id(), anchor_side_pipe);
1979
1980
0
        PipelinePtr rec_side_pipe = add_pipeline(cur_pipe);
1981
0
        _dag[downstream_pipeline_id].push_back(rec_side_pipe->id());
1982
1983
0
        DataSinkOperatorPtr rec_sink;
1984
0
        rec_sink = std::make_shared<RecCTESinkOperatorX>(next_sink_operator_id(), op->operator_id(),
1985
0
                                                         tnode, descs);
1986
0
        RETURN_IF_ERROR(rec_side_pipe->set_sink(rec_sink));
1987
0
        RETURN_IF_ERROR(rec_side_pipe->sink()->init(tnode, _runtime_state.get()));
1988
0
        _pipeline_parent_map.push(op->node_id(), rec_side_pipe);
1989
1990
0
        break;
1991
0
    }
1992
0
    case TPlanNodeType::REC_CTE_SCAN_NODE: {
1993
0
        op = std::make_shared<RecCTEScanOperatorX>(pool, tnode, next_operator_id(), descs);
1994
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
1995
0
        break;
1996
0
    }
1997
0
    case TPlanNodeType::LOCAL_EXCHANGE_NODE: {
1998
0
        op = std::make_shared<LocalExchangeSourceOperatorX>(pool, tnode, next_operator_id(), descs);
1999
        // The downstream pipeline (containing LocalExchangeSource) must have
2000
        // _num_instances tasks — matching BE-native _inherit_pipeline_properties
2001
        // which sets pipe_with_source.set_num_tasks(_num_instances).
2002
        // Without this, when the parent pipeline was reduced by a serial operator
2003
        // (e.g., serial Exchange with use_serial_exchange=true, or UNPARTITIONED
2004
        // Exchange), the downstream inherits the reduced num_tasks via
2005
        // add_pipeline(parent).  The deferred exchanger creates _num_instances
2006
        // channels but only fewer source tasks initialize mem_counters — the
2007
        // sink round-robins to all channels and crashes on uninitialized ones.
2008
0
        auto downstream_num_tasks = _num_instances;
2009
0
        RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
2010
        // Restore downstream pipeline's num_tasks (mirroring _inherit_pipeline_properties:
2011
        // downstream keeps _num_instances, upstream gets the serial/reduced count)
2012
0
        cur_pipe->set_num_tasks(downstream_num_tasks);
2013
2014
0
        const auto downstream_pipeline_id = cur_pipe->id();
2015
0
        if (!_dag.contains(downstream_pipeline_id)) {
2016
0
            _dag.insert({downstream_pipeline_id, {}});
2017
0
        }
2018
0
        cur_pipe = add_pipeline(cur_pipe);
2019
        // If this local exchange was inserted because of a serial scan (is_serial_operator),
2020
        // the upstream pipeline (cur_pipe) should have num_tasks=1 (only 1 scan task).
2021
        // We set this now so the exchanger is created with the correct sender count.
2022
        // Child operators added later (serial scan) will also set num_tasks=1, which is
2023
        // consistent with this.
2024
0
        if (op->is_serial_operator() && _parallel_instances > 0) {
2025
0
            cur_pipe->set_num_tasks(_parallel_instances);
2026
0
        }
2027
0
        _dag[downstream_pipeline_id].push_back(cur_pipe->id());
2028
0
        int num_partitions = 0;
2029
0
        std::map<int, int> shuffle_id_to_instance_idx;
2030
0
        auto partition_type = tnode.local_exchange_node.partition_type;
2031
0
        switch (partition_type) {
2032
0
        case TLocalPartitionType::BUCKET_HASH_SHUFFLE:
2033
0
            num_partitions = _params.num_buckets;
2034
0
            shuffle_id_to_instance_idx = _params.bucket_seq_to_instance_idx;
2035
0
            break;
2036
0
        case TLocalPartitionType::LOCAL_EXECUTION_HASH_SHUFFLE:
2037
0
            for (int i = 0; i < _num_instances; i++) {
2038
0
                shuffle_id_to_instance_idx[i] = i;
2039
0
            }
2040
0
            num_partitions = _num_instances;
2041
0
            break;
2042
0
        case TLocalPartitionType::GLOBAL_EXECUTION_HASH_SHUFFLE:
2043
0
            num_partitions = _total_instances;
2044
0
            shuffle_id_to_instance_idx = _params.shuffle_idx_to_instance_idx;
2045
0
            break;
2046
0
        default:
2047
0
            break;
2048
0
        }
2049
0
        auto local_exchange_id = op->operator_id();
2050
0
        auto sink_id = next_sink_operator_id();
2051
0
        DataSinkOperatorPtr sink = std::make_shared<LocalExchangeSinkOperatorX>(
2052
0
                sink_id, local_exchange_id, tnode, num_partitions, shuffle_id_to_instance_idx);
2053
0
        sink_ops.push_back(sink);
2054
0
        RETURN_IF_ERROR(cur_pipe->set_sink(sink));
2055
0
        RETURN_IF_ERROR(cur_pipe->sink()->init(tnode, _runtime_state.get()));
2056
2057
        // For FE-planned local exchange, we need to:
2058
        // 1. Initialize the partitioner for hash shuffle types
2059
        // 2. Defer exchanger creation until after the full plan tree is built
2060
        //    (child operators like serial ExchangeNode may change cur_pipe->num_tasks())
2061
        // 3. Register shared state so pipeline tasks can find it
2062
0
        RETURN_IF_ERROR(static_cast<LocalExchangeSinkOperatorX*>(cur_pipe->sink())
2063
0
                                ->init_partitioner(_runtime_state.get()));
2064
2065
0
        int free_blocks_limit =
2066
0
                _runtime_state->query_options().__isset.local_exchange_free_blocks_limit
2067
0
                        ? cast_set<int>(
2068
0
                                  _runtime_state->query_options().local_exchange_free_blocks_limit)
2069
0
                        : 0;
2070
0
        auto shared_state = LocalExchangeSharedState::create_shared(_num_instances);
2071
0
        shared_state->create_source_dependencies(_num_instances, local_exchange_id,
2072
0
                                                 local_exchange_id, "LOCAL_EXCHANGE_OPERATOR");
2073
0
        shared_state->create_sink_dependency(sink_id, local_exchange_id, "LOCAL_EXCHANGE_SINK");
2074
0
        _op_id_to_shared_state.insert({local_exchange_id, {shared_state, shared_state->sink_deps}});
2075
        // Defer exchanger creation: sender count depends on final upstream num_tasks
2076
0
        _deferred_exchangers.push_back({shared_state, cur_pipe, partition_type, num_partitions,
2077
0
                                        free_blocks_limit, local_exchange_id, sink_id});
2078
0
        break;
2079
0
    }
2080
0
    default:
2081
0
        return Status::InternalError("Unsupported exec type in pipeline: {}",
2082
0
                                     print_plan_node_type(tnode.node_type));
2083
0
    }
2084
0
    if (_params.__isset.parallel_instances && fe_with_old_version) {
2085
0
        cur_pipe->set_num_tasks(_params.parallel_instances);
2086
0
        op->set_serial_operator();
2087
0
    }
2088
2089
0
    return Status::OK();
2090
0
}
2091
// NOLINTEND(readability-function-cognitive-complexity)
2092
// NOLINTEND(readability-function-size)
2093
2094
template <bool is_intersect>
2095
Status PipelineFragmentContext::_build_operators_for_set_operation_node(
2096
        ObjectPool* pool, const TPlanNode& tnode, const DescriptorTbl& descs, OperatorPtr& op,
2097
0
        PipelinePtr& cur_pipe, std::vector<DataSinkOperatorPtr>& sink_ops) {
2098
0
    op.reset(new SetSourceOperatorX<is_intersect>(pool, tnode, next_operator_id(), descs));
2099
0
    RETURN_IF_ERROR(cur_pipe->add_operator(op, _parallel_instances));
2100
2101
0
    const auto downstream_pipeline_id = cur_pipe->id();
2102
0
    if (!_dag.contains(downstream_pipeline_id)) {
2103
0
        _dag.insert({downstream_pipeline_id, {}});
2104
0
    }
2105
2106
0
    for (int child_id = 0; child_id < tnode.num_children; child_id++) {
2107
0
        PipelinePtr probe_side_pipe = add_pipeline(cur_pipe);
2108
0
        _dag[downstream_pipeline_id].push_back(probe_side_pipe->id());
2109
2110
0
        if (child_id == 0) {
2111
0
            sink_ops.push_back(std::make_shared<SetSinkOperatorX<is_intersect>>(
2112
0
                    child_id, next_sink_operator_id(), op->operator_id(), pool, tnode, descs));
2113
0
        } else {
2114
0
            sink_ops.push_back(std::make_shared<SetProbeSinkOperatorX<is_intersect>>(
2115
0
                    child_id, next_sink_operator_id(), op->operator_id(), pool, tnode, descs));
2116
0
        }
2117
0
        RETURN_IF_ERROR(probe_side_pipe->set_sink(sink_ops.back()));
2118
0
        RETURN_IF_ERROR(probe_side_pipe->sink()->init(tnode, _runtime_state.get()));
2119
        // prepare children pipelines. if any pipeline found this as its father, will use the prepared pipeline to build.
2120
0
        _pipeline_parent_map.push(op->node_id(), probe_side_pipe);
2121
0
    }
2122
2123
0
    return Status::OK();
2124
0
}
Unexecuted instantiation: _ZN5doris23PipelineFragmentContext39_build_operators_for_set_operation_nodeILb1EEENS_6StatusEPNS_10ObjectPoolERKNS_9TPlanNodeERKNS_13DescriptorTblERSt10shared_ptrINS_13OperatorXBaseEERSB_INS_8PipelineEERSt6vectorISB_INS_21DataSinkOperatorXBaseEESaISK_EE
Unexecuted instantiation: _ZN5doris23PipelineFragmentContext39_build_operators_for_set_operation_nodeILb0EEENS_6StatusEPNS_10ObjectPoolERKNS_9TPlanNodeERKNS_13DescriptorTblERSt10shared_ptrINS_13OperatorXBaseEERSB_INS_8PipelineEERSt6vectorISB_INS_21DataSinkOperatorXBaseEESaISK_EE
2125
2126
0
Status PipelineFragmentContext::submit() {
2127
0
    if (_submitted) {
2128
0
        return Status::InternalError("submitted");
2129
0
    }
2130
0
    _submitted = true;
2131
2132
0
    int submit_tasks = 0;
2133
0
    Status st;
2134
0
    auto* scheduler = _query_ctx->get_pipe_exec_scheduler();
2135
0
    for (auto& task : _tasks) {
2136
0
        for (auto& t : task) {
2137
0
            st = scheduler->submit(t.first);
2138
0
            DBUG_EXECUTE_IF("PipelineFragmentContext.submit.failed",
2139
0
                            { st = Status::Aborted("PipelineFragmentContext.submit.failed"); });
2140
0
            if (!st) {
2141
0
                cancel(Status::InternalError("submit context to executor fail"));
2142
0
                std::lock_guard<std::mutex> l(_task_mutex);
2143
0
                _total_tasks = submit_tasks;
2144
0
                break;
2145
0
            }
2146
0
            submit_tasks++;
2147
0
        }
2148
0
    }
2149
0
    if (!st.ok()) {
2150
0
        bool need_remove = false;
2151
0
        {
2152
0
            std::lock_guard<std::mutex> l(_task_mutex);
2153
0
            if (_closed_tasks >= _total_tasks) {
2154
0
                need_remove = _close_fragment_instance();
2155
0
            }
2156
0
        }
2157
        // Call remove_pipeline_context() outside _task_mutex to avoid ABBA deadlock.
2158
0
        if (need_remove) {
2159
0
            _exec_env->fragment_mgr()->remove_pipeline_context({_query_id, _fragment_id});
2160
0
        }
2161
0
        return Status::InternalError("Submit pipeline failed. err = {}, BE: {}", st.to_string(),
2162
0
                                     BackendOptions::get_localhost());
2163
0
    } else {
2164
0
        return st;
2165
0
    }
2166
0
}
2167
2168
0
void PipelineFragmentContext::print_profile(const std::string& extra_info) {
2169
0
    if (_runtime_state->enable_profile()) {
2170
0
        std::stringstream ss;
2171
0
        for (auto runtime_profile_ptr : _runtime_state->pipeline_id_to_profile()) {
2172
0
            runtime_profile_ptr->pretty_print(&ss);
2173
0
        }
2174
2175
0
        if (_runtime_state->load_channel_profile()) {
2176
0
            _runtime_state->load_channel_profile()->pretty_print(&ss);
2177
0
        }
2178
2179
0
        auto profile_str =
2180
0
                fmt::format("Query {} fragment {} {}, profile, {}", print_id(this->_query_id),
2181
0
                            this->_fragment_id, extra_info, ss.str());
2182
0
        LOG_LONG_STRING(INFO, profile_str);
2183
0
    }
2184
0
}
2185
// If all pipeline tasks binded to the fragment instance are finished, then we could
2186
// close the fragment instance.
2187
// Returns true if the caller should call remove_pipeline_context() **after** releasing
2188
// _task_mutex. We must not call remove_pipeline_context() here because it acquires
2189
// _pipeline_map's shard lock, and this function is called while _task_mutex is held.
2190
// Acquiring _pipeline_map while holding _task_mutex creates an ABBA deadlock with
2191
// dump_pipeline_tasks(), which acquires _pipeline_map first and then _task_mutex
2192
// (via debug_string()).
2193
0
bool PipelineFragmentContext::_close_fragment_instance() {
2194
0
    if (_is_fragment_instance_closed) {
2195
0
        return false;
2196
0
    }
2197
0
    Defer defer_op {[&]() { _is_fragment_instance_closed = true; }};
2198
0
    _fragment_level_profile->total_time_counter()->update(_fragment_watcher.elapsed_time());
2199
0
    if (!_need_notify_close) {
2200
0
        auto st = send_report(true);
2201
0
        if (!st) {
2202
0
            LOG(WARNING) << fmt::format("Failed to send report for query {}, fragment {}: {}",
2203
0
                                        print_id(_query_id), _fragment_id, st.to_string());
2204
0
        }
2205
0
    }
2206
    // Print profile content in info log is a tempoeray solution for stream load and external_connector.
2207
    // Since stream load does not have someting like coordinator on FE, so
2208
    // backend can not report profile to FE, ant its profile can not be shown
2209
    // in the same way with other query. So we print the profile content to info log.
2210
2211
0
    if (_runtime_state->enable_profile() &&
2212
0
        (_query_ctx->get_query_source() == QuerySource::STREAM_LOAD ||
2213
0
         _query_ctx->get_query_source() == QuerySource::EXTERNAL_CONNECTOR ||
2214
0
         _query_ctx->get_query_source() == QuerySource::GROUP_COMMIT_LOAD)) {
2215
0
        std::stringstream ss;
2216
        // Compute the _local_time_percent before pretty_print the runtime_profile
2217
        // Before add this operation, the print out like that:
2218
        // UNION_NODE (id=0):(Active: 56.720us, non-child: 00.00%)
2219
        // After add the operation, the print out like that:
2220
        // UNION_NODE (id=0):(Active: 56.720us, non-child: 82.53%)
2221
        // We can easily know the exec node execute time without child time consumed.
2222
0
        for (auto runtime_profile_ptr : _runtime_state->pipeline_id_to_profile()) {
2223
0
            runtime_profile_ptr->pretty_print(&ss);
2224
0
        }
2225
2226
0
        if (_runtime_state->load_channel_profile()) {
2227
0
            _runtime_state->load_channel_profile()->pretty_print(&ss);
2228
0
        }
2229
2230
0
        LOG_INFO("Query {} fragment {} profile:\n {}", print_id(_query_id), _fragment_id, ss.str());
2231
0
    }
2232
2233
0
    if (_query_ctx->enable_profile()) {
2234
0
        _query_ctx->add_fragment_profile(_fragment_id, collect_realtime_profile(),
2235
0
                                         collect_realtime_load_channel_profile());
2236
0
    }
2237
2238
    // Return whether the caller needs to remove from the pipeline map.
2239
    // The caller must do this after releasing _task_mutex.
2240
0
    return !_need_notify_close;
2241
0
}
2242
2243
0
void PipelineFragmentContext::decrement_running_task(PipelineId pipeline_id) {
2244
    // If all tasks of this pipeline has been closed, upstream tasks is never needed, and we just make those runnable here
2245
0
    DCHECK(_pip_id_to_pipeline.contains(pipeline_id));
2246
0
    if (_pip_id_to_pipeline[pipeline_id]->close_task()) {
2247
0
        if (_dag.contains(pipeline_id)) {
2248
0
            for (auto dep : _dag[pipeline_id]) {
2249
0
                _pip_id_to_pipeline[dep]->make_all_runnable(pipeline_id);
2250
0
            }
2251
0
        }
2252
0
    }
2253
0
    bool need_remove = false;
2254
0
    {
2255
0
        std::lock_guard<std::mutex> l(_task_mutex);
2256
0
        ++_closed_tasks;
2257
        // Update query-level finished task progress in real time.
2258
0
        _query_ctx->inc_finished_task_num();
2259
0
        if (_closed_tasks >= _total_tasks) {
2260
0
            need_remove = _close_fragment_instance();
2261
0
        }
2262
0
    }
2263
    // Call remove_pipeline_context() outside _task_mutex to avoid ABBA deadlock.
2264
0
    if (need_remove) {
2265
0
        _exec_env->fragment_mgr()->remove_pipeline_context({_query_id, _fragment_id});
2266
0
    }
2267
0
}
2268
2269
0
std::string PipelineFragmentContext::get_load_error_url() {
2270
0
    if (const auto& str = _runtime_state->get_error_log_file_path(); !str.empty()) {
2271
0
        return to_load_error_http_path(str);
2272
0
    }
2273
0
    for (auto& tasks : _tasks) {
2274
0
        for (auto& task : tasks) {
2275
0
            if (const auto& str = task.second->get_error_log_file_path(); !str.empty()) {
2276
0
                return to_load_error_http_path(str);
2277
0
            }
2278
0
        }
2279
0
    }
2280
0
    return "";
2281
0
}
2282
2283
0
std::string PipelineFragmentContext::get_first_error_msg() {
2284
0
    if (const auto& str = _runtime_state->get_first_error_msg(); !str.empty()) {
2285
0
        return str;
2286
0
    }
2287
0
    for (auto& tasks : _tasks) {
2288
0
        for (auto& task : tasks) {
2289
0
            if (const auto& str = task.second->get_first_error_msg(); !str.empty()) {
2290
0
                return str;
2291
0
            }
2292
0
        }
2293
0
    }
2294
0
    return "";
2295
0
}
2296
2297
0
std::string PipelineFragmentContext::_to_http_path(const std::string& file_name) const {
2298
0
    std::stringstream url;
2299
0
    url << "http://" << BackendOptions::get_localhost() << ":" << config::webserver_port
2300
0
        << "/api/_download_load?"
2301
0
        << "token=" << _exec_env->token() << "&file=" << file_name;
2302
0
    return url.str();
2303
0
}
2304
2305
0
void PipelineFragmentContext::_coordinator_callback(const ReportStatusRequest& req) {
2306
0
    DBUG_EXECUTE_IF("FragmentMgr::coordinator_callback.report_delay", {
2307
0
        int random_seconds = req.status.is<ErrorCode::DATA_QUALITY_ERROR>() ? 8 : 2;
2308
0
        LOG_INFO("sleep : ").tag("time", random_seconds).tag("query_id", print_id(req.query_id));
2309
0
        std::this_thread::sleep_for(std::chrono::seconds(random_seconds));
2310
0
        LOG_INFO("sleep done").tag("query_id", print_id(req.query_id));
2311
0
    });
2312
2313
0
    DCHECK(req.status.ok() || req.done); // if !status.ok() => done
2314
0
    if (req.coord_addr.hostname == "external") {
2315
        // External query (flink/spark read tablets) not need to report to FE.
2316
0
        return;
2317
0
    }
2318
0
    int callback_retries = 10;
2319
0
    const int sleep_ms = 1000;
2320
0
    Status exec_status = req.status;
2321
0
    Status coord_status;
2322
0
    std::unique_ptr<FrontendServiceConnection> coord = nullptr;
2323
0
    do {
2324
0
        coord = std::make_unique<FrontendServiceConnection>(_exec_env->frontend_client_cache(),
2325
0
                                                            req.coord_addr, &coord_status);
2326
0
        if (!coord_status.ok()) {
2327
0
            std::this_thread::sleep_for(std::chrono::milliseconds(sleep_ms));
2328
0
        }
2329
0
    } while (!coord_status.ok() && callback_retries-- > 0);
2330
2331
0
    if (!coord_status.ok()) {
2332
0
        UniqueId uid(req.query_id.hi, req.query_id.lo);
2333
0
        static_cast<void>(req.cancel_fn(Status::InternalError(
2334
0
                "query_id: {}, couldn't get a client for {}, reason is {}", uid.to_string(),
2335
0
                PrintThriftNetworkAddress(req.coord_addr), coord_status.to_string())));
2336
0
        return;
2337
0
    }
2338
2339
0
    TReportExecStatusParams params;
2340
0
    params.protocol_version = FrontendServiceVersion::V1;
2341
0
    params.__set_query_id(req.query_id);
2342
0
    params.__set_backend_num(req.backend_num);
2343
0
    params.__set_fragment_instance_id(req.fragment_instance_id);
2344
0
    params.__set_fragment_id(req.fragment_id);
2345
0
    params.__set_status(exec_status.to_thrift());
2346
0
    params.__set_done(req.done);
2347
0
    params.__set_query_type(req.runtime_state->query_type());
2348
0
    params.__isset.profile = false;
2349
2350
0
    DCHECK(req.runtime_state != nullptr);
2351
2352
0
    if (req.runtime_state->query_type() == TQueryType::LOAD) {
2353
0
        params.__set_loaded_rows(req.runtime_state->num_rows_load_total());
2354
0
        params.__set_loaded_bytes(req.runtime_state->num_bytes_load_total());
2355
0
    } else {
2356
0
        DCHECK(!req.runtime_states.empty());
2357
0
        if (!req.runtime_state->output_files().empty()) {
2358
0
            params.__isset.delta_urls = true;
2359
0
            for (auto& it : req.runtime_state->output_files()) {
2360
0
                params.delta_urls.push_back(_to_http_path(it));
2361
0
            }
2362
0
        }
2363
0
        if (!params.delta_urls.empty()) {
2364
0
            params.__isset.delta_urls = true;
2365
0
        }
2366
0
    }
2367
2368
0
    static std::string s_dpp_normal_all = "dpp.norm.ALL";
2369
0
    static std::string s_dpp_abnormal_all = "dpp.abnorm.ALL";
2370
0
    static std::string s_unselected_rows = "unselected.rows";
2371
0
    int64_t num_rows_load_success = 0;
2372
0
    int64_t num_rows_load_filtered = 0;
2373
0
    int64_t num_rows_load_unselected = 0;
2374
0
    if (req.runtime_state->num_rows_load_total() > 0 ||
2375
0
        req.runtime_state->num_rows_load_filtered() > 0 ||
2376
0
        req.runtime_state->num_finished_range() > 0) {
2377
0
        params.__isset.load_counters = true;
2378
2379
0
        num_rows_load_success = req.runtime_state->num_rows_load_success();
2380
0
        num_rows_load_filtered = req.runtime_state->num_rows_load_filtered();
2381
0
        num_rows_load_unselected = req.runtime_state->num_rows_load_unselected();
2382
0
        params.__isset.fragment_instance_reports = true;
2383
0
        TFragmentInstanceReport t;
2384
0
        t.__set_fragment_instance_id(req.runtime_state->fragment_instance_id());
2385
0
        t.__set_num_finished_range(cast_set<int>(req.runtime_state->num_finished_range()));
2386
0
        t.__set_loaded_rows(req.runtime_state->num_rows_load_total());
2387
0
        t.__set_loaded_bytes(req.runtime_state->num_bytes_load_total());
2388
0
        params.fragment_instance_reports.push_back(t);
2389
0
    } else if (!req.runtime_states.empty()) {
2390
0
        for (auto* rs : req.runtime_states) {
2391
0
            if (rs->num_rows_load_total() > 0 || rs->num_rows_load_filtered() > 0 ||
2392
0
                rs->num_finished_range() > 0) {
2393
0
                params.__isset.load_counters = true;
2394
0
                num_rows_load_success += rs->num_rows_load_success();
2395
0
                num_rows_load_filtered += rs->num_rows_load_filtered();
2396
0
                num_rows_load_unselected += rs->num_rows_load_unselected();
2397
0
                params.__isset.fragment_instance_reports = true;
2398
0
                TFragmentInstanceReport t;
2399
0
                t.__set_fragment_instance_id(rs->fragment_instance_id());
2400
0
                t.__set_num_finished_range(cast_set<int>(rs->num_finished_range()));
2401
0
                t.__set_loaded_rows(rs->num_rows_load_total());
2402
0
                t.__set_loaded_bytes(rs->num_bytes_load_total());
2403
0
                params.fragment_instance_reports.push_back(t);
2404
0
            }
2405
0
        }
2406
0
    }
2407
0
    params.load_counters.emplace(s_dpp_normal_all, std::to_string(num_rows_load_success));
2408
0
    params.load_counters.emplace(s_dpp_abnormal_all, std::to_string(num_rows_load_filtered));
2409
0
    params.load_counters.emplace(s_unselected_rows, std::to_string(num_rows_load_unselected));
2410
2411
0
    if (!req.load_error_url.empty()) {
2412
0
        params.__set_tracking_url(req.load_error_url);
2413
0
    }
2414
0
    if (!req.first_error_msg.empty()) {
2415
0
        params.__set_first_error_msg(req.first_error_msg);
2416
0
    }
2417
0
    for (auto* rs : req.runtime_states) {
2418
0
        if (rs->wal_id() > 0) {
2419
0
            params.__set_txn_id(rs->wal_id());
2420
0
            params.__set_label(rs->import_label());
2421
0
        }
2422
0
    }
2423
0
    if (!req.runtime_state->export_output_files().empty()) {
2424
0
        params.__isset.export_files = true;
2425
0
        params.export_files = req.runtime_state->export_output_files();
2426
0
    } else if (!req.runtime_states.empty()) {
2427
0
        for (auto* rs : req.runtime_states) {
2428
0
            if (!rs->export_output_files().empty()) {
2429
0
                params.__isset.export_files = true;
2430
0
                params.export_files.insert(params.export_files.end(),
2431
0
                                           rs->export_output_files().begin(),
2432
0
                                           rs->export_output_files().end());
2433
0
            }
2434
0
        }
2435
0
    }
2436
0
    if (auto tci = req.runtime_state->tablet_commit_infos(); !tci.empty()) {
2437
0
        params.__isset.commitInfos = true;
2438
0
        params.commitInfos.insert(params.commitInfos.end(), tci.begin(), tci.end());
2439
0
    } else if (!req.runtime_states.empty()) {
2440
0
        for (auto* rs : req.runtime_states) {
2441
0
            if (auto rs_tci = rs->tablet_commit_infos(); !rs_tci.empty()) {
2442
0
                params.__isset.commitInfos = true;
2443
0
                params.commitInfos.insert(params.commitInfos.end(), rs_tci.begin(), rs_tci.end());
2444
0
            }
2445
0
        }
2446
0
    }
2447
0
    if (auto eti = req.runtime_state->error_tablet_infos(); !eti.empty()) {
2448
0
        params.__isset.errorTabletInfos = true;
2449
0
        params.errorTabletInfos.insert(params.errorTabletInfos.end(), eti.begin(), eti.end());
2450
0
    } else if (!req.runtime_states.empty()) {
2451
0
        for (auto* rs : req.runtime_states) {
2452
0
            if (auto rs_eti = rs->error_tablet_infos(); !rs_eti.empty()) {
2453
0
                params.__isset.errorTabletInfos = true;
2454
0
                params.errorTabletInfos.insert(params.errorTabletInfos.end(), rs_eti.begin(),
2455
0
                                               rs_eti.end());
2456
0
            }
2457
0
        }
2458
0
    }
2459
0
    if (auto hpu = req.runtime_state->hive_partition_updates(); !hpu.empty()) {
2460
0
        params.__isset.hive_partition_updates = true;
2461
0
        params.hive_partition_updates.insert(params.hive_partition_updates.end(), hpu.begin(),
2462
0
                                             hpu.end());
2463
0
    } else if (!req.runtime_states.empty()) {
2464
0
        for (auto* rs : req.runtime_states) {
2465
0
            if (auto rs_hpu = rs->hive_partition_updates(); !rs_hpu.empty()) {
2466
0
                params.__isset.hive_partition_updates = true;
2467
0
                params.hive_partition_updates.insert(params.hive_partition_updates.end(),
2468
0
                                                     rs_hpu.begin(), rs_hpu.end());
2469
0
            }
2470
0
        }
2471
0
    }
2472
0
    if (auto icd = req.runtime_state->iceberg_commit_datas(); !icd.empty()) {
2473
0
        params.__isset.iceberg_commit_datas = true;
2474
0
        params.iceberg_commit_datas.insert(params.iceberg_commit_datas.end(), icd.begin(),
2475
0
                                           icd.end());
2476
0
    } else if (!req.runtime_states.empty()) {
2477
0
        for (auto* rs : req.runtime_states) {
2478
0
            if (auto rs_icd = rs->iceberg_commit_datas(); !rs_icd.empty()) {
2479
0
                params.__isset.iceberg_commit_datas = true;
2480
0
                params.iceberg_commit_datas.insert(params.iceberg_commit_datas.end(),
2481
0
                                                   rs_icd.begin(), rs_icd.end());
2482
0
            }
2483
0
        }
2484
0
    }
2485
2486
0
    if (auto mcd = req.runtime_state->mc_commit_datas(); !mcd.empty()) {
2487
0
        params.__isset.mc_commit_datas = true;
2488
0
        params.mc_commit_datas.insert(params.mc_commit_datas.end(), mcd.begin(), mcd.end());
2489
0
    } else if (!req.runtime_states.empty()) {
2490
0
        for (auto* rs : req.runtime_states) {
2491
0
            if (auto rs_mcd = rs->mc_commit_datas(); !rs_mcd.empty()) {
2492
0
                params.__isset.mc_commit_datas = true;
2493
0
                params.mc_commit_datas.insert(params.mc_commit_datas.end(), rs_mcd.begin(),
2494
0
                                              rs_mcd.end());
2495
0
            }
2496
0
        }
2497
0
    }
2498
2499
0
    req.runtime_state->get_unreported_errors(&(params.error_log));
2500
0
    params.__isset.error_log = (!params.error_log.empty());
2501
2502
0
    if (_exec_env->cluster_info()->backend_id != 0) {
2503
0
        params.__set_backend_id(_exec_env->cluster_info()->backend_id);
2504
0
    }
2505
2506
0
    TReportExecStatusResult res;
2507
0
    Status rpc_status;
2508
2509
0
    VLOG_DEBUG << "reportExecStatus params is "
2510
0
               << apache::thrift::ThriftDebugString(params).c_str();
2511
0
    if (!exec_status.ok()) {
2512
0
        LOG(WARNING) << "report error status: " << exec_status.msg()
2513
0
                     << " to coordinator: " << req.coord_addr
2514
0
                     << ", query id: " << print_id(req.query_id);
2515
0
    }
2516
0
    try {
2517
0
        try {
2518
0
            (*coord)->reportExecStatus(res, params);
2519
0
        } catch ([[maybe_unused]] apache::thrift::transport::TTransportException& e) {
2520
#ifndef ADDRESS_SANITIZER
2521
            LOG(WARNING) << "Retrying ReportExecStatus. query id: " << print_id(req.query_id)
2522
                         << ", instance id: " << print_id(req.fragment_instance_id) << " to "
2523
                         << req.coord_addr << ", err: " << e.what();
2524
#endif
2525
0
            rpc_status = coord->reopen();
2526
2527
0
            if (!rpc_status.ok()) {
2528
0
                req.cancel_fn(rpc_status);
2529
0
                return;
2530
0
            }
2531
0
            (*coord)->reportExecStatus(res, params);
2532
0
        }
2533
2534
0
        rpc_status = Status::create<false>(res.status);
2535
0
    } catch (apache::thrift::TException& e) {
2536
0
        rpc_status = Status::InternalError("ReportExecStatus() to {} failed: {}",
2537
0
                                           PrintThriftNetworkAddress(req.coord_addr), e.what());
2538
0
    }
2539
2540
0
    if (!rpc_status.ok()) {
2541
0
        LOG_INFO("Going to cancel query {} since report exec status got rpc failed: {}",
2542
0
                 print_id(req.query_id), rpc_status.to_string());
2543
0
        req.cancel_fn(rpc_status);
2544
0
    }
2545
0
}
2546
2547
0
Status PipelineFragmentContext::send_report(bool done) {
2548
0
    Status exec_status = _query_ctx->exec_status();
2549
    // If plan is done successfully, but _is_report_success is false,
2550
    // no need to send report.
2551
    // Load will set _is_report_success to true because load wants to know
2552
    // the process.
2553
0
    if (!_is_report_success && done && exec_status.ok()) {
2554
0
        return Status::OK();
2555
0
    }
2556
2557
    // If both _is_report_success and _is_report_on_cancel are false,
2558
    // which means no matter query is success or failed, no report is needed.
2559
    // This may happen when the query limit reached and
2560
    // a internal cancellation being processed
2561
    // When limit is reached the fragment is also cancelled, but _is_report_on_cancel will
2562
    // be set to false, to avoid sending fault report to FE.
2563
0
    if (!_is_report_success && !_is_report_on_cancel) {
2564
0
        if (done) {
2565
            // if done is true, which means the query is finished successfully, we can safely close the fragment instance without sending report to FE, and just return OK status here.
2566
0
            return Status::OK();
2567
0
        }
2568
0
        return Status::NeedSendAgain("");
2569
0
    }
2570
2571
0
    std::vector<RuntimeState*> runtime_states;
2572
2573
0
    for (auto& tasks : _tasks) {
2574
0
        for (auto& task : tasks) {
2575
0
            runtime_states.push_back(task.second.get());
2576
0
        }
2577
0
    }
2578
2579
0
    std::string load_eror_url = _query_ctx->get_load_error_url().empty()
2580
0
                                        ? get_load_error_url()
2581
0
                                        : _query_ctx->get_load_error_url();
2582
0
    std::string first_error_msg = _query_ctx->get_first_error_msg().empty()
2583
0
                                          ? get_first_error_msg()
2584
0
                                          : _query_ctx->get_first_error_msg();
2585
2586
0
    ReportStatusRequest req {.status = exec_status,
2587
0
                             .runtime_states = runtime_states,
2588
0
                             .done = done || !exec_status.ok(),
2589
0
                             .coord_addr = _query_ctx->coord_addr,
2590
0
                             .query_id = _query_id,
2591
0
                             .fragment_id = _fragment_id,
2592
0
                             .fragment_instance_id = TUniqueId(),
2593
0
                             .backend_num = -1,
2594
0
                             .runtime_state = _runtime_state.get(),
2595
0
                             .load_error_url = load_eror_url,
2596
0
                             .first_error_msg = first_error_msg,
2597
0
                             .cancel_fn = [this](const Status& reason) { cancel(reason); }};
2598
0
    auto ctx = std::dynamic_pointer_cast<PipelineFragmentContext>(shared_from_this());
2599
0
    return _exec_env->fragment_mgr()->get_thread_pool()->submit_func([this, req, ctx]() {
2600
0
        SCOPED_ATTACH_TASK(ctx->get_query_ctx()->query_mem_tracker());
2601
0
        _coordinator_callback(req);
2602
0
        if (!req.done) {
2603
0
            ctx->refresh_next_report_time();
2604
0
        }
2605
0
    });
2606
0
}
2607
2608
0
size_t PipelineFragmentContext::get_revocable_size(bool* has_running_task) const {
2609
0
    size_t res = 0;
2610
    // _tasks will be cleared during ~PipelineFragmentContext, so that it's safe
2611
    // here to traverse the vector.
2612
0
    for (const auto& task_instances : _tasks) {
2613
0
        for (const auto& task : task_instances) {
2614
0
            if (task.first->is_running()) {
2615
0
                LOG_EVERY_N(INFO, 50) << "Query: " << print_id(_query_id)
2616
0
                                      << " is running, task: " << (void*)task.first.get()
2617
0
                                      << ", is_running: " << task.first->is_running();
2618
0
                *has_running_task = true;
2619
0
                return 0;
2620
0
            }
2621
2622
0
            size_t revocable_size = task.first->get_revocable_size();
2623
0
            if (revocable_size >= SpillFile::MIN_SPILL_WRITE_BATCH_MEM) {
2624
0
                res += revocable_size;
2625
0
            }
2626
0
        }
2627
0
    }
2628
0
    return res;
2629
0
}
2630
2631
0
std::vector<PipelineTask*> PipelineFragmentContext::get_revocable_tasks() const {
2632
0
    std::vector<PipelineTask*> revocable_tasks;
2633
0
    for (const auto& task_instances : _tasks) {
2634
0
        for (const auto& task : task_instances) {
2635
0
            size_t revocable_size_ = task.first->get_revocable_size();
2636
2637
0
            if (revocable_size_ >= SpillFile::MIN_SPILL_WRITE_BATCH_MEM) {
2638
0
                revocable_tasks.emplace_back(task.first.get());
2639
0
            }
2640
0
        }
2641
0
    }
2642
0
    return revocable_tasks;
2643
0
}
2644
2645
0
std::string PipelineFragmentContext::debug_string() {
2646
0
    std::lock_guard<std::mutex> l(_task_mutex);
2647
0
    fmt::memory_buffer debug_string_buffer;
2648
0
    fmt::format_to(debug_string_buffer,
2649
0
                   "PipelineFragmentContext Info: _closed_tasks={}, _total_tasks={}, "
2650
0
                   "need_notify_close={}, fragment_id={}, _rec_cte_stage={}\n",
2651
0
                   _closed_tasks, _total_tasks, _need_notify_close, _fragment_id, _rec_cte_stage);
2652
0
    for (size_t j = 0; j < _tasks.size(); j++) {
2653
0
        fmt::format_to(debug_string_buffer, "Tasks in instance {}:\n", j);
2654
0
        for (size_t i = 0; i < _tasks[j].size(); i++) {
2655
0
            fmt::format_to(debug_string_buffer, "Task {}: {}\n", i,
2656
0
                           _tasks[j][i].first->debug_string());
2657
0
        }
2658
0
    }
2659
2660
0
    return fmt::to_string(debug_string_buffer);
2661
0
}
2662
2663
std::vector<std::shared_ptr<TRuntimeProfileTree>>
2664
0
PipelineFragmentContext::collect_realtime_profile() const {
2665
0
    std::vector<std::shared_ptr<TRuntimeProfileTree>> res;
2666
2667
    // we do not have mutex to protect pipeline_id_to_profile
2668
    // so we need to make sure this funciton is invoked after fragment context
2669
    // has already been prepared.
2670
0
    if (!_prepared) {
2671
0
        std::string msg =
2672
0
                "Query " + print_id(_query_id) + " collecting profile, but its not prepared";
2673
0
        DCHECK(false) << msg;
2674
0
        LOG_ERROR(msg);
2675
0
        return res;
2676
0
    }
2677
2678
    // Make sure first profile is fragment level profile
2679
0
    auto fragment_profile = std::make_shared<TRuntimeProfileTree>();
2680
0
    _fragment_level_profile->to_thrift(fragment_profile.get(), _runtime_state->profile_level());
2681
0
    res.push_back(fragment_profile);
2682
2683
    // pipeline_id_to_profile is initialized in prepare stage
2684
0
    for (auto pipeline_profile : _runtime_state->pipeline_id_to_profile()) {
2685
0
        auto profile_ptr = std::make_shared<TRuntimeProfileTree>();
2686
0
        pipeline_profile->to_thrift(profile_ptr.get(), _runtime_state->profile_level());
2687
0
        res.push_back(profile_ptr);
2688
0
    }
2689
2690
0
    return res;
2691
0
}
2692
2693
std::shared_ptr<TRuntimeProfileTree>
2694
0
PipelineFragmentContext::collect_realtime_load_channel_profile() const {
2695
    // we do not have mutex to protect pipeline_id_to_profile
2696
    // so we need to make sure this funciton is invoked after fragment context
2697
    // has already been prepared.
2698
0
    if (!_prepared) {
2699
0
        std::string msg =
2700
0
                "Query " + print_id(_query_id) + " collecting profile, but its not prepared";
2701
0
        DCHECK(false) << msg;
2702
0
        LOG_ERROR(msg);
2703
0
        return nullptr;
2704
0
    }
2705
2706
0
    for (const auto& tasks : _tasks) {
2707
0
        for (const auto& task : tasks) {
2708
0
            if (task.second->load_channel_profile() == nullptr) {
2709
0
                continue;
2710
0
            }
2711
2712
0
            auto tmp_load_channel_profile = std::make_shared<TRuntimeProfileTree>();
2713
2714
0
            task.second->load_channel_profile()->to_thrift(tmp_load_channel_profile.get(),
2715
0
                                                           _runtime_state->profile_level());
2716
0
            _runtime_state->load_channel_profile()->update(*tmp_load_channel_profile);
2717
0
        }
2718
0
    }
2719
2720
0
    auto load_channel_profile = std::make_shared<TRuntimeProfileTree>();
2721
0
    _runtime_state->load_channel_profile()->to_thrift(load_channel_profile.get(),
2722
0
                                                      _runtime_state->profile_level());
2723
0
    return load_channel_profile;
2724
0
}
2725
2726
// Collect runtime filter IDs registered by all tasks in this PFC.
2727
// Used during recursive CTE stage transitions to know which filters to deregister
2728
// before creating the new PFC for the next recursion round.
2729
// Called from rerun_fragment(wait_for_destroy) while tasks are still closing.
2730
// Thread safety: safe because _tasks is structurally immutable after prepare() —
2731
// the vector sizes do not change, and individual RuntimeState filter sets are
2732
// written only during open() which has completed by the time we reach rerun.
2733
0
std::set<int> PipelineFragmentContext::get_deregister_runtime_filter() const {
2734
0
    std::set<int> result;
2735
0
    for (const auto& _task : _tasks) {
2736
0
        for (const auto& task : _task) {
2737
0
            auto set = task.first->runtime_state()->get_deregister_runtime_filter();
2738
0
            result.merge(set);
2739
0
        }
2740
0
    }
2741
0
    if (_runtime_state) {
2742
0
        auto set = _runtime_state->get_deregister_runtime_filter();
2743
0
        result.merge(set);
2744
0
    }
2745
0
    return result;
2746
0
}
2747
2748
27
void PipelineFragmentContext::_release_resource() {
2749
27
    std::lock_guard<std::mutex> l(_task_mutex);
2750
    // The memory released by the query end is recorded in the query mem tracker.
2751
27
    SCOPED_SWITCH_THREAD_MEM_TRACKER_LIMITER(_query_ctx->query_mem_tracker());
2752
27
    auto st = _query_ctx->exec_status();
2753
27
    for (auto& _task : _tasks) {
2754
0
        if (!_task.empty()) {
2755
0
            _call_back(_task.front().first->runtime_state(), &st);
2756
0
        }
2757
0
    }
2758
27
    _tasks.clear();
2759
27
    _dag.clear();
2760
27
    _pip_id_to_pipeline.clear();
2761
27
    _pipelines.clear();
2762
27
    _sink.reset();
2763
27
    _root_op.reset();
2764
27
    _runtime_filter_mgr_map.clear();
2765
27
    _op_id_to_shared_state.clear();
2766
27
}
2767
2768
} // namespace doris