Coverage Report

Created: 2026-04-24 20:42

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