diff options
author | Daniel Baumann <daniel.baumann@progress-linux.org> | 2024-04-27 18:24:20 +0000 |
---|---|---|
committer | Daniel Baumann <daniel.baumann@progress-linux.org> | 2024-04-27 18:24:20 +0000 |
commit | 483eb2f56657e8e7f419ab1a4fab8dce9ade8609 (patch) | |
tree | e5d88d25d870d5dedacb6bbdbe2a966086a0a5cf /src/crimson/thread | |
parent | Initial commit. (diff) | |
download | ceph-483eb2f56657e8e7f419ab1a4fab8dce9ade8609.tar.xz ceph-483eb2f56657e8e7f419ab1a4fab8dce9ade8609.zip |
Adding upstream version 14.2.21.upstream/14.2.21upstream
Signed-off-by: Daniel Baumann <daniel.baumann@progress-linux.org>
Diffstat (limited to 'src/crimson/thread')
-rw-r--r-- | src/crimson/thread/Condition.h | 36 | ||||
-rw-r--r-- | src/crimson/thread/ThreadPool.cc | 76 | ||||
-rw-r--r-- | src/crimson/thread/ThreadPool.h | 118 | ||||
-rw-r--r-- | src/crimson/thread/Throttle.cc | 59 | ||||
-rw-r--r-- | src/crimson/thread/Throttle.h | 36 |
5 files changed, 325 insertions, 0 deletions
diff --git a/src/crimson/thread/Condition.h b/src/crimson/thread/Condition.h new file mode 100644 index 00000000..2a5c643d --- /dev/null +++ b/src/crimson/thread/Condition.h @@ -0,0 +1,36 @@ +// -*- mode:C++; tab-width:8; c-basic-offset:2; indent-tabs-mode:t -*- +// vim: ts=8 sw=2 smarttab + +#pragma once + +#include <seastar/core/reactor.hh> +#include <sys/eventfd.h> + +namespace ceph::thread { + +/// a synchronization primitive can be used to block a seastar thread, until +/// another thread notifies it. +class Condition { + seastar::file_desc file_desc; + int fd; + seastar::pollable_fd_state fd_state; + eventfd_t event = 0; +public: + Condition() + : file_desc{seastar::file_desc::eventfd(0, 0)}, + fd(file_desc.get()), + fd_state{std::move(file_desc)} + {} + seastar::future<> wait() { + return seastar::engine().read_some(fd_state, &event, sizeof(event)) + .then([](size_t) { + return seastar::now(); + }); + } + void notify() { + eventfd_t result = 1; + ::eventfd_write(fd, result); + } +}; + +} // namespace ceph::thread diff --git a/src/crimson/thread/ThreadPool.cc b/src/crimson/thread/ThreadPool.cc new file mode 100644 index 00000000..9df849b5 --- /dev/null +++ b/src/crimson/thread/ThreadPool.cc @@ -0,0 +1,76 @@ +#include "ThreadPool.h" + +#include <pthread.h> +#include "crimson/net/Config.h" +#include "include/intarith.h" + +#include "include/ceph_assert.h" + +namespace ceph::thread { + +ThreadPool::ThreadPool(size_t n_threads, + size_t queue_sz, + unsigned cpu_id) + : queue_size{round_up_to(queue_sz, seastar::smp::count)}, + pending{queue_size} +{ + for (size_t i = 0; i < n_threads; i++) { + threads.emplace_back([this, cpu_id] { + pin(cpu_id); + loop(); + }); + } +} + +ThreadPool::~ThreadPool() +{ + for (auto& thread : threads) { + thread.join(); + } +} + +void ThreadPool::pin(unsigned cpu_id) +{ + cpu_set_t cs; + CPU_ZERO(&cs); + CPU_SET(cpu_id, &cs); + [[maybe_unused]] auto r = pthread_setaffinity_np(pthread_self(), + sizeof(cs), &cs); + ceph_assert(r == 0); +} + +void ThreadPool::loop() +{ + for (;;) { + WorkItem* work_item = nullptr; + { + std::unique_lock lock{mutex}; + cond.wait_for(lock, + ceph::net::conf.threadpool_empty_queue_max_wait, + [this, &work_item] { + return pending.pop(work_item) || is_stopping(); + }); + } + if (work_item) { + work_item->process(); + } else if (is_stopping()) { + break; + } + } +} + +seastar::future<> ThreadPool::start() +{ + auto slots_per_shard = queue_size / seastar::smp::count; + return submit_queue.start(slots_per_shard); +} + +seastar::future<> ThreadPool::stop() +{ + return submit_queue.stop().then([this] { + stopping = true; + cond.notify_all(); + }); +} + +} // namespace ceph::thread diff --git a/src/crimson/thread/ThreadPool.h b/src/crimson/thread/ThreadPool.h new file mode 100644 index 00000000..cfd72d2a --- /dev/null +++ b/src/crimson/thread/ThreadPool.h @@ -0,0 +1,118 @@ +// -*- mode:C++; tab-width:8; c-basic-offset:2; indent-tabs-mode:t -*- +// vim: ts=8 sw=2 smarttab +#pragma once + +#include <atomic> +#include <condition_variable> +#include <tuple> +#include <type_traits> +#include <boost/lockfree/queue.hpp> +#include <boost/optional.hpp> +#include <seastar/core/future.hh> +#include <seastar/core/gate.hh> +#include <seastar/core/semaphore.hh> +#include <seastar/core/sharded.hh> + +#include "Condition.h" + +namespace ceph::thread { + +struct WorkItem { + virtual ~WorkItem() {} + virtual void process() = 0; +}; + +template<typename Func, typename T = std::invoke_result_t<Func>> +struct Task final : WorkItem { + Func func; + seastar::future_state<T> state; + ceph::thread::Condition on_done; +public: + explicit Task(Func&& f) + : func(std::move(f)) + {} + void process() override { + try { + state.set(func()); + } catch (...) { + state.set_exception(std::current_exception()); + } + on_done.notify(); + } + seastar::future<T> get_future() { + return on_done.wait().then([this] { + return seastar::make_ready_future<T>(state.get0(std::move(state).get())); + }); + } +}; + +struct SubmitQueue { + seastar::semaphore free_slots; + seastar::gate pending_tasks; + explicit SubmitQueue(size_t num_free_slots) + : free_slots(num_free_slots) + {} + seastar::future<> stop() { + return pending_tasks.close(); + } +}; + +/// an engine for scheduling non-seastar tasks from seastar fibers +class ThreadPool { + std::atomic<bool> stopping = false; + std::mutex mutex; + std::condition_variable cond; + std::vector<std::thread> threads; + seastar::sharded<SubmitQueue> submit_queue; + const size_t queue_size; + boost::lockfree::queue<WorkItem*> pending; + + void loop(); + bool is_stopping() const { + return stopping.load(std::memory_order_relaxed); + } + static void pin(unsigned cpu_id); + seastar::semaphore& local_free_slots() { + return submit_queue.local().free_slots; + } + ThreadPool(const ThreadPool&) = delete; + ThreadPool& operator=(const ThreadPool&) = delete; +public: + /** + * @param queue_sz the depth of pending queue. before a task is scheduled, + * it waits in this queue. we will round this number to + * multiple of the number of cores. + * @param n_threads the number of threads in this thread pool. + * @param cpu the CPU core to which this thread pool is assigned + * @note each @c Task has its own ceph::thread::Condition, which possesses + * possesses an fd, so we should keep the size of queue under a reasonable + * limit. + */ + ThreadPool(size_t n_threads, size_t queue_sz, unsigned cpu); + ~ThreadPool(); + seastar::future<> start(); + seastar::future<> stop(); + template<typename Func, typename...Args> + auto submit(Func&& func, Args&&... args) { + auto packaged = [func=std::move(func), + args=std::forward_as_tuple(args...)] { + return std::apply(std::move(func), std::move(args)); + }; + return seastar::with_gate(submit_queue.local().pending_tasks, + [packaged=std::move(packaged), this] { + return local_free_slots().wait() + .then([packaged=std::move(packaged), this] { + auto task = new Task{std::move(packaged)}; + auto fut = task->get_future(); + pending.push(task); + cond.notify_one(); + return fut.finally([task, this] { + local_free_slots().signal(); + delete task; + }); + }); + }); + } +}; + +} // namespace ceph::thread diff --git a/src/crimson/thread/Throttle.cc b/src/crimson/thread/Throttle.cc new file mode 100644 index 00000000..1d67e723 --- /dev/null +++ b/src/crimson/thread/Throttle.cc @@ -0,0 +1,59 @@ +#include "Throttle.h" + +namespace ceph::thread { + +int64_t Throttle::take(int64_t c) +{ + if (!max) { + return 0; + } + count += c; + return count; +} + +int64_t Throttle::put(int64_t c) +{ + if (!max) { + return 0; + } + if (!c) { + return count; + } + on_free_slots.signal(); + count -= c; + return count; +} + +seastar::future<> Throttle::get(size_t c) +{ + if (!max) { + return seastar::now(); + } + return on_free_slots.wait([this, c] { + return !_should_wait(c); + }).then([this, c] { + count += c; + return seastar::now(); + }); +} + +void Throttle::reset_max(size_t m) { + if (max == m) { + return; + } + + if (m > max) { + on_free_slots.signal(); + } + max = m; +} + +bool Throttle::_should_wait(size_t c) const { + if (!max) { + return false; + } + return ((c <= max && count + c > max) || // normally stay under max + (c >= max && count > max)); // except for large c +} + +} // namespace ceph::thread::seastar diff --git a/src/crimson/thread/Throttle.h b/src/crimson/thread/Throttle.h new file mode 100644 index 00000000..c2342171 --- /dev/null +++ b/src/crimson/thread/Throttle.h @@ -0,0 +1,36 @@ +// -*- mode:C++; tab-width:8; c-basic-offset:2; indent-tabs-mode:nil -*- +// vim: ts=8 sw=2 smarttab + +#pragma once + +#include <seastar/core/condition-variable.hh> + +#include "common/ThrottleInterface.h" + +namespace ceph::thread { + +class Throttle final : public ThrottleInterface { + size_t max = 0; + size_t count = 0; + // we cannot change the "count" of seastar::semaphore after it is created, + // so use condition_variable instead. + seastar::condition_variable on_free_slots; +public: + explicit Throttle(size_t m) + : max(m) + {} + int64_t take(int64_t c = 1) override; + int64_t put(int64_t c = 1) override; + seastar::future<> get(size_t c); + size_t get_current() const { + return count; + } + size_t get_max() const { + return max; + } + void reset_max(size_t m); +private: + bool _should_wait(size_t c) const; +}; + +} // namespace ceph::thread |