Fix periodic_task unable to re-register the same task type (#10379)
Summary: Timer has a limitation that it cannot re-register a task with the same name, because the cancel only mark the task as invalid and wait for the Timer thread to clean it up later, before the task is cleaned up, the same task name cannot be added. Which makes the task option update likely to fail, which basically cancel and re-register the same task name. Change the periodic task name to a random unique id and store it in periodic_task_scheduler. Also refactor the `periodic_work` to `periodic_task` to make each job function as a `task`. Pull Request resolved: https://github.com/facebook/rocksdb/pull/10379 Test Plan: unittests Reviewed By: ajkr Differential Revision: D38000615 Pulled By: jay-zhuang fbshipit-source-id: e4135f9422e3b53aaec8eda54f4e18ce633a279emain
parent
3f57d84af4
commit
d9e71fb2c5
@ -0,0 +1,113 @@ |
||||
// Copyright (c) Meta Platforms, Inc. and affiliates.
|
||||
//
|
||||
// This source code is licensed under both the GPLv2 (found in the
|
||||
// COPYING file in the root directory) and Apache 2.0 License
|
||||
// (found in the LICENSE.Apache file in the root directory).
|
||||
|
||||
#include "db/periodic_task_scheduler.h" |
||||
|
||||
#include "rocksdb/system_clock.h" |
||||
|
||||
#ifndef ROCKSDB_LITE |
||||
namespace ROCKSDB_NAMESPACE { |
||||
|
||||
// `timer_mutex` is a global mutex serves 3 purposes currently:
|
||||
// (1) to ensure calls to `Start()` and `Shutdown()` are serialized, as
|
||||
// they are currently not implemented in a thread-safe way; and
|
||||
// (2) to ensure the `Timer::Add()`s and `Timer::Start()` run atomically, and
|
||||
// the `Timer::Cancel()`s and `Timer::Shutdown()` run atomically.
|
||||
// (3) protect tasks_map_ in PeriodicTaskScheduler
|
||||
// Note: It's not efficient to have a static global mutex, for
|
||||
// PeriodicTaskScheduler it should be okay, as the operations are called
|
||||
// infrequently.
|
||||
static port::Mutex timer_mutex; |
||||
|
||||
static const std::map<PeriodicTaskType, uint64_t> kDefaultPeriodSeconds = { |
||||
{PeriodicTaskType::kDumpStats, kInvalidPeriodSec}, |
||||
{PeriodicTaskType::kPersistStats, kInvalidPeriodSec}, |
||||
{PeriodicTaskType::kFlushInfoLog, 10}, |
||||
{PeriodicTaskType::kRecordSeqnoTime, kInvalidPeriodSec}, |
||||
}; |
||||
|
||||
static const std::map<PeriodicTaskType, std::string> kPeriodicTaskTypeNames = { |
||||
{PeriodicTaskType::kDumpStats, "dump_st"}, |
||||
{PeriodicTaskType::kPersistStats, "pst_st"}, |
||||
{PeriodicTaskType::kFlushInfoLog, "flush_info_log"}, |
||||
{PeriodicTaskType::kRecordSeqnoTime, "record_seq_time"}, |
||||
}; |
||||
|
||||
Status PeriodicTaskScheduler::Register(PeriodicTaskType task_type, |
||||
const PeriodicTaskFunc& fn) { |
||||
return Register(task_type, fn, kDefaultPeriodSeconds.at(task_type)); |
||||
} |
||||
|
||||
Status PeriodicTaskScheduler::Register(PeriodicTaskType task_type, |
||||
const PeriodicTaskFunc& fn, |
||||
uint64_t repeat_period_seconds) { |
||||
MutexLock l(&timer_mutex); |
||||
static std::atomic<uint64_t> initial_delay(0); |
||||
|
||||
if (repeat_period_seconds == kInvalidPeriodSec) { |
||||
return Status::InvalidArgument("Invalid task repeat period"); |
||||
} |
||||
auto it = tasks_map_.find(task_type); |
||||
if (it != tasks_map_.end()) { |
||||
// the task already exists and it's the same, no update needed
|
||||
if (it->second.repeat_every_sec == repeat_period_seconds) { |
||||
return Status::OK(); |
||||
} |
||||
// cancel the existing one before register new one
|
||||
timer_->Cancel(it->second.name); |
||||
tasks_map_.erase(it); |
||||
} |
||||
|
||||
timer_->Start(); |
||||
// put task type name as prefix, for easy debug
|
||||
std::string unique_id = |
||||
kPeriodicTaskTypeNames.at(task_type) + std::to_string(id_++); |
||||
|
||||
bool succeeded = timer_->Add( |
||||
fn, unique_id, |
||||
(initial_delay.fetch_add(1) % repeat_period_seconds) * kMicrosInSecond, |
||||
repeat_period_seconds * kMicrosInSecond); |
||||
if (!succeeded) { |
||||
return Status::Aborted("Failed to register periodic task"); |
||||
} |
||||
auto result = tasks_map_.try_emplace( |
||||
task_type, TaskInfo{unique_id, repeat_period_seconds}); |
||||
if (!result.second) { |
||||
return Status::Aborted("Failed to add periodic task"); |
||||
}; |
||||
return Status::OK(); |
||||
} |
||||
|
||||
Status PeriodicTaskScheduler::Unregister(PeriodicTaskType task_type) { |
||||
MutexLock l(&timer_mutex); |
||||
auto it = tasks_map_.find(task_type); |
||||
if (it != tasks_map_.end()) { |
||||
timer_->Cancel(it->second.name); |
||||
tasks_map_.erase(it); |
||||
} |
||||
if (!timer_->HasPendingTask()) { |
||||
timer_->Shutdown(); |
||||
} |
||||
return Status::OK(); |
||||
} |
||||
|
||||
Timer* PeriodicTaskScheduler::Default() { |
||||
static Timer timer(SystemClock::Default().get()); |
||||
return &timer; |
||||
} |
||||
|
||||
#ifndef NDEBUG |
||||
void PeriodicTaskScheduler::TEST_OverrideTimer(SystemClock* clock) { |
||||
static Timer test_timer(clock); |
||||
test_timer.TEST_OverrideTimer(clock); |
||||
MutexLock l(&timer_mutex); |
||||
timer_ = &test_timer; |
||||
} |
||||
#endif // NDEBUG
|
||||
|
||||
} // namespace ROCKSDB_NAMESPACE
|
||||
|
||||
#endif // ROCKSDB_LITE
|
@ -0,0 +1,110 @@ |
||||
// Copyright (c) Meta Platforms, Inc. and affiliates.
|
||||
//
|
||||
// This source code is licensed under both the GPLv2 (found in the
|
||||
// COPYING file in the root directory) and Apache 2.0 License
|
||||
// (found in the LICENSE.Apache file in the root directory).
|
||||
|
||||
#pragma once |
||||
|
||||
#ifndef ROCKSDB_LITE |
||||
|
||||
#include "util/timer.h" |
||||
|
||||
namespace ROCKSDB_NAMESPACE { |
||||
class SystemClock; |
||||
|
||||
using PeriodicTaskFunc = std::function<void()>; |
||||
|
||||
constexpr uint64_t kInvalidPeriodSec = 0; |
||||
|
||||
// List of task types
|
||||
enum class PeriodicTaskType : uint8_t { |
||||
kDumpStats = 0, |
||||
kPersistStats, |
||||
kFlushInfoLog, |
||||
kRecordSeqnoTime, |
||||
kMax, |
||||
}; |
||||
|
||||
// PeriodicTaskScheduler contains the periodic task scheduled from the DB
|
||||
// instance. It's used to schedule/unschedule DumpStats(), PersistStats(),
|
||||
// FlushInfoLog(), etc. Each type of the task can only have one instance,
|
||||
// re-register the same task type would only update the repeat period.
|
||||
//
|
||||
// Internally, it uses a global single threaded timer object to run the periodic
|
||||
// task functions. Timer thread will always be started since the info log
|
||||
// flushing cannot be disabled.
|
||||
class PeriodicTaskScheduler { |
||||
public: |
||||
explicit PeriodicTaskScheduler() = default; |
||||
|
||||
PeriodicTaskScheduler(const PeriodicTaskScheduler&) = delete; |
||||
PeriodicTaskScheduler(PeriodicTaskScheduler&&) = delete; |
||||
PeriodicTaskScheduler& operator=(const PeriodicTaskScheduler&) = delete; |
||||
PeriodicTaskScheduler& operator=(PeriodicTaskScheduler&&) = delete; |
||||
|
||||
// Register a task with its default repeat period
|
||||
Status Register(PeriodicTaskType task_type, const PeriodicTaskFunc& fn); |
||||
|
||||
// Register a task with specified repeat period. 0 is an invalid argument
|
||||
// (kInvalidPeriodSec). To stop the task, please use Unregister() specifically
|
||||
Status Register(PeriodicTaskType task_type, const PeriodicTaskFunc& fn, |
||||
uint64_t repeat_period_seconds); |
||||
|
||||
// Unregister the task
|
||||
Status Unregister(PeriodicTaskType task_type); |
||||
|
||||
#ifndef NDEBUG |
||||
// Override the timer for the unittest
|
||||
void TEST_OverrideTimer(SystemClock* clock); |
||||
|
||||
// Call Timer TEST_WaitForRun() which wait until Timer starting waiting.
|
||||
void TEST_WaitForRun(const std::function<void()>& callback) const { |
||||
if (timer_ != nullptr) { |
||||
timer_->TEST_WaitForRun(callback); |
||||
} |
||||
} |
||||
|
||||
// Get global valid task number in the Timer
|
||||
size_t TEST_GetValidTaskNum() const { |
||||
if (timer_ != nullptr) { |
||||
return timer_->TEST_GetPendingTaskNum(); |
||||
} |
||||
return 0; |
||||
} |
||||
|
||||
// If it has the specified task type registered
|
||||
bool TEST_HasTask(PeriodicTaskType task_type) const { |
||||
auto it = tasks_map_.find(task_type); |
||||
return it != tasks_map_.end(); |
||||
} |
||||
#endif // NDEBUG
|
||||
|
||||
private: |
||||
// default global Timer instance
|
||||
static Timer* Default(); |
||||
|
||||
// Internal structure to store task information
|
||||
struct TaskInfo { |
||||
TaskInfo(std::string _name, uint64_t _repeat_every_sec) |
||||
: name(std::move(_name)), repeat_every_sec(_repeat_every_sec) {} |
||||
std::string name; |
||||
uint64_t repeat_every_sec; |
||||
}; |
||||
|
||||
// Internal tasks map
|
||||
std::map<PeriodicTaskType, TaskInfo> tasks_map_; |
||||
|
||||
// Global timer pointer, which doesn't support synchronous add/cancel tasks
|
||||
// so having a global `timer_mutex` for add/cancel task.
|
||||
Timer* timer_ = Default(); |
||||
|
||||
// Global task id, protected by the global `timer_mutex`
|
||||
inline static uint64_t id_; |
||||
|
||||
static constexpr uint64_t kMicrosInSecond = 1000U * 1000U; |
||||
}; |
||||
|
||||
} // namespace ROCKSDB_NAMESPACE
|
||||
|
||||
#endif // ROCKSDB_LITE
|
@ -1,168 +0,0 @@ |
||||
// Copyright (c) 2011-present, Facebook, Inc. All rights reserved.
|
||||
// This source code is licensed under both the GPLv2 (found in the
|
||||
// COPYING file in the root directory) and Apache 2.0 License
|
||||
// (found in the LICENSE.Apache file in the root directory).
|
||||
|
||||
#include "db/periodic_work_scheduler.h" |
||||
|
||||
#include "db/db_impl/db_impl.h" |
||||
#include "rocksdb/system_clock.h" |
||||
|
||||
#ifndef ROCKSDB_LITE |
||||
namespace ROCKSDB_NAMESPACE { |
||||
|
||||
const std::string PeriodicWorkTaskNames::kDumpStats = "dump_st"; |
||||
const std::string PeriodicWorkTaskNames::kPersistStats = "pst_st"; |
||||
const std::string PeriodicWorkTaskNames::kFlushInfoLog = "flush_info_log"; |
||||
const std::string PeriodicWorkTaskNames::kRecordSeqnoTime = "record_seq_time"; |
||||
|
||||
PeriodicWorkScheduler::PeriodicWorkScheduler( |
||||
const std::shared_ptr<SystemClock>& clock) { |
||||
timer = std::unique_ptr<Timer>(new Timer(clock.get())); |
||||
} |
||||
|
||||
Status PeriodicWorkScheduler::Register(DBImpl* dbi, |
||||
unsigned int stats_dump_period_sec, |
||||
unsigned int stats_persist_period_sec) { |
||||
MutexLock l(&timer_mu_); |
||||
static std::atomic<uint64_t> initial_delay(0); |
||||
timer->Start(); |
||||
if (stats_dump_period_sec > 0) { |
||||
bool succeeded = timer->Add( |
||||
[dbi]() { dbi->DumpStats(); }, |
||||
GetTaskName(dbi, PeriodicWorkTaskNames::kDumpStats), |
||||
initial_delay.fetch_add(1) % |
||||
static_cast<uint64_t>(stats_dump_period_sec) * kMicrosInSecond, |
||||
static_cast<uint64_t>(stats_dump_period_sec) * kMicrosInSecond); |
||||
if (!succeeded) { |
||||
return Status::Aborted("Unable to add periodic task DumpStats"); |
||||
} |
||||
} |
||||
if (stats_persist_period_sec > 0) { |
||||
bool succeeded = timer->Add( |
||||
[dbi]() { dbi->PersistStats(); }, |
||||
GetTaskName(dbi, PeriodicWorkTaskNames::kPersistStats), |
||||
initial_delay.fetch_add(1) % |
||||
static_cast<uint64_t>(stats_persist_period_sec) * kMicrosInSecond, |
||||
static_cast<uint64_t>(stats_persist_period_sec) * kMicrosInSecond); |
||||
if (!succeeded) { |
||||
return Status::Aborted("Unable to add periodic task PersistStats"); |
||||
} |
||||
} |
||||
bool succeeded = |
||||
timer->Add([dbi]() { dbi->FlushInfoLog(); }, |
||||
GetTaskName(dbi, PeriodicWorkTaskNames::kFlushInfoLog), |
||||
initial_delay.fetch_add(1) % kDefaultFlushInfoLogPeriodSec * |
||||
kMicrosInSecond, |
||||
kDefaultFlushInfoLogPeriodSec * kMicrosInSecond); |
||||
if (!succeeded) { |
||||
return Status::Aborted("Unable to add periodic task FlushInfoLog"); |
||||
} |
||||
return Status::OK(); |
||||
} |
||||
|
||||
Status PeriodicWorkScheduler::RegisterRecordSeqnoTimeWorker( |
||||
DBImpl* dbi, uint64_t record_cadence_sec) { |
||||
MutexLock l(&timer_mu_); |
||||
timer->Start(); |
||||
static std::atomic_uint64_t initial_delay(0); |
||||
bool succeeded = timer->Add( |
||||
[dbi]() { dbi->RecordSeqnoToTimeMapping(); }, |
||||
GetTaskName(dbi, PeriodicWorkTaskNames::kRecordSeqnoTime), |
||||
initial_delay.fetch_add(1) % record_cadence_sec * kMicrosInSecond, |
||||
record_cadence_sec * kMicrosInSecond); |
||||
if (!succeeded) { |
||||
return Status::NotSupported( |
||||
"Updating seqno to time worker cadence is not supported yet"); |
||||
} |
||||
return Status::OK(); |
||||
} |
||||
|
||||
void PeriodicWorkScheduler::UnregisterRecordSeqnoTimeWorker(DBImpl* dbi) { |
||||
MutexLock l(&timer_mu_); |
||||
timer->Cancel(GetTaskName(dbi, PeriodicWorkTaskNames::kRecordSeqnoTime)); |
||||
if (!timer->HasPendingTask()) { |
||||
timer->Shutdown(); |
||||
} |
||||
} |
||||
|
||||
void PeriodicWorkScheduler::Unregister(DBImpl* dbi) { |
||||
MutexLock l(&timer_mu_); |
||||
timer->Cancel(GetTaskName(dbi, PeriodicWorkTaskNames::kDumpStats)); |
||||
timer->Cancel(GetTaskName(dbi, PeriodicWorkTaskNames::kPersistStats)); |
||||
timer->Cancel(GetTaskName(dbi, PeriodicWorkTaskNames::kFlushInfoLog)); |
||||
if (!timer->HasPendingTask()) { |
||||
timer->Shutdown(); |
||||
} |
||||
} |
||||
|
||||
PeriodicWorkScheduler* PeriodicWorkScheduler::Default() { |
||||
// Always use the default SystemClock for the scheduler, as we only use the
|
||||
// NowMicros which is the same for all clocks. The Env could only be
|
||||
// overridden in test.
|
||||
static PeriodicWorkScheduler scheduler(SystemClock::Default()); |
||||
return &scheduler; |
||||
} |
||||
|
||||
std::string PeriodicWorkScheduler::GetTaskName( |
||||
const DBImpl* dbi, const std::string& func_name) const { |
||||
std::string db_session_id; |
||||
// TODO: Should this error be ignored?
|
||||
dbi->GetDbSessionId(db_session_id).PermitUncheckedError(); |
||||
return db_session_id + ":" + func_name; |
||||
} |
||||
|
||||
#ifndef NDEBUG |
||||
|
||||
// Get the static scheduler. For a new SystemClock, it needs to re-create the
|
||||
// internal timer, so only re-create it when there's no running task. Otherwise,
|
||||
// return the existing scheduler. Which means if the unittest needs to update
|
||||
// MockClock, Close all db instances and then re-open them.
|
||||
PeriodicWorkTestScheduler* PeriodicWorkTestScheduler::Default( |
||||
const std::shared_ptr<SystemClock>& clock) { |
||||
static PeriodicWorkTestScheduler scheduler(clock); |
||||
static port::Mutex mutex; |
||||
{ |
||||
MutexLock l(&mutex); |
||||
if (scheduler.timer.get() != nullptr && |
||||
scheduler.timer->TEST_GetPendingTaskNum() == 0) { |
||||
{ |
||||
MutexLock timer_mu_guard(&scheduler.timer_mu_); |
||||
scheduler.timer->Shutdown(); |
||||
} |
||||
scheduler.timer.reset(new Timer(clock.get())); |
||||
} |
||||
} |
||||
return &scheduler; |
||||
} |
||||
|
||||
void PeriodicWorkTestScheduler::TEST_WaitForRun( |
||||
std::function<void()> callback) const { |
||||
if (timer != nullptr) { |
||||
timer->TEST_WaitForRun(callback); |
||||
} |
||||
} |
||||
|
||||
size_t PeriodicWorkTestScheduler::TEST_GetValidTaskNum() const { |
||||
if (timer != nullptr) { |
||||
return timer->TEST_GetPendingTaskNum(); |
||||
} |
||||
return 0; |
||||
} |
||||
|
||||
bool PeriodicWorkTestScheduler::TEST_HasValidTask( |
||||
const DBImpl* dbi, const std::string& func_name) const { |
||||
if (timer == nullptr) { |
||||
return false; |
||||
} |
||||
return timer->TEST_HasVaildTask(GetTaskName(dbi, func_name)); |
||||
} |
||||
|
||||
PeriodicWorkTestScheduler::PeriodicWorkTestScheduler( |
||||
const std::shared_ptr<SystemClock>& clock) |
||||
: PeriodicWorkScheduler(clock) {} |
||||
|
||||
#endif // !NDEBUG
|
||||
} // namespace ROCKSDB_NAMESPACE
|
||||
|
||||
#endif // ROCKSDB_LITE
|
@ -1,90 +0,0 @@ |
||||
// Copyright (c) 2011-present, Facebook, Inc. All rights reserved.
|
||||
// This source code is licensed under both the GPLv2 (found in the
|
||||
// COPYING file in the root directory) and Apache 2.0 License
|
||||
// (found in the LICENSE.Apache file in the root directory).
|
||||
|
||||
#pragma once |
||||
|
||||
#ifndef ROCKSDB_LITE |
||||
|
||||
#include "db/db_impl/db_impl.h" |
||||
#include "util/timer.h" |
||||
|
||||
namespace ROCKSDB_NAMESPACE { |
||||
class SystemClock; |
||||
|
||||
// PeriodicWorkScheduler is a singleton object, which is scheduling/running
|
||||
// DumpStats(), PersistStats(), and FlushInfoLog() for all DB instances. All DB
|
||||
// instances use the same object from `Default()`.
|
||||
//
|
||||
// Internally, it uses a single threaded timer object to run the periodic work
|
||||
// functions. Timer thread will always be started since the info log flushing
|
||||
// cannot be disabled.
|
||||
class PeriodicWorkScheduler { |
||||
public: |
||||
static PeriodicWorkScheduler* Default(); |
||||
|
||||
PeriodicWorkScheduler() = delete; |
||||
PeriodicWorkScheduler(const PeriodicWorkScheduler&) = delete; |
||||
PeriodicWorkScheduler(PeriodicWorkScheduler&&) = delete; |
||||
PeriodicWorkScheduler& operator=(const PeriodicWorkScheduler&) = delete; |
||||
PeriodicWorkScheduler& operator=(PeriodicWorkScheduler&&) = delete; |
||||
|
||||
Status Register(DBImpl* dbi, unsigned int stats_dump_period_sec, |
||||
unsigned int stats_persist_period_sec); |
||||
Status RegisterRecordSeqnoTimeWorker(DBImpl* dbi, uint64_t record_cadence); |
||||
|
||||
void Unregister(DBImpl* dbi); |
||||
void UnregisterRecordSeqnoTimeWorker(DBImpl* dbi); |
||||
|
||||
// Periodically flush info log out of application buffer at a low frequency.
|
||||
// This improves debuggability in case of RocksDB hanging since it ensures the
|
||||
// log messages leading up to the hang will eventually become visible in the
|
||||
// log.
|
||||
static const uint64_t kDefaultFlushInfoLogPeriodSec = 10; |
||||
|
||||
protected: |
||||
std::unique_ptr<Timer> timer; |
||||
// `timer_mu_` serves two purposes currently:
|
||||
// (1) to ensure calls to `Start()` and `Shutdown()` are serialized, as
|
||||
// they are currently not implemented in a thread-safe way; and
|
||||
// (2) to ensure the `Timer::Add()`s and `Timer::Start()` run atomically, and
|
||||
// the `Timer::Cancel()`s and `Timer::Shutdown()` run atomically.
|
||||
port::Mutex timer_mu_; |
||||
|
||||
explicit PeriodicWorkScheduler(const std::shared_ptr<SystemClock>& clock); |
||||
|
||||
// Get the unique task name (prefix with db session id)
|
||||
std::string GetTaskName(const DBImpl* dbi, |
||||
const std::string& func_name) const; |
||||
}; |
||||
|
||||
#ifndef NDEBUG |
||||
// PeriodicWorkTestScheduler is for unittest, which can specify the SystemClock
|
||||
// It also contains functions for unittest.
|
||||
class PeriodicWorkTestScheduler : public PeriodicWorkScheduler { |
||||
public: |
||||
static PeriodicWorkTestScheduler* Default( |
||||
const std::shared_ptr<SystemClock>& clock); |
||||
|
||||
void TEST_WaitForRun(std::function<void()> callback) const; |
||||
|
||||
size_t TEST_GetValidTaskNum() const; |
||||
|
||||
bool TEST_HasValidTask(const DBImpl* dbi, const std::string& func_name) const; |
||||
|
||||
private: |
||||
explicit PeriodicWorkTestScheduler(const std::shared_ptr<SystemClock>& clock); |
||||
}; |
||||
#endif // !NDEBUG
|
||||
|
||||
struct PeriodicWorkTaskNames { |
||||
static const std::string kDumpStats; |
||||
static const std::string kPersistStats; |
||||
static const std::string kFlushInfoLog; |
||||
static const std::string kRecordSeqnoTime; |
||||
}; |
||||
|
||||
} // namespace ROCKSDB_NAMESPACE
|
||||
|
||||
#endif // ROCKSDB_LITE
|
Loading…
Reference in new issue