![]() |
|
|||
File indexing completed on 2025-08-28 08:27:10
0001 // Licensed to the Apache Software Foundation (ASF) under one 0002 // or more contributor license agreements. See the NOTICE file 0003 // distributed with this work for additional information 0004 // regarding copyright ownership. The ASF licenses this file 0005 // to you under the Apache License, Version 2.0 (the 0006 // "License"); you may not use this file except in compliance 0007 // with the License. You may obtain a copy of the License at 0008 // 0009 // http://www.apache.org/licenses/LICENSE-2.0 0010 // 0011 // Unless required by applicable law or agreed to in writing, 0012 // software distributed under the License is distributed on an 0013 // "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY 0014 // KIND, either express or implied. See the License for the 0015 // specific language governing permissions and limitations 0016 // under the License. 0017 0018 #pragma once 0019 0020 #include <memory> 0021 #include <utility> 0022 0023 #include "arrow/status.h" 0024 #include "arrow/type_fwd.h" 0025 #include "arrow/util/cancel.h" 0026 #include "arrow/util/functional.h" 0027 #include "arrow/util/macros.h" 0028 #include "arrow/util/type_fwd.h" 0029 #include "arrow/util/visibility.h" 0030 0031 namespace arrow { 0032 namespace internal { 0033 0034 /// \brief A group of related tasks 0035 /// 0036 /// A TaskGroup executes tasks with the signature `Status()`. 0037 /// Execution can be serial or parallel, depending on the TaskGroup 0038 /// implementation. When Finish() returns, it is guaranteed that all 0039 /// tasks have finished, or at least one has errored. 0040 /// 0041 /// Once an error has occurred any tasks that are submitted to the task group 0042 /// will not run. The call to Append will simply return without scheduling the 0043 /// task. 0044 /// 0045 /// If the task group is parallel it is possible that multiple tasks could be 0046 /// running at the same time and one of those tasks fails. This will put the 0047 /// task group in a failure state (so additional tasks cannot be run) however 0048 /// it will not interrupt running tasks. Finish will not complete 0049 /// until all running tasks have finished, even if one task fails. 0050 /// 0051 /// Once a task group has finished new tasks may not be added to it. If you need to start 0052 /// a new batch of work then you should create a new task group. 0053 class ARROW_EXPORT TaskGroup : public std::enable_shared_from_this<TaskGroup> { 0054 public: 0055 /// Add a Status-returning function to execute. Execution order is 0056 /// undefined. The function may be executed immediately or later. 0057 template <typename Function> 0058 void Append(Function&& func) { 0059 return AppendReal(std::forward<Function>(func)); 0060 } 0061 0062 /// Wait for execution of all tasks (and subgroups) to be finished, 0063 /// or for at least one task (or subgroup) to error out. 0064 /// The returned Status propagates the error status of the first failing 0065 /// task (or subgroup). 0066 virtual Status Finish() = 0; 0067 0068 /// Returns a future that will complete the first time all tasks are finished. 0069 /// This should be called only after all top level tasks 0070 /// have been added to the task group. 0071 /// 0072 /// If you are using a TaskGroup asynchronously there are a few considerations to keep 0073 /// in mind. The tasks should not block on I/O, etc (defeats the purpose of using 0074 /// futures) and should not be doing any nested locking or you run the risk of the tasks 0075 /// getting stuck in the thread pool waiting for tasks which cannot get scheduled. 0076 /// 0077 /// Primarily this call is intended to help migrate existing work written with TaskGroup 0078 /// in mind to using futures without having to do a complete conversion on the first 0079 /// pass. 0080 virtual Future<> FinishAsync() = 0; 0081 0082 /// The current aggregate error Status. Non-blocking, useful for stopping early. 0083 virtual Status current_status() = 0; 0084 0085 /// Whether some tasks have already failed. Non-blocking, useful for stopping early. 0086 virtual bool ok() const = 0; 0087 0088 /// How many tasks can typically be executed in parallel. 0089 /// This is only a hint, useful for testing or debugging. 0090 virtual int parallelism() = 0; 0091 0092 static std::shared_ptr<TaskGroup> MakeSerial(StopToken = StopToken::Unstoppable()); 0093 static std::shared_ptr<TaskGroup> MakeThreaded(internal::Executor*, 0094 StopToken = StopToken::Unstoppable()); 0095 0096 virtual ~TaskGroup() = default; 0097 0098 protected: 0099 TaskGroup() = default; 0100 ARROW_DISALLOW_COPY_AND_ASSIGN(TaskGroup); 0101 0102 virtual void AppendReal(FnOnce<Status()> task) = 0; 0103 }; 0104 0105 } // namespace internal 0106 } // namespace arrow
[ Source navigation ] | [ Diff markup ] | [ Identifier search ] | [ general search ] |
This page was automatically generated by the 2.3.7 LXR engine. The LXR team |
![]() ![]() |