Coverage Report

Created: 2026-03-30 11:06

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