Coverage Report

Created: 2026-03-20 06:04

next uncovered line (L), next uncovered region (R), next uncovered branch (B)
be/src/exec/pipeline/pipeline_task.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_task.h"
19
20
#include <fmt/core.h>
21
#include <fmt/format.h>
22
#include <gen_cpp/Metrics_types.h>
23
#include <glog/logging.h>
24
25
#include <algorithm>
26
#include <memory>
27
#include <ostream>
28
#include <vector>
29
30
#include "common/logging.h"
31
#include "common/status.h"
32
#include "core/block/block.h"
33
#include "exec/operator/exchange_source_operator.h"
34
#include "exec/operator/operator.h"
35
#include "exec/operator/rec_cte_source_operator.h"
36
#include "exec/operator/scan_operator.h"
37
#include "exec/pipeline/dependency.h"
38
#include "exec/pipeline/pipeline.h"
39
#include "exec/pipeline/pipeline_fragment_context.h"
40
#include "exec/pipeline/revokable_task.h"
41
#include "exec/pipeline/task_queue.h"
42
#include "exec/pipeline/task_scheduler.h"
43
#include "exec/spill/spill_file.h"
44
#include "runtime/descriptors.h"
45
#include "runtime/exec_env.h"
46
#include "runtime/query_context.h"
47
#include "runtime/runtime_profile.h"
48
#include "runtime/runtime_profile_counter_names.h"
49
#include "runtime/thread_context.h"
50
#include "runtime/workload_group/workload_group_manager.h"
51
#include "util/defer_op.h"
52
#include "util/mem_info.h"
53
#include "util/uid_util.h"
54
55
namespace doris {
56
class RuntimeState;
57
} // namespace doris
58
59
namespace doris {
60
#include "common/compile_check_begin.h"
61
62
PipelineTask::PipelineTask(PipelinePtr& pipeline, uint32_t task_id, RuntimeState* state,
63
                           std::shared_ptr<PipelineFragmentContext> fragment_context,
64
                           RuntimeProfile* parent_profile,
65
                           std::map<int, std::pair<std::shared_ptr<BasicSharedState>,
66
                                                   std::vector<std::shared_ptr<Dependency>>>>
67
                                   shared_state_map,
68
                           int task_idx)
69
        :
70
#ifdef BE_TEST
71
          _query_id(fragment_context ? fragment_context->get_query_id() : TUniqueId()),
72
#else
73
1.72M
          _query_id(fragment_context->get_query_id()),
74
#endif
75
1.72M
          _index(task_id),
76
1.72M
          _pipeline(pipeline),
77
1.72M
          _opened(false),
78
1.72M
          _state(state),
79
1.72M
          _fragment_context(fragment_context),
80
1.72M
          _parent_profile(parent_profile),
81
1.72M
          _operators(pipeline->operators()),
82
1.72M
          _source(_operators.front().get()),
83
1.72M
          _root(_operators.back().get()),
84
1.72M
          _sink(pipeline->sink_shared_pointer()),
85
1.72M
          _shared_state_map(std::move(shared_state_map)),
86
1.72M
          _task_idx(task_idx),
87
1.72M
          _memory_sufficient_dependency(state->get_query_ctx()->get_memory_sufficient_dependency()),
88
1.72M
          _pipeline_name(_pipeline->name()) {
89
1.72M
#ifndef BE_TEST
90
1.72M
    _query_mem_tracker = fragment_context->get_query_ctx()->query_mem_tracker();
91
1.72M
#endif
92
1.72M
    _execution_dependencies.push_back(state->get_query_ctx()->get_execution_dependency());
93
1.72M
    if (!_shared_state_map.contains(_sink->dests_id().front())) {
94
1.44M
        auto shared_state = _sink->create_shared_state();
95
1.44M
        if (shared_state) {
96
1.43M
            _sink_shared_state = shared_state;
97
1.43M
        }
98
1.44M
    }
99
1.72M
}
100
101
1.73M
PipelineTask::~PipelineTask() {
102
1.80M
    auto reset_member = [&]() {
103
1.80M
        _shared_state_map.clear();
104
1.80M
        _sink_shared_state.reset();
105
1.80M
        _op_shared_states.clear();
106
1.80M
        _sink.reset();
107
1.80M
        _operators.clear();
108
1.80M
        _block.reset();
109
1.80M
        _pipeline.reset();
110
1.80M
    };
111
// PipelineTask is also hold by task queue( https://github.com/apache/doris/pull/49753),
112
// so that it maybe the last one to be destructed.
113
// But pipeline task hold some objects, like operators, shared state, etc. So that should release
114
// memory manually.
115
1.73M
#ifndef BE_TEST
116
1.73M
    if (_query_mem_tracker) {
117
1.73M
        SCOPED_SWITCH_THREAD_MEM_TRACKER_LIMITER(_query_mem_tracker);
118
1.73M
        reset_member();
119
1.73M
        return;
120
1.73M
    }
121
18.4E
#endif
122
18.4E
    reset_member();
123
18.4E
}
124
125
Status PipelineTask::prepare(const std::vector<TScanRangeParams>& scan_range, const int sender_id,
126
1.72M
                             const TDataSink& tsink) {
127
1.72M
    DCHECK(_sink);
128
1.72M
    _init_profile();
129
1.72M
    SCOPED_TIMER(_task_profile->total_time_counter());
130
1.72M
    SCOPED_CPU_TIMER(_task_cpu_timer);
131
1.72M
    SCOPED_TIMER(_prepare_timer);
132
1.72M
    DBUG_EXECUTE_IF("fault_inject::PipelineXTask::prepare", {
133
1.72M
        Status status = Status::Error<INTERNAL_ERROR>("fault_inject pipeline_task prepare failed");
134
1.72M
        return status;
135
1.72M
    });
136
1.72M
    {
137
        // set sink local state
138
1.72M
        LocalSinkStateInfo info {_task_idx,         _task_profile.get(),
139
1.72M
                                 sender_id,         get_sink_shared_state().get(),
140
1.72M
                                 _shared_state_map, tsink};
141
1.72M
        RETURN_IF_ERROR(_sink->setup_local_state(_state, info));
142
1.72M
    }
143
144
1.72M
    _scan_ranges = scan_range;
145
1.72M
    auto* parent_profile = _state->get_sink_local_state()->operator_profile();
146
147
4.04M
    for (int op_idx = cast_set<int>(_operators.size() - 1); op_idx >= 0; op_idx--) {
148
2.31M
        auto& op = _operators[op_idx];
149
2.31M
        LocalStateInfo info {parent_profile, _scan_ranges, get_op_shared_state(op->operator_id()),
150
2.31M
                             _shared_state_map, _task_idx};
151
2.31M
        RETURN_IF_ERROR(op->setup_local_state(_state, info));
152
2.31M
        parent_profile = _state->get_local_state(op->operator_id())->operator_profile();
153
2.31M
    }
154
1.72M
    {
155
1.72M
        const auto& deps =
156
1.72M
                _state->get_local_state(_source->operator_id())->execution_dependencies();
157
1.72M
        std::unique_lock<std::mutex> lc(_dependency_lock);
158
1.72M
        std::copy(deps.begin(), deps.end(),
159
1.72M
                  std::inserter(_execution_dependencies, _execution_dependencies.end()));
160
1.72M
    }
161
1.72M
    if (auto fragment = _fragment_context.lock()) {
162
1.72M
        if (fragment->get_query_ctx()->is_cancelled()) {
163
0
            terminate();
164
0
            return fragment->get_query_ctx()->exec_status();
165
0
        }
166
18.4E
    } else {
167
18.4E
        return Status::InternalError("Fragment already finished! Query: {}", print_id(_query_id));
168
18.4E
    }
169
1.72M
    _block = doris::Block::create_unique();
170
1.72M
    return _state_transition(State::RUNNABLE);
171
1.72M
}
172
173
1.72M
Status PipelineTask::_extract_dependencies() {
174
1.72M
    std::vector<std::vector<Dependency*>> read_dependencies;
175
1.72M
    std::vector<Dependency*> write_dependencies;
176
1.72M
    std::vector<Dependency*> finish_dependencies;
177
1.72M
    read_dependencies.resize(_operators.size());
178
1.72M
    size_t i = 0;
179
2.32M
    for (auto& op : _operators) {
180
2.32M
        auto* local_state = _state->get_local_state(op->operator_id());
181
2.32M
        DCHECK(local_state);
182
2.32M
        read_dependencies[i] = local_state->dependencies();
183
2.32M
        auto* fin_dep = local_state->finishdependency();
184
2.32M
        if (fin_dep) {
185
8
            finish_dependencies.push_back(fin_dep);
186
8
        }
187
2.32M
        i++;
188
2.32M
    }
189
1.72M
    DBUG_EXECUTE_IF("fault_inject::PipelineXTask::_extract_dependencies", {
190
1.72M
        Status status = Status::Error<INTERNAL_ERROR>(
191
1.72M
                "fault_inject pipeline_task _extract_dependencies failed");
192
1.72M
        return status;
193
1.72M
    });
194
1.72M
    {
195
1.72M
        auto* local_state = _state->get_sink_local_state();
196
1.72M
        write_dependencies = local_state->dependencies();
197
1.72M
        auto* fin_dep = local_state->finishdependency();
198
1.72M
        if (fin_dep) {
199
747k
            finish_dependencies.push_back(fin_dep);
200
747k
        }
201
1.72M
    }
202
1.72M
    {
203
1.72M
        std::unique_lock<std::mutex> lc(_dependency_lock);
204
1.72M
        read_dependencies.swap(_read_dependencies);
205
1.72M
        write_dependencies.swap(_write_dependencies);
206
1.72M
        finish_dependencies.swap(_finish_dependencies);
207
1.72M
    }
208
1.72M
    return Status::OK();
209
1.72M
}
210
211
997k
bool PipelineTask::inject_shared_state(std::shared_ptr<BasicSharedState> shared_state) {
212
997k
    if (!shared_state) {
213
545k
        return false;
214
545k
    }
215
    // Shared state is created by upstream task's sink operator and shared by source operator of
216
    // this task.
217
566k
    for (auto& op : _operators) {
218
566k
        if (shared_state->related_op_ids.contains(op->operator_id())) {
219
442k
            _op_shared_states.insert({op->operator_id(), shared_state});
220
442k
            return true;
221
442k
        }
222
566k
    }
223
    // Shared state is created by the first sink operator and shared by sink operator of this task.
224
    // For example, Set operations.
225
10.5k
    if (shared_state->related_op_ids.contains(_sink->dests_id().front())) {
226
10.5k
        DCHECK_EQ(_sink_shared_state, nullptr)
227
0
                << " Sink: " << _sink->get_name() << " dest id: " << _sink->dests_id().front();
228
10.5k
        _sink_shared_state = shared_state;
229
10.5k
        return true;
230
10.5k
    }
231
18.4E
    return false;
232
9.10k
}
233
234
1.72M
void PipelineTask::_init_profile() {
235
1.72M
    _task_profile = std::make_unique<RuntimeProfile>(fmt::format("PipelineTask(index={})", _index));
236
1.72M
    _parent_profile->add_child(_task_profile.get(), true, nullptr);
237
1.72M
    _task_cpu_timer = ADD_TIMER(_task_profile, profile::TASK_CPU_TIME);
238
239
1.72M
    static const char* exec_time = profile::EXECUTE_TIME;
240
1.72M
    _exec_timer = ADD_TIMER(_task_profile, exec_time);
241
1.72M
    _prepare_timer = ADD_CHILD_TIMER(_task_profile, profile::PREPARE_TIME, exec_time);
242
1.72M
    _open_timer = ADD_CHILD_TIMER(_task_profile, profile::OPEN_TIME, exec_time);
243
1.72M
    _get_block_timer = ADD_CHILD_TIMER(_task_profile, profile::GET_BLOCK_TIME, exec_time);
244
1.72M
    _get_block_counter = ADD_COUNTER(_task_profile, profile::GET_BLOCK_COUNTER, TUnit::UNIT);
245
1.72M
    _sink_timer = ADD_CHILD_TIMER(_task_profile, profile::SINK_TIME, exec_time);
246
1.72M
    _close_timer = ADD_CHILD_TIMER(_task_profile, profile::CLOSE_TIME, exec_time);
247
248
1.72M
    _wait_worker_timer = ADD_TIMER_WITH_LEVEL(_task_profile, profile::WAIT_WORKER_TIME, 1);
249
250
1.72M
    _schedule_counts = ADD_COUNTER(_task_profile, profile::NUM_SCHEDULE_TIMES, TUnit::UNIT);
251
1.72M
    _yield_counts = ADD_COUNTER(_task_profile, profile::NUM_YIELD_TIMES, TUnit::UNIT);
252
1.72M
    _core_change_times = ADD_COUNTER(_task_profile, profile::CORE_CHANGE_TIMES, TUnit::UNIT);
253
1.72M
    _memory_reserve_times = ADD_COUNTER(_task_profile, profile::MEMORY_RESERVE_TIMES, TUnit::UNIT);
254
1.72M
    _memory_reserve_failed_times =
255
1.72M
            ADD_COUNTER(_task_profile, profile::MEMORY_RESERVE_FAILED_TIMES, TUnit::UNIT);
256
1.72M
}
257
258
1.72M
void PipelineTask::_fresh_profile_counter() {
259
1.72M
    COUNTER_SET(_schedule_counts, (int64_t)_schedule_time);
260
1.72M
    COUNTER_SET(_wait_worker_timer, (int64_t)_wait_worker_watcher.elapsed_time());
261
1.72M
}
262
263
1.72M
Status PipelineTask::_open() {
264
1.72M
    SCOPED_TIMER(_task_profile->total_time_counter());
265
1.72M
    SCOPED_CPU_TIMER(_task_cpu_timer);
266
1.72M
    SCOPED_TIMER(_open_timer);
267
1.72M
    _dry_run = _sink->should_dry_run(_state);
268
2.32M
    for (auto& o : _operators) {
269
2.32M
        RETURN_IF_ERROR(_state->get_local_state(o->operator_id())->open(_state));
270
2.32M
    }
271
1.72M
    RETURN_IF_ERROR(_state->get_sink_local_state()->open(_state));
272
1.72M
    RETURN_IF_ERROR(_extract_dependencies());
273
1.72M
    DBUG_EXECUTE_IF("fault_inject::PipelineXTask::open", {
274
1.72M
        Status status = Status::Error<INTERNAL_ERROR>("fault_inject pipeline_task open failed");
275
1.72M
        return status;
276
1.72M
    });
277
1.72M
    _opened = true;
278
1.72M
    return Status::OK();
279
1.72M
}
280
281
9.81M
Status PipelineTask::_prepare() {
282
9.81M
    SCOPED_TIMER(_task_profile->total_time_counter());
283
9.81M
    SCOPED_CPU_TIMER(_task_cpu_timer);
284
12.9M
    for (auto& o : _operators) {
285
12.9M
        RETURN_IF_ERROR(_state->get_local_state(o->operator_id())->prepare(_state));
286
12.9M
    }
287
9.81M
    RETURN_IF_ERROR(_state->get_sink_local_state()->prepare(_state));
288
9.81M
    return Status::OK();
289
9.81M
}
290
291
5.68M
bool PipelineTask::_wait_to_start() {
292
    // Before task starting, we should make sure
293
    // 1. Execution dependency is ready (which is controlled by FE 2-phase commit)
294
    // 2. Runtime filter dependencies are ready
295
    // 3. All tablets are loaded into local storage
296
5.68M
    return std::any_of(
297
5.68M
            _execution_dependencies.begin(), _execution_dependencies.end(),
298
7.10M
            [&](Dependency* dep) -> bool { return dep->is_blocked_by(shared_from_this()); });
299
5.68M
}
300
301
2.05M
bool PipelineTask::_is_pending_finish() {
302
    // Spilling may be in progress if eos is true.
303
2.05M
    return std::ranges::any_of(_finish_dependencies, [&](Dependency* dep) -> bool {
304
1.08M
        return dep->is_blocked_by(shared_from_this());
305
1.08M
    });
306
2.05M
}
307
308
6.03M
bool PipelineTask::is_blockable() const {
309
    // Before task starting, we should make sure
310
    // 1. Execution dependency is ready (which is controlled by FE 2-phase commit)
311
    // 2. Runtime filter dependencies are ready
312
    // 3. All tablets are loaded into local storage
313
314
6.03M
    if (_state->enable_fuzzy_blockable_task()) {
315
27.0k
        if ((_schedule_time + _task_idx) % 2 == 0) {
316
16.0k
            return true;
317
16.0k
        }
318
27.0k
    }
319
320
6.02M
    return std::ranges::any_of(_operators,
321
8.05M
                               [&](OperatorPtr op) -> bool { return op->is_blockable(_state); }) ||
322
6.02M
           _sink->is_blockable(_state);
323
6.03M
}
324
325
7.90M
bool PipelineTask::_is_blocked() {
326
    // `_dry_run = true` means we do not need data from source operator.
327
7.90M
    if (!_dry_run) {
328
14.5M
        for (int i = cast_set<int>(_read_dependencies.size() - 1); i >= 0; i--) {
329
            // `_read_dependencies` is organized according to operators. For each operator, running condition is met iff all dependencies are ready.
330
10.8M
            for (auto* dep : _read_dependencies[i]) {
331
10.8M
                if (dep->is_blocked_by(shared_from_this())) {
332
2.33M
                    return true;
333
2.33M
                }
334
10.8M
            }
335
            // If all dependencies are ready for this operator, we can execute this task if no datum is needed from upstream operators.
336
6.80M
            if (!_operators[i]->need_more_input_data(_state)) {
337
90.0k
                break;
338
90.0k
            }
339
6.80M
        }
340
7.88M
    }
341
5.56M
    return _memory_sufficient_dependency->is_blocked_by(shared_from_this()) ||
342
9.64M
           std::ranges::any_of(_write_dependencies, [&](Dependency* dep) -> bool {
343
9.64M
               return dep->is_blocked_by(shared_from_this());
344
9.64M
           });
345
7.90M
}
346
347
1.05M
void PipelineTask::terminate() {
348
    // We use a lock to assure all dependencies are not deconstructed here.
349
1.05M
    std::unique_lock<std::mutex> lc(_dependency_lock);
350
1.05M
    auto fragment = _fragment_context.lock();
351
1.05M
    if (!is_finalized() && fragment) {
352
42.4k
        try {
353
42.4k
            DCHECK(_wake_up_early || fragment->is_canceled());
354
42.4k
            std::ranges::for_each(_write_dependencies,
355
109k
                                  [&](Dependency* dep) { dep->set_always_ready(); });
356
42.4k
            std::ranges::for_each(_finish_dependencies,
357
42.4k
                                  [&](Dependency* dep) { dep->set_always_ready(); });
358
49.5k
            std::ranges::for_each(_read_dependencies, [&](std::vector<Dependency*>& deps) {
359
52.2k
                std::ranges::for_each(deps, [&](Dependency* dep) { dep->set_always_ready(); });
360
49.5k
            });
361
            // All `_execution_deps` will never be set blocking from ready. So we just set ready here.
362
42.4k
            std::ranges::for_each(_execution_dependencies,
363
60.3k
                                  [&](Dependency* dep) { dep->set_ready(); });
364
42.4k
            _memory_sufficient_dependency->set_ready();
365
42.4k
        } catch (const doris::Exception& e) {
366
0
            LOG(WARNING) << "Terminate failed: " << e.code() << ", " << e.to_string();
367
0
        }
368
42.4k
    }
369
1.05M
}
370
371
// When current memory pressure is low, memory usage may increase significantly in the next
372
// operator run, while there is no revocable memory available for spilling.
373
// Trigger memory revoking when pressure is high and revocable memory is significant.
374
// Memory pressure is evaluated using two signals:
375
// 1. Query memory usage exceeds a threshold ratio of the query memory limit.
376
// 2. Workload group memory usage reaches the workload group low-watermark threshold.
377
6.76M
bool PipelineTask::_should_trigger_revoking(const size_t reserve_size) const {
378
6.76M
    if (!_state->enable_spill()) {
379
6.71M
        return false;
380
6.71M
    }
381
382
49.9k
    auto query_mem_tracker = _state->get_query_ctx()->query_mem_tracker();
383
49.9k
    auto wg = _state->get_query_ctx()->workload_group();
384
57.5k
    if (!query_mem_tracker || !wg) {
385
1.48k
        return false;
386
1.48k
    }
387
388
48.4k
    const auto parallelism = std::max(1, _pipeline->num_tasks());
389
48.4k
    const auto query_water_mark = 90; // 90%
390
48.4k
    const auto group_mem_limit = wg->memory_limit();
391
48.4k
    auto query_limit = query_mem_tracker->limit();
392
48.4k
    if (query_limit <= 0) {
393
1
        query_limit = group_mem_limit;
394
48.4k
    } else if (query_limit > group_mem_limit && group_mem_limit > 0) {
395
2.13k
        query_limit = group_mem_limit;
396
2.13k
    }
397
398
48.4k
    if (query_limit <= 0) {
399
1
        return false;
400
1
    }
401
402
56.0k
    if ((reserve_size * parallelism) <= (query_limit / 5)) {
403
56.0k
        return false;
404
56.0k
    }
405
406
18.4E
    bool is_high_memory_pressure = false;
407
18.4E
    const auto used_mem = query_mem_tracker->consumption() + reserve_size * parallelism;
408
18.4E
    if (used_mem >= int64_t((double(query_limit) * query_water_mark / 100))) {
409
2
        is_high_memory_pressure = true;
410
2
    }
411
412
18.4E
    if (!is_high_memory_pressure) {
413
4
        bool is_low_watermark;
414
4
        bool is_high_watermark;
415
4
        wg->check_mem_used(&is_low_watermark, &is_high_watermark);
416
4
        is_high_memory_pressure = is_low_watermark || is_high_watermark;
417
4
    }
418
419
18.4E
    if (is_high_memory_pressure) {
420
4
        const auto revocable_size = [&]() {
421
4
            size_t total = _sink->revocable_mem_size(_state);
422
4
            for (const auto& op : _operators) {
423
4
                total += op->revocable_mem_size(_state);
424
4
            }
425
4
            return total;
426
4
        }();
427
428
4
        const auto total_estimated_revocable = revocable_size * parallelism;
429
4
        return total_estimated_revocable >= int64_t(double(query_limit) * 0.2);
430
4
    }
431
432
18.4E
    return false;
433
18.4E
}
434
435
/**
436
 * `_eos` indicates whether the execution phase is done. `done` indicates whether we could close
437
 * this task.
438
 *
439
 * For example,
440
 * 1. if `_eos` is false which means we should continue to get next block so we cannot close (e.g.
441
 *    `done` is false)
442
 * 2. if `_eos` is true which means all blocks from source are exhausted but `_is_pending_finish()`
443
 *    is true which means we should wait for a pending dependency ready (maybe a running rpc), so we
444
 *    cannot close (e.g. `done` is false)
445
 * 3. if `_eos` is true which means all blocks from source are exhausted and `_is_pending_finish()`
446
 *    is false which means we can close immediately (e.g. `done` is true)
447
 * @param done
448
 * @return
449
 */
450
6.03M
Status PipelineTask::execute(bool* done) {
451
6.03M
    if (_exec_state != State::RUNNABLE || _blocked_dep != nullptr) [[unlikely]] {
452
#ifdef BE_TEST
453
        return Status::InternalError("Pipeline task is not runnable! Task info: {}",
454
                                     debug_string());
455
#else
456
1
        return Status::FatalError("Pipeline task is not runnable! Task info: {}", debug_string());
457
1
#endif
458
1
    }
459
460
6.03M
    auto fragment_context = _fragment_context.lock();
461
6.03M
    if (!fragment_context) {
462
0
        return Status::InternalError("Fragment already finished! Query: {}", print_id(_query_id));
463
0
    }
464
6.03M
    int64_t time_spent = 0;
465
6.03M
    ThreadCpuStopWatch cpu_time_stop_watch;
466
6.03M
    cpu_time_stop_watch.start();
467
6.03M
    SCOPED_ATTACH_TASK(_state);
468
6.03M
    Defer running_defer {[&]() {
469
6.02M
        int64_t delta_cpu_time = cpu_time_stop_watch.elapsed_time();
470
6.02M
        _task_cpu_timer->update(delta_cpu_time);
471
6.02M
        fragment_context->get_query_ctx()->resource_ctx()->cpu_context()->update_cpu_cost_ms(
472
6.02M
                delta_cpu_time);
473
474
        // If task is woke up early, we should terminate all operators, and this task could be closed immediately.
475
6.02M
        if (_wake_up_early) {
476
8.11k
            terminate();
477
8.11k
            THROW_IF_ERROR(_root->terminate(_state));
478
8.11k
            THROW_IF_ERROR(_sink->terminate(_state));
479
8.11k
            _eos = true;
480
8.11k
            *done = true;
481
6.01M
        } else if (_eos && !_spilling &&
482
6.01M
                   (fragment_context->is_canceled() || !_is_pending_finish())) {
483
1.72M
            *done = true;
484
1.72M
        }
485
6.02M
    }};
486
6.03M
    const auto query_id = _state->query_id();
487
    // If this task is already EOS and block is empty (which means we already output all blocks),
488
    // just return here.
489
6.03M
    if (_eos && !_spilling) {
490
341k
        return Status::OK();
491
341k
    }
492
    // If this task is blocked by a spilling request and waken up immediately, the spilling
493
    // dependency will not block this task and we should just run here.
494
5.69M
    if (!_block->empty()) {
495
0
        LOG(INFO) << "Query: " << print_id(query_id) << " has pending block, size: "
496
0
                  << PrettyPrinter::print_bytes(_block->allocated_bytes());
497
0
        DCHECK(_spilling);
498
0
    }
499
500
5.69M
    SCOPED_TIMER(_task_profile->total_time_counter());
501
5.69M
    SCOPED_TIMER(_exec_timer);
502
503
5.70M
    if (!_wake_up_early) {
504
5.70M
        RETURN_IF_ERROR(_prepare());
505
5.70M
    }
506
5.69M
    DBUG_EXECUTE_IF("fault_inject::PipelineXTask::execute", {
507
5.69M
        Status status = Status::Error<INTERNAL_ERROR>("fault_inject pipeline_task execute failed");
508
5.69M
        return status;
509
5.69M
    });
510
    // `_wake_up_early` must be after `_wait_to_start()`
511
5.69M
    if (_wait_to_start() || _wake_up_early) {
512
1.56M
        return Status::OK();
513
1.56M
    }
514
4.13M
    RETURN_IF_ERROR(_prepare());
515
516
    // The status must be runnable
517
4.13M
    if (!_opened && !fragment_context->is_canceled()) {
518
1.73M
        DBUG_EXECUTE_IF("PipelineTask::execute.open_sleep", {
519
1.73M
            auto required_pipeline_id =
520
1.73M
                    DebugPoints::instance()->get_debug_param_or_default<int32_t>(
521
1.73M
                            "PipelineTask::execute.open_sleep", "pipeline_id", -1);
522
1.73M
            auto required_task_id = DebugPoints::instance()->get_debug_param_or_default<int32_t>(
523
1.73M
                    "PipelineTask::execute.open_sleep", "task_id", -1);
524
1.73M
            if (required_pipeline_id == pipeline_id() && required_task_id == task_id()) {
525
1.73M
                LOG(WARNING) << "PipelineTask::execute.open_sleep sleep 5s";
526
1.73M
                sleep(5);
527
1.73M
            }
528
1.73M
        });
529
530
1.73M
        SCOPED_RAW_TIMER(&time_spent);
531
1.73M
        RETURN_IF_ERROR(_open());
532
1.73M
    }
533
534
7.91M
    while (!fragment_context->is_canceled()) {
535
7.91M
        SCOPED_RAW_TIMER(&time_spent);
536
7.91M
        Defer defer {[&]() {
537
            // If this run is pended by a spilling request, the block will be output in next run.
538
7.90M
            if (!_spilling) {
539
7.90M
                _block->clear_column_data(_root->row_desc().num_materialized_slots());
540
7.90M
            }
541
7.90M
        }};
542
        // `_wake_up_early` must be after `_is_blocked()`
543
7.91M
        if (_is_blocked() || _wake_up_early) {
544
2.35M
            return Status::OK();
545
2.35M
        }
546
547
        /// When a task is cancelled,
548
        /// its blocking state will be cleared and it will transition to a ready state (though it is not truly ready).
549
        /// Here, checking whether it is cancelled to prevent tasks in a blocking state from being re-executed.
550
5.55M
        if (fragment_context->is_canceled()) {
551
0
            break;
552
0
        }
553
554
5.55M
        if (time_spent > _exec_time_slice) {
555
46.7k
            COUNTER_UPDATE(_yield_counts, 1);
556
46.7k
            break;
557
46.7k
        }
558
5.50M
        auto* block = _block.get();
559
560
5.50M
        DBUG_EXECUTE_IF("fault_inject::PipelineXTask::executing", {
561
5.50M
            Status status =
562
5.50M
                    Status::Error<INTERNAL_ERROR>("fault_inject pipeline_task executing failed");
563
5.50M
            return status;
564
5.50M
        });
565
566
        // `_sink->is_finished(_state)` means sink operator should be finished
567
5.50M
        if (_sink->is_finished(_state)) {
568
10
            set_wake_up_early();
569
10
            return Status::OK();
570
10
        }
571
572
        // `_dry_run` means sink operator need no more data
573
5.50M
        _eos = _dry_run || _eos;
574
5.50M
        _spilling = false;
575
5.50M
        auto workload_group = _state->workload_group();
576
        // If last run is pended by a spilling request, `_block` is produced with some rows in last
577
        // run, so we will resume execution using the block.
578
5.50M
        if (!_eos && _block->empty()) {
579
5.49M
            SCOPED_TIMER(_get_block_timer);
580
5.49M
            if (_state->low_memory_mode()) {
581
1.52k
                _sink->set_low_memory_mode(_state);
582
1.53k
                for (auto& op : _operators) {
583
1.53k
                    op->set_low_memory_mode(_state);
584
1.53k
                }
585
1.52k
            }
586
5.49M
            DEFER_RELEASE_RESERVED();
587
5.49M
            _get_block_counter->update(1);
588
            // Sum reserve sizes across all operators in this pipeline.
589
            // Each operator reports only its own requirement (non-recursive).
590
5.49M
            size_t reserve_size = 0;
591
6.29M
            for (auto& op : _operators) {
592
6.29M
                reserve_size += op->get_reserve_mem_size(_state);
593
6.29M
                op->reset_reserve_mem_size(_state);
594
6.29M
            }
595
5.49M
            if (workload_group &&
596
5.49M
                _state->get_query_ctx()
597
4.70M
                        ->resource_ctx()
598
4.70M
                        ->task_controller()
599
4.70M
                        ->is_enable_reserve_memory() &&
600
5.49M
                reserve_size > 0) {
601
4.66M
                if (_should_trigger_revoking(reserve_size)) {
602
0
                    LOG(INFO) << fmt::format(
603
0
                            "Query: {} sink: {}, node id: {}, task id: {}, reserve size: {} when "
604
0
                            "high memory pressure, try to spill",
605
0
                            print_id(_query_id), _sink->get_name(), _sink->node_id(),
606
0
                            _state->task_id(), reserve_size);
607
0
                    ExecEnv::GetInstance()->workload_group_mgr()->add_paused_query(
608
0
                            _state->get_query_ctx()->resource_ctx()->shared_from_this(),
609
0
                            reserve_size,
610
0
                            Status::Error<ErrorCode::QUERY_MEMORY_EXCEEDED>(
611
0
                                    "high memory pressure, try to spill"));
612
0
                    _spilling = true;
613
0
                    continue;
614
0
                }
615
4.66M
                if (!_try_to_reserve_memory(reserve_size, _root)) {
616
777
                    continue;
617
777
                }
618
4.66M
            }
619
620
5.49M
            bool eos = false;
621
5.49M
            RETURN_IF_ERROR(_root->get_block_after_projects(_state, block, &eos));
622
5.48M
            RETURN_IF_ERROR(block->check_type_and_column());
623
5.48M
            _eos = eos;
624
5.48M
        }
625
626
5.50M
        if (!_block->empty() || _eos) {
627
2.29M
            SCOPED_TIMER(_sink_timer);
628
2.29M
            Status status = Status::OK();
629
2.29M
            DEFER_RELEASE_RESERVED();
630
2.29M
            if (_state->get_query_ctx()
631
2.29M
                        ->resource_ctx()
632
2.29M
                        ->task_controller()
633
2.29M
                        ->is_enable_reserve_memory() &&
634
2.29M
                workload_group && !(_wake_up_early || _dry_run)) {
635
2.27M
                const auto sink_reserve_size = _sink->get_reserve_mem_size(_state, _eos);
636
637
2.27M
                if (sink_reserve_size > 0 && _should_trigger_revoking(sink_reserve_size)) {
638
0
                    LOG(INFO) << fmt::format(
639
0
                            "Query: {} sink: {}, node id: {}, task id: {}, reserve size: {} when "
640
0
                            "high memory pressure, try to spill",
641
0
                            print_id(_query_id), _sink->get_name(), _sink->node_id(),
642
0
                            _state->task_id(), sink_reserve_size);
643
0
                    ExecEnv::GetInstance()->workload_group_mgr()->add_paused_query(
644
0
                            _state->get_query_ctx()->resource_ctx()->shared_from_this(),
645
0
                            sink_reserve_size,
646
0
                            Status::Error<ErrorCode::QUERY_MEMORY_EXCEEDED>(
647
0
                                    "high memory pressure, try to spill"));
648
0
                    _spilling = true;
649
0
                    continue;
650
0
                }
651
652
2.27M
                if (sink_reserve_size > 0 &&
653
2.27M
                    !_try_to_reserve_memory(sink_reserve_size, _sink.get())) {
654
726
                    continue;
655
726
                }
656
2.27M
            }
657
658
2.29M
            DBUG_EXECUTE_IF("PipelineTask::execute.sink_eos_sleep", {
659
2.29M
                auto required_pipeline_id =
660
2.29M
                        DebugPoints::instance()->get_debug_param_or_default<int32_t>(
661
2.29M
                                "PipelineTask::execute.sink_eos_sleep", "pipeline_id", -1);
662
2.29M
                auto required_task_id =
663
2.29M
                        DebugPoints::instance()->get_debug_param_or_default<int32_t>(
664
2.29M
                                "PipelineTask::execute.sink_eos_sleep", "task_id", -1);
665
2.29M
                if (required_pipeline_id == pipeline_id() && required_task_id == task_id()) {
666
2.29M
                    LOG(WARNING) << "PipelineTask::execute.sink_eos_sleep sleep 10s";
667
2.29M
                    sleep(10);
668
2.29M
                }
669
2.29M
            });
670
671
2.29M
            DBUG_EXECUTE_IF("PipelineTask::execute.terminate", {
672
2.29M
                if (_eos) {
673
2.29M
                    auto required_pipeline_id =
674
2.29M
                            DebugPoints::instance()->get_debug_param_or_default<int32_t>(
675
2.29M
                                    "PipelineTask::execute.terminate", "pipeline_id", -1);
676
2.29M
                    auto required_task_id =
677
2.29M
                            DebugPoints::instance()->get_debug_param_or_default<int32_t>(
678
2.29M
                                    "PipelineTask::execute.terminate", "task_id", -1);
679
2.29M
                    auto required_fragment_id =
680
2.29M
                            DebugPoints::instance()->get_debug_param_or_default<int32_t>(
681
2.29M
                                    "PipelineTask::execute.terminate", "fragment_id", -1);
682
2.29M
                    if (required_pipeline_id == pipeline_id() && required_task_id == task_id() &&
683
2.29M
                        fragment_context->get_fragment_id() == required_fragment_id) {
684
2.29M
                        _wake_up_early = true;
685
2.29M
                        terminate();
686
2.29M
                    } else if (required_pipeline_id == pipeline_id() &&
687
2.29M
                               fragment_context->get_fragment_id() == required_fragment_id) {
688
2.29M
                        LOG(WARNING) << "PipelineTask::execute.terminate sleep 5s";
689
2.29M
                        sleep(5);
690
2.29M
                    }
691
2.29M
                }
692
2.29M
            });
693
2.29M
            RETURN_IF_ERROR(block->check_type_and_column());
694
2.29M
            status = _sink->sink(_state, block, _eos);
695
696
2.29M
            if (_eos) {
697
1.73M
                if (_sink->reset_to_rerun(_state, _root)) {
698
1.87k
                    _eos = false;
699
1.72M
                } else {
700
1.72M
                    RETURN_IF_ERROR(close(Status::OK(), false));
701
1.72M
                }
702
1.73M
            }
703
704
2.29M
            if (status.is<ErrorCode::END_OF_FILE>()) {
705
6
                set_wake_up_early();
706
6
                return Status::OK();
707
2.29M
            } else if (!status) {
708
16
                return status;
709
16
            }
710
711
2.29M
            if (_eos) { // just return, the scheduler will do finish work
712
1.72M
                return Status::OK();
713
1.72M
            }
714
2.29M
        }
715
5.50M
    }
716
717
45.2k
    RETURN_IF_ERROR(_state->get_query_ctx()->get_pipe_exec_scheduler()->submit(shared_from_this()));
718
45.2k
    return Status::OK();
719
45.2k
}
720
721
23
Status PipelineTask::do_revoke_memory(const std::shared_ptr<SpillContext>& spill_context) {
722
23
    auto fragment_context = _fragment_context.lock();
723
23
    if (!fragment_context) {
724
1
        return Status::InternalError("Fragment already finished! Query: {}", print_id(_query_id));
725
1
    }
726
727
22
    SCOPED_ATTACH_TASK(_state);
728
22
    ThreadCpuStopWatch cpu_time_stop_watch;
729
22
    cpu_time_stop_watch.start();
730
22
    Defer running_defer {[&]() {
731
22
        int64_t delta_cpu_time = cpu_time_stop_watch.elapsed_time();
732
22
        _task_cpu_timer->update(delta_cpu_time);
733
22
        fragment_context->get_query_ctx()->resource_ctx()->cpu_context()->update_cpu_cost_ms(
734
22
                delta_cpu_time);
735
736
        // If task is woke up early, we should terminate all operators, and this task could be closed immediately.
737
22
        if (_wake_up_early) {
738
1
            terminate();
739
1
            THROW_IF_ERROR(_root->terminate(_state));
740
1
            THROW_IF_ERROR(_sink->terminate(_state));
741
1
            _eos = true;
742
1
        }
743
744
        // SpillContext tracks pipeline task count, not operator count.
745
        // Notify completion once after all operators + sink have finished revoking.
746
22
        if (spill_context) {
747
19
            spill_context->on_task_finished();
748
19
        }
749
22
    }};
750
751
    // Revoke memory from every operator that has enough revocable memory,
752
    // then revoke from the sink.
753
22
    for (auto& op : _operators) {
754
22
        if (op->revocable_mem_size(_state) >= SpillFile::MIN_SPILL_WRITE_BATCH_MEM) {
755
2
            RETURN_IF_ERROR(op->revoke_memory(_state));
756
2
        }
757
22
    }
758
759
22
    if (_sink->revocable_mem_size(_state) >= SpillFile::MIN_SPILL_WRITE_BATCH_MEM) {
760
17
        RETURN_IF_ERROR(_sink->revoke_memory(_state));
761
17
    }
762
22
    return Status::OK();
763
22
}
764
765
6.78M
bool PipelineTask::_try_to_reserve_memory(const size_t reserve_size, OperatorBase* op) {
766
6.78M
    auto st = thread_context()->thread_mem_tracker_mgr->try_reserve(reserve_size);
767
    // If reserve memory failed and the query is not enable spill, just disable reserve memory(this will enable
768
    // memory hard limit check, and will cancel the query if allocate memory failed) and let it run.
769
6.78M
    if (!st.ok() && !_state->enable_spill()) {
770
2
        LOG(INFO) << print_id(_query_id) << " reserve memory failed due to " << st
771
2
                  << ", and it is not enable spill, disable reserve memory and let it run";
772
2
        _state->get_query_ctx()->resource_ctx()->task_controller()->disable_reserve_memory();
773
2
        return true;
774
2
    }
775
6.78M
    COUNTER_UPDATE(_memory_reserve_times, 1);
776
777
    // Compute total revocable memory across all operators and the sink.
778
6.78M
    size_t total_revocable_mem_size = 0;
779
6.78M
    size_t operator_max_revocable_mem_size = 0;
780
781
6.78M
    if (!st.ok() || _state->enable_force_spill()) {
782
        // Compute total revocable memory across all operators and the sink.
783
23.7k
        total_revocable_mem_size = _sink->revocable_mem_size(_state);
784
23.7k
        operator_max_revocable_mem_size = total_revocable_mem_size;
785
26.3k
        for (auto& cur_op : _operators) {
786
26.3k
            total_revocable_mem_size += cur_op->revocable_mem_size(_state);
787
26.3k
            operator_max_revocable_mem_size =
788
26.3k
                    std::max(cur_op->revocable_mem_size(_state), operator_max_revocable_mem_size);
789
26.3k
        }
790
23.7k
    }
791
792
    // During enable force spill, other operators like scan opeartor will also try to reserve memory and will failed
793
    // here, if not add this check, it will always paused and resumed again.
794
6.78M
    if (st.ok() && _state->enable_force_spill()) {
795
22.2k
        if (operator_max_revocable_mem_size >= _state->spill_min_revocable_mem()) {
796
16
            st = Status::Error<ErrorCode::QUERY_MEMORY_EXCEEDED>(
797
16
                    "force spill and there is an operator has memory "
798
16
                    "size {} exceeds min mem size {}",
799
16
                    PrettyPrinter::print_bytes(operator_max_revocable_mem_size),
800
16
                    PrettyPrinter::print_bytes(_state->spill_min_revocable_mem()));
801
16
        }
802
22.2k
    }
803
804
6.78M
    if (!st.ok()) {
805
1.50k
        COUNTER_UPDATE(_memory_reserve_failed_times, 1);
806
        // build per-operator revocable memory info string for debugging
807
1.50k
        std::string ops_revocable_info;
808
1.50k
        {
809
1.50k
            fmt::memory_buffer buf;
810
1.50k
            for (auto& cur_op : _operators) {
811
1.50k
                fmt::format_to(buf, "{}({})-> ", cur_op->get_name(),
812
1.50k
                               PrettyPrinter::print_bytes(cur_op->revocable_mem_size(_state)));
813
1.50k
            }
814
1.50k
            if (_sink) {
815
1.50k
                fmt::format_to(buf, "{}({}) ", _sink->get_name(),
816
1.50k
                               PrettyPrinter::print_bytes(_sink->revocable_mem_size(_state)));
817
1.50k
            }
818
1.50k
            ops_revocable_info = fmt::to_string(buf);
819
1.50k
        }
820
821
1.50k
        auto debug_msg = fmt::format(
822
1.50k
                "Query: {} , try to reserve: {}, total revocable mem size: {}, failed reason: {}",
823
1.50k
                print_id(_query_id), PrettyPrinter::print_bytes(reserve_size),
824
1.50k
                PrettyPrinter::print_bytes(total_revocable_mem_size), st.to_string());
825
1.50k
        if (!ops_revocable_info.empty()) {
826
1.50k
            debug_msg += fmt::format(", ops_revocable=[{}]", ops_revocable_info);
827
1.50k
        }
828
        // PROCESS_MEMORY_EXCEEDED error msg already contains process_mem_log_str
829
1.50k
        if (!st.is<ErrorCode::PROCESS_MEMORY_EXCEEDED>()) {
830
1.50k
            debug_msg +=
831
1.50k
                    fmt::format(", debug info: {}", GlobalMemoryArbitrator::process_mem_log_str());
832
1.50k
        }
833
1.50k
        LOG(INFO) << debug_msg;
834
1.50k
        ExecEnv::GetInstance()->workload_group_mgr()->add_paused_query(
835
1.50k
                _state->get_query_ctx()->resource_ctx()->shared_from_this(), reserve_size, st);
836
1.50k
        _spilling = true;
837
1.50k
        return false;
838
1.50k
    }
839
6.78M
    return true;
840
6.78M
}
841
842
1.35M
void PipelineTask::stop_if_finished() {
843
1.35M
    auto fragment = _fragment_context.lock();
844
1.35M
    if (!fragment) {
845
0
        return;
846
0
    }
847
1.35M
    SCOPED_SWITCH_THREAD_MEM_TRACKER_LIMITER(fragment->get_query_ctx()->query_mem_tracker());
848
1.35M
    if (auto sink = _sink) {
849
1.07M
        if (sink->is_finished(_state)) {
850
2.49k
            set_wake_up_early();
851
2.49k
            terminate();
852
2.49k
        }
853
1.07M
    }
854
1.35M
}
855
856
1.72M
Status PipelineTask::finalize() {
857
1.72M
    auto fragment = _fragment_context.lock();
858
1.72M
    if (!fragment) {
859
0
        return Status::OK();
860
0
    }
861
1.72M
    SCOPED_SWITCH_THREAD_MEM_TRACKER_LIMITER(fragment->get_query_ctx()->query_mem_tracker());
862
1.72M
    RETURN_IF_ERROR(_state_transition(State::FINALIZED));
863
1.72M
    std::unique_lock<std::mutex> lc(_dependency_lock);
864
1.72M
    _sink_shared_state.reset();
865
1.72M
    _op_shared_states.clear();
866
1.72M
    _shared_state_map.clear();
867
1.72M
    _block.reset();
868
1.72M
    _operators.clear();
869
1.72M
    _sink.reset();
870
1.72M
    _pipeline.reset();
871
1.72M
    return Status::OK();
872
1.72M
}
873
874
3.45M
Status PipelineTask::close(Status exec_status, bool close_sink) {
875
3.45M
    int64_t close_ns = 0;
876
3.45M
    Status s;
877
3.45M
    {
878
3.45M
        SCOPED_RAW_TIMER(&close_ns);
879
3.45M
        if (close_sink) {
880
1.73M
            s = _sink->close(_state, exec_status);
881
1.73M
        }
882
4.65M
        for (auto& op : _operators) {
883
4.65M
            auto tem = op->close(_state);
884
4.65M
            if (!tem.ok() && s.ok()) {
885
9
                s = std::move(tem);
886
9
            }
887
4.65M
        }
888
3.45M
    }
889
3.46M
    if (_opened) {
890
3.46M
        COUNTER_UPDATE(_close_timer, close_ns);
891
3.46M
        COUNTER_UPDATE(_task_profile->total_time_counter(), close_ns);
892
3.46M
    }
893
894
3.45M
    if (close_sink && _opened) {
895
1.73M
        _task_profile->add_info_string("WakeUpEarly", std::to_string(_wake_up_early.load()));
896
1.73M
        _fresh_profile_counter();
897
1.73M
    }
898
899
3.45M
    if (close_sink) {
900
1.73M
        RETURN_IF_ERROR(_state_transition(State::FINISHED));
901
1.73M
    }
902
3.45M
    return s;
903
3.45M
}
904
905
46.8k
std::string PipelineTask::debug_string() {
906
46.8k
    fmt::memory_buffer debug_string_buffer;
907
908
46.8k
    fmt::format_to(debug_string_buffer, "QueryId: {}\n", print_id(_query_id));
909
46.8k
    fmt::format_to(debug_string_buffer, "InstanceId: {}\n",
910
46.8k
                   print_id(_state->fragment_instance_id()));
911
912
46.8k
    fmt::format_to(debug_string_buffer,
913
46.8k
                   "PipelineTask[id = {}, open = {}, eos = {}, state = {}, dry run = "
914
46.8k
                   "{}, _wake_up_early = {}, _wake_up_by = {}, time elapsed since last state "
915
46.8k
                   "changing = {}s, spilling = {}, is running = {}]",
916
46.8k
                   _index, _opened, _eos, _to_string(_exec_state), _dry_run, _wake_up_early.load(),
917
46.8k
                   _wake_by, _state_change_watcher.elapsed_time() / NANOS_PER_SEC, _spilling,
918
46.8k
                   is_running());
919
46.8k
    std::unique_lock<std::mutex> lc(_dependency_lock);
920
46.8k
    auto* cur_blocked_dep = _blocked_dep;
921
46.8k
    auto fragment = _fragment_context.lock();
922
46.8k
    if (is_finalized() || !fragment) {
923
1.20k
        fmt::format_to(debug_string_buffer, " pipeline name = {}", _pipeline_name);
924
1.20k
        return fmt::to_string(debug_string_buffer);
925
1.20k
    }
926
45.6k
    auto elapsed = fragment->elapsed_time() / NANOS_PER_SEC;
927
45.6k
    fmt::format_to(debug_string_buffer, " elapse time = {}s, block dependency = [{}]\n", elapsed,
928
45.6k
                   cur_blocked_dep && !is_finalized() ? cur_blocked_dep->debug_string() : "NULL");
929
930
45.6k
    if (_state && _state->local_runtime_filter_mgr()) {
931
434
        fmt::format_to(debug_string_buffer, "local_runtime_filter_mgr: [{}]\n",
932
434
                       _state->local_runtime_filter_mgr()->debug_string());
933
434
    }
934
935
45.6k
    fmt::format_to(debug_string_buffer, "operators: ");
936
91.2k
    for (size_t i = 0; i < _operators.size(); i++) {
937
45.6k
        fmt::format_to(debug_string_buffer, "\n{}",
938
45.6k
                       _opened && !is_finalized()
939
45.6k
                               ? _operators[i]->debug_string(_state, cast_set<int>(i))
940
45.6k
                               : _operators[i]->debug_string(cast_set<int>(i)));
941
45.6k
    }
942
45.6k
    fmt::format_to(debug_string_buffer, "\n{}\n",
943
45.6k
                   _opened && !is_finalized()
944
45.6k
                           ? _sink->debug_string(_state, cast_set<int>(_operators.size()))
945
45.6k
                           : _sink->debug_string(cast_set<int>(_operators.size())));
946
947
45.6k
    fmt::format_to(debug_string_buffer, "\nRead Dependency Information: \n");
948
949
45.6k
    size_t i = 0;
950
91.0k
    for (; i < _read_dependencies.size(); i++) {
951
90.8k
        for (size_t j = 0; j < _read_dependencies[i].size(); j++) {
952
45.4k
            fmt::format_to(debug_string_buffer, "{}. {}\n", i,
953
45.4k
                           _read_dependencies[i][j]->debug_string(cast_set<int>(i) + 1));
954
45.4k
        }
955
45.4k
    }
956
957
45.6k
    fmt::format_to(debug_string_buffer, "{}. {}\n", i,
958
45.6k
                   _memory_sufficient_dependency->debug_string(cast_set<int>(i++)));
959
960
45.6k
    fmt::format_to(debug_string_buffer, "\nWrite Dependency Information: \n");
961
91.2k
    for (size_t j = 0; j < _write_dependencies.size(); j++, i++) {
962
45.6k
        fmt::format_to(debug_string_buffer, "{}. {}\n", i,
963
45.6k
                       _write_dependencies[j]->debug_string(cast_set<int>(j) + 1));
964
45.6k
    }
965
966
45.6k
    fmt::format_to(debug_string_buffer, "\nExecution Dependency Information: \n");
967
136k
    for (size_t j = 0; j < _execution_dependencies.size(); j++, i++) {
968
91.2k
        fmt::format_to(debug_string_buffer, "{}. {}\n", i,
969
91.2k
                       _execution_dependencies[j]->debug_string(cast_set<int>(i) + 1));
970
91.2k
    }
971
972
45.6k
    fmt::format_to(debug_string_buffer, "Finish Dependency Information: \n");
973
136k
    for (size_t j = 0; j < _finish_dependencies.size(); j++, i++) {
974
90.5k
        fmt::format_to(debug_string_buffer, "{}. {}\n", i,
975
90.5k
                       _finish_dependencies[j]->debug_string(cast_set<int>(i) + 1));
976
90.5k
    }
977
45.6k
    return fmt::to_string(debug_string_buffer);
978
46.8k
}
979
980
694
size_t PipelineTask::get_revocable_size() const {
981
694
    if (!_opened || is_finalized() || _running || (_eos && !_spilling)) {
982
66
        return 0;
983
66
    }
984
985
    // Sum revocable memory from every operator in the pipeline + the sink.
986
    // Each operator reports only its own revocable memory (no child recursion).
987
628
    size_t total = _sink->revocable_mem_size(_state);
988
724
    for (const auto& op : _operators) {
989
724
        total += op->revocable_mem_size(_state);
990
724
    }
991
628
    return total;
992
694
}
993
994
19
Status PipelineTask::revoke_memory(const std::shared_ptr<SpillContext>& spill_context) {
995
19
    DCHECK(spill_context);
996
19
    if (is_finalized()) {
997
1
        spill_context->on_task_finished();
998
1
        VLOG_DEBUG << "Query: " << print_id(_state->query_id()) << ", task: " << ((void*)this)
999
0
                   << " finalized";
1000
1
        return Status::OK();
1001
1
    }
1002
1003
18
    const auto revocable_size = get_revocable_size();
1004
18
    if (revocable_size >= SpillFile::MIN_SPILL_WRITE_BATCH_MEM) {
1005
17
        auto revokable_task = std::make_shared<RevokableTask>(shared_from_this(), spill_context);
1006
        // Submit a revocable task to run, the run method will call revoke memory. Currently the
1007
        // underline pipeline task is still blocked.
1008
17
        RETURN_IF_ERROR(_state->get_query_ctx()->get_pipe_exec_scheduler()->submit(revokable_task));
1009
17
    } else {
1010
1
        spill_context->on_task_finished();
1011
1
        VLOG_DEBUG << "Query: " << print_id(_state->query_id()) << ", task: " << ((void*)this)
1012
0
                   << " has not enough data to revoke: " << revocable_size;
1013
1
    }
1014
18
    return Status::OK();
1015
18
}
1016
1017
4.25M
Status PipelineTask::wake_up(Dependency* dep, std::unique_lock<std::mutex>& /* dep_lock */) {
1018
    // call by dependency
1019
4.25M
    DCHECK_EQ(_blocked_dep, dep) << "dep : " << dep->debug_string(0) << "task: " << debug_string();
1020
4.25M
    _blocked_dep = nullptr;
1021
4.25M
    auto holder = std::dynamic_pointer_cast<PipelineTask>(shared_from_this());
1022
4.25M
    RETURN_IF_ERROR(_state_transition(PipelineTask::State::RUNNABLE));
1023
4.26M
    if (auto f = _fragment_context.lock(); f) {
1024
4.26M
        RETURN_IF_ERROR(_state->get_query_ctx()->get_pipe_exec_scheduler()->submit(holder));
1025
4.26M
    }
1026
4.25M
    return Status::OK();
1027
4.25M
}
1028
1029
13.6M
Status PipelineTask::_state_transition(State new_state) {
1030
13.6M
    if (_exec_state != new_state) {
1031
13.6M
        _state_change_watcher.reset();
1032
13.6M
        _state_change_watcher.start();
1033
13.6M
    }
1034
13.6M
    _task_profile->add_info_string("TaskState", _to_string(new_state));
1035
13.6M
    _task_profile->add_info_string("BlockedByDependency", _blocked_dep ? _blocked_dep->name() : "");
1036
13.6M
    if (!LEGAL_STATE_TRANSITION[(int)new_state].contains(_exec_state)) {
1037
17
        return Status::InternalError(
1038
17
                "Task state transition from {} to {} is not allowed! Task info: {}",
1039
17
                _to_string(_exec_state), _to_string(new_state), debug_string());
1040
17
    }
1041
13.6M
    _exec_state = new_state;
1042
13.6M
    return Status::OK();
1043
13.6M
}
1044
1045
#include "common/compile_check_end.h"
1046
} // namespace doris