Summary: This diff adds three sets of APIs to RocksDB. = GetColumnFamilyMetaData = * This APIs allow users to obtain the current state of a RocksDB instance on one column family. * See GetColumnFamilyMetaData in include/rocksdb/db.h = EventListener = * A virtual class that allows users to implement a set of call-back functions which will be called when specific events of a RocksDB instance happens. * To register EventListener, simply insert an EventListener to ColumnFamilyOptions::listeners = CompactFiles = * CompactFiles API inputs a set of file numbers and an output level, and RocksDB will try to compact those files into the specified level. = Example = * Example code can be found in example/compact_files_example.cc, which implements a simple external compactor using EventListener, GetColumnFamilyMetaData, and CompactFiles API. Test Plan: listener_test compactor_test example/compact_files_example export ROCKSDB_TESTS=CompactFiles db_test export ROCKSDB_TESTS=MetaData db_test Reviewers: ljin, igor, rven, sdong Reviewed By: sdong Subscribers: MarkCallaghan, dhruba, leveldb Differential Revision: https://reviews.facebook.net/D24705main
parent
5c93090530
commit
28c82ff1b3
@ -0,0 +1,344 @@ |
||||
// Copyright (c) 2013, Facebook, Inc. All rights reserved.
|
||||
// This source code is licensed under the BSD-style license found in the
|
||||
// LICENSE file in the root directory of this source tree. An additional grant
|
||||
// of patent rights can be found in the PATENTS file in the same directory.
|
||||
|
||||
#include "db/dbformat.h" |
||||
#include "db/db_impl.h" |
||||
#include "db/filename.h" |
||||
#include "db/version_set.h" |
||||
#include "db/write_batch_internal.h" |
||||
#include "rocksdb/cache.h" |
||||
#include "rocksdb/compaction_filter.h" |
||||
#include "rocksdb/db.h" |
||||
#include "rocksdb/env.h" |
||||
#include "rocksdb/filter_policy.h" |
||||
#include "rocksdb/perf_context.h" |
||||
#include "rocksdb/slice.h" |
||||
#include "rocksdb/slice_transform.h" |
||||
#include "rocksdb/table.h" |
||||
#include "rocksdb/options.h" |
||||
#include "rocksdb/table_properties.h" |
||||
#include "table/block_based_table_factory.h" |
||||
#include "table/plain_table_factory.h" |
||||
#include "util/hash.h" |
||||
#include "util/hash_linklist_rep.h" |
||||
#include "utilities/merge_operators.h" |
||||
#include "util/logging.h" |
||||
#include "util/mutexlock.h" |
||||
#include "util/rate_limiter.h" |
||||
#include "util/statistics.h" |
||||
#include "util/testharness.h" |
||||
#include "util/sync_point.h" |
||||
#include "util/testutil.h" |
||||
|
||||
#ifndef ROCKSDB_LITE |
||||
|
||||
namespace rocksdb { |
||||
|
||||
class EventListenerTest { |
||||
public: |
||||
EventListenerTest() { |
||||
dbname_ = test::TmpDir() + "/listener_test"; |
||||
ASSERT_OK(DestroyDB(dbname_, Options())); |
||||
db_ = nullptr; |
||||
Reopen(); |
||||
} |
||||
|
||||
~EventListenerTest() { |
||||
Close(); |
||||
Options options; |
||||
options.db_paths.emplace_back(dbname_, 0); |
||||
options.db_paths.emplace_back(dbname_ + "_2", 0); |
||||
options.db_paths.emplace_back(dbname_ + "_3", 0); |
||||
options.db_paths.emplace_back(dbname_ + "_4", 0); |
||||
ASSERT_OK(DestroyDB(dbname_, options)); |
||||
} |
||||
|
||||
void CreateColumnFamilies(const std::vector<std::string>& cfs, |
||||
const ColumnFamilyOptions* options = nullptr) { |
||||
ColumnFamilyOptions cf_opts; |
||||
cf_opts = ColumnFamilyOptions(Options()); |
||||
int cfi = handles_.size(); |
||||
handles_.resize(cfi + cfs.size()); |
||||
for (auto cf : cfs) { |
||||
ASSERT_OK(db_->CreateColumnFamily(cf_opts, cf, &handles_[cfi++])); |
||||
} |
||||
} |
||||
|
||||
void Close() { |
||||
for (auto h : handles_) { |
||||
delete h; |
||||
} |
||||
handles_.clear(); |
||||
delete db_; |
||||
db_ = nullptr; |
||||
} |
||||
|
||||
void ReopenWithColumnFamilies(const std::vector<std::string>& cfs, |
||||
const Options* options = nullptr) { |
||||
ASSERT_OK(TryReopenWithColumnFamilies(cfs, options)); |
||||
} |
||||
|
||||
Status TryReopenWithColumnFamilies(const std::vector<std::string>& cfs, |
||||
const Options* options = nullptr) { |
||||
Close(); |
||||
Options opts = (options == nullptr) ? Options() : *options; |
||||
std::vector<const Options*> v_opts(cfs.size(), &opts); |
||||
return TryReopenWithColumnFamilies(cfs, v_opts); |
||||
} |
||||
|
||||
Status TryReopenWithColumnFamilies( |
||||
const std::vector<std::string>& cfs, |
||||
const std::vector<const Options*>& options) { |
||||
Close(); |
||||
ASSERT_EQ(cfs.size(), options.size()); |
||||
std::vector<ColumnFamilyDescriptor> column_families; |
||||
for (size_t i = 0; i < cfs.size(); ++i) { |
||||
column_families.push_back(ColumnFamilyDescriptor(cfs[i], *options[i])); |
||||
} |
||||
DBOptions db_opts = DBOptions(*options[0]); |
||||
return DB::Open(db_opts, dbname_, column_families, &handles_, &db_); |
||||
} |
||||
|
||||
Status TryReopen(Options* options = nullptr) { |
||||
Close(); |
||||
Options opts; |
||||
if (options != nullptr) { |
||||
opts = *options; |
||||
} else { |
||||
opts.create_if_missing = true; |
||||
} |
||||
|
||||
return DB::Open(opts, dbname_, &db_); |
||||
} |
||||
|
||||
void Reopen(Options* options = nullptr) { |
||||
ASSERT_OK(TryReopen(options)); |
||||
} |
||||
|
||||
void CreateAndReopenWithCF(const std::vector<std::string>& cfs, |
||||
const Options* options = nullptr) { |
||||
CreateColumnFamilies(cfs, options); |
||||
std::vector<std::string> cfs_plus_default = cfs; |
||||
cfs_plus_default.insert(cfs_plus_default.begin(), kDefaultColumnFamilyName); |
||||
ReopenWithColumnFamilies(cfs_plus_default, options); |
||||
} |
||||
|
||||
DBImpl* dbfull() { |
||||
return reinterpret_cast<DBImpl*>(db_); |
||||
} |
||||
|
||||
Status Put(int cf, const Slice& k, const Slice& v, |
||||
WriteOptions wo = WriteOptions()) { |
||||
return db_->Put(wo, handles_[cf], k, v); |
||||
} |
||||
|
||||
Status Flush(int cf = 0) { |
||||
if (cf == 0) { |
||||
return db_->Flush(FlushOptions()); |
||||
} else { |
||||
return db_->Flush(FlushOptions(), handles_[cf]); |
||||
} |
||||
} |
||||
|
||||
|
||||
DB* db_; |
||||
std::string dbname_; |
||||
std::vector<ColumnFamilyHandle*> handles_; |
||||
}; |
||||
|
||||
class TestFlushListener : public EventListener { |
||||
public: |
||||
void OnFlushCompleted( |
||||
DB* db, const std::string& name, |
||||
const std::string& file_path, |
||||
bool triggered_writes_slowdown, |
||||
bool triggered_writes_stop) override { |
||||
flushed_dbs_.push_back(db); |
||||
flushed_column_family_names_.push_back(name); |
||||
if (triggered_writes_slowdown) { |
||||
slowdown_count++; |
||||
} |
||||
if (triggered_writes_stop) { |
||||
stop_count++; |
||||
} |
||||
} |
||||
|
||||
std::vector<std::string> flushed_column_family_names_; |
||||
std::vector<DB*> flushed_dbs_; |
||||
int slowdown_count; |
||||
int stop_count; |
||||
}; |
||||
|
||||
TEST(EventListenerTest, OnSingleDBFlushTest) { |
||||
Options options; |
||||
TestFlushListener* listener = new TestFlushListener(); |
||||
options.listeners.emplace_back(listener); |
||||
std::vector<std::string> cf_names = { |
||||
"pikachu", "ilya", "muromec", "dobrynia", |
||||
"nikitich", "alyosha", "popovich"}; |
||||
CreateAndReopenWithCF(cf_names, &options); |
||||
|
||||
ASSERT_OK(Put(1, "pikachu", "pikachu")); |
||||
ASSERT_OK(Put(2, "ilya", "ilya")); |
||||
ASSERT_OK(Put(3, "muromec", "muromec")); |
||||
ASSERT_OK(Put(4, "dobrynia", "dobrynia")); |
||||
ASSERT_OK(Put(5, "nikitich", "nikitich")); |
||||
ASSERT_OK(Put(6, "alyosha", "alyosha")); |
||||
ASSERT_OK(Put(7, "popovich", "popovich")); |
||||
for (size_t i = 1; i < 8; ++i) { |
||||
Flush(i); |
||||
dbfull()->TEST_WaitForFlushMemTable(); |
||||
ASSERT_EQ(listener->flushed_dbs_.size(), i); |
||||
ASSERT_EQ(listener->flushed_column_family_names_.size(), i); |
||||
} |
||||
|
||||
// make sure call-back functions are called in the right order
|
||||
for (size_t i = 0; i < cf_names.size(); ++i) { |
||||
ASSERT_EQ(listener->flushed_dbs_[i], db_); |
||||
ASSERT_EQ(listener->flushed_column_family_names_[i], cf_names[i]); |
||||
} |
||||
} |
||||
|
||||
TEST(EventListenerTest, MultiCF) { |
||||
Options options; |
||||
TestFlushListener* listener = new TestFlushListener(); |
||||
options.listeners.emplace_back(listener); |
||||
std::vector<std::string> cf_names = { |
||||
"pikachu", "ilya", "muromec", "dobrynia", |
||||
"nikitich", "alyosha", "popovich"}; |
||||
CreateAndReopenWithCF(cf_names, &options); |
||||
|
||||
ASSERT_OK(Put(1, "pikachu", "pikachu")); |
||||
ASSERT_OK(Put(2, "ilya", "ilya")); |
||||
ASSERT_OK(Put(3, "muromec", "muromec")); |
||||
ASSERT_OK(Put(4, "dobrynia", "dobrynia")); |
||||
ASSERT_OK(Put(5, "nikitich", "nikitich")); |
||||
ASSERT_OK(Put(6, "alyosha", "alyosha")); |
||||
ASSERT_OK(Put(7, "popovich", "popovich")); |
||||
for (size_t i = 1; i < 8; ++i) { |
||||
Flush(i); |
||||
ASSERT_EQ(listener->flushed_dbs_.size(), i); |
||||
ASSERT_EQ(listener->flushed_column_family_names_.size(), i); |
||||
} |
||||
|
||||
// make sure call-back functions are called in the right order
|
||||
for (size_t i = 0; i < cf_names.size(); i++) { |
||||
ASSERT_EQ(listener->flushed_dbs_[i], db_); |
||||
ASSERT_EQ(listener->flushed_column_family_names_[i], cf_names[i]); |
||||
} |
||||
} |
||||
|
||||
TEST(EventListenerTest, MultiDBMultiListeners) { |
||||
std::vector<TestFlushListener*> listeners; |
||||
const int kNumDBs = 5; |
||||
const int kNumListeners = 10; |
||||
for (int i = 0; i < kNumListeners; ++i) { |
||||
listeners.emplace_back(new TestFlushListener()); |
||||
} |
||||
|
||||
std::vector<std::string> cf_names = { |
||||
"pikachu", "ilya", "muromec", "dobrynia", |
||||
"nikitich", "alyosha", "popovich"}; |
||||
|
||||
Options options; |
||||
options.create_if_missing = true; |
||||
for (int i = 0; i < kNumListeners; ++i) { |
||||
options.listeners.emplace_back(listeners[i]); |
||||
} |
||||
DBOptions db_opts(options); |
||||
ColumnFamilyOptions cf_opts(options); |
||||
|
||||
std::vector<DB*> dbs; |
||||
std::vector<std::vector<ColumnFamilyHandle *>> vec_handles; |
||||
|
||||
for (int d = 0; d < kNumDBs; ++d) { |
||||
ASSERT_OK(DestroyDB(dbname_ + std::to_string(d), options)); |
||||
DB* db; |
||||
std::vector<ColumnFamilyHandle*> handles; |
||||
ASSERT_OK(DB::Open(options, dbname_ + std::to_string(d), &db)); |
||||
for (size_t c = 0; c < cf_names.size(); ++c) { |
||||
ColumnFamilyHandle* handle; |
||||
db->CreateColumnFamily(cf_opts, cf_names[c], &handle); |
||||
handles.push_back(handle); |
||||
} |
||||
|
||||
vec_handles.push_back(std::move(handles)); |
||||
dbs.push_back(db); |
||||
} |
||||
|
||||
for (int d = 0; d < kNumDBs; ++d) { |
||||
for (size_t c = 0; c < cf_names.size(); ++c) { |
||||
ASSERT_OK(dbs[d]->Put(WriteOptions(), vec_handles[d][c], |
||||
cf_names[c], cf_names[c])); |
||||
} |
||||
} |
||||
|
||||
for (size_t c = 0; c < cf_names.size(); ++c) { |
||||
for (int d = 0; d < kNumDBs; ++d) { |
||||
ASSERT_OK(dbs[d]->Flush(FlushOptions(), vec_handles[d][c])); |
||||
reinterpret_cast<DBImpl*>(dbs[d])->TEST_WaitForFlushMemTable(); |
||||
} |
||||
} |
||||
|
||||
for (auto* listener : listeners) { |
||||
int pos = 0; |
||||
for (size_t c = 0; c < cf_names.size(); ++c) { |
||||
for (int d = 0; d < kNumDBs; ++d) { |
||||
ASSERT_EQ(listener->flushed_dbs_[pos], dbs[d]); |
||||
ASSERT_EQ(listener->flushed_column_family_names_[pos], cf_names[c]); |
||||
pos++; |
||||
} |
||||
} |
||||
} |
||||
|
||||
for (auto handles : vec_handles) { |
||||
for (auto h : handles) { |
||||
delete h; |
||||
} |
||||
handles.clear(); |
||||
} |
||||
vec_handles.clear(); |
||||
|
||||
for (auto db : dbs) { |
||||
delete db; |
||||
} |
||||
} |
||||
|
||||
TEST(EventListenerTest, DisableBGCompaction) { |
||||
Options options; |
||||
TestFlushListener* listener = new TestFlushListener(); |
||||
const int kSlowdownTrigger = 5; |
||||
const int kStopTrigger = 10; |
||||
options.level0_slowdown_writes_trigger = kSlowdownTrigger; |
||||
options.level0_stop_writes_trigger = kStopTrigger; |
||||
options.listeners.emplace_back(listener); |
||||
// BG compaction is disabled. Number of L0 files will simply keeps
|
||||
// increasing in this test.
|
||||
options.compaction_style = kCompactionStyleNone; |
||||
options.compression = kNoCompression; |
||||
options.write_buffer_size = 100000; // Small write buffer
|
||||
|
||||
CreateAndReopenWithCF({"pikachu"}, &options); |
||||
WriteOptions wopts; |
||||
wopts.timeout_hint_us = 100000; |
||||
ColumnFamilyMetaData cf_meta; |
||||
db_->GetColumnFamilyMetaData(handles_[1], &cf_meta); |
||||
// keep writing until writes are forced to stop.
|
||||
for (int i = 0; static_cast<int>(cf_meta.file_count) < kStopTrigger; ++i) { |
||||
Put(1, std::to_string(i), std::string(100000, 'x'), wopts); |
||||
db_->GetColumnFamilyMetaData(handles_[1], &cf_meta); |
||||
} |
||||
ASSERT_GE(listener->slowdown_count, kStopTrigger - kSlowdownTrigger); |
||||
ASSERT_GE(listener->stop_count, 1); |
||||
} |
||||
|
||||
} // namespace rocksdb
|
||||
|
||||
#endif // ROCKSDB_LITE
|
||||
|
||||
int main(int argc, char** argv) { |
||||
return rocksdb::test::RunAllTests(); |
||||
} |
||||
|
@ -0,0 +1,175 @@ |
||||
// Copyright (c) 2014, Facebook, Inc. All rights reserved.
|
||||
// This source code is licensed under the BSD-style license found in the
|
||||
// LICENSE file in the root directory of this source tree. An additional grant
|
||||
// of patent rights can be found in the PATENTS file in the same directory.
|
||||
//
|
||||
// An example code demonstrating how to use CompactFiles, EventListener,
|
||||
// and GetColumnFamilyMetaData APIs to implement custom compaction algorithm.
|
||||
|
||||
#include <mutex> |
||||
#include <string> |
||||
#include "rocksdb/db.h" |
||||
#include "rocksdb/env.h" |
||||
#include "rocksdb/options.h" |
||||
|
||||
using namespace rocksdb; |
||||
std::string kDBPath = "/tmp/rocksdb_compact_files_example"; |
||||
class CompactionTask; |
||||
|
||||
// This is an example interface of external-compaction algorithm.
|
||||
// Compaction algorithm can be implemented outside the core-RocksDB
|
||||
// code by using the pluggable compaction APIs that RocksDb provides.
|
||||
class Compactor : public EventListener { |
||||
public: |
||||
// Picks and returns a compaction task given the specified DB
|
||||
// and column family. It is the caller's responsibility to
|
||||
// destroy the returned CompactionTask. Returns "nullptr"
|
||||
// if it cannot find a proper compaction task.
|
||||
virtual CompactionTask* PickCompaction( |
||||
DB* db, const std::string& cf_name) = 0; |
||||
|
||||
// Schedule and run the specified compaction task in background.
|
||||
virtual void ScheduleCompaction(CompactionTask *task) = 0; |
||||
}; |
||||
|
||||
// Example structure that describes a compaction task.
|
||||
struct CompactionTask { |
||||
CompactionTask( |
||||
DB* db, Compactor* compactor, |
||||
const std::string& column_family_name, |
||||
const std::vector<std::string>& input_file_names, |
||||
const int output_level, |
||||
const CompactionOptions& compact_options, |
||||
bool retry_on_fail) |
||||
: db(db), |
||||
compactor(compactor), |
||||
column_family_name(column_family_name), |
||||
input_file_names(input_file_names), |
||||
output_level(output_level), |
||||
compact_options(compact_options), |
||||
retry_on_fail(false) {} |
||||
DB* db; |
||||
Compactor* compactor; |
||||
const std::string& column_family_name; |
||||
std::vector<std::string> input_file_names; |
||||
int output_level; |
||||
CompactionOptions compact_options; |
||||
bool retry_on_fail; |
||||
}; |
||||
|
||||
// A simple compaction algorithm that always compacts everything
|
||||
// to the highest level whenever possible.
|
||||
class FullCompactor : public Compactor { |
||||
public: |
||||
explicit FullCompactor(const Options options) : options_(options) { |
||||
compact_options_.compression = options_.compression; |
||||
compact_options_.output_file_size_limit = |
||||
options_.target_file_size_base; |
||||
} |
||||
|
||||
// When flush happens, it determins whether to trigger compaction.
|
||||
// If triggered_writes_stop is true, it will also set the retry
|
||||
// flag of compaction-task to true.
|
||||
void OnFlushCompleted( |
||||
DB* db, const std::string& cf_name, |
||||
const std::string& file_path, |
||||
bool triggered_writes_slowdown, |
||||
bool triggered_writes_stop) override { |
||||
CompactionTask* task = PickCompaction(db, cf_name); |
||||
if (task != nullptr) { |
||||
if (triggered_writes_stop) { |
||||
task->retry_on_fail = true; |
||||
} |
||||
// Schedule compaction in a different thread.
|
||||
ScheduleCompaction(task); |
||||
} |
||||
} |
||||
|
||||
// Always pick a compaction which includes all files whenever possible.
|
||||
CompactionTask* PickCompaction( |
||||
DB* db, const std::string& cf_name) override { |
||||
ColumnFamilyMetaData cf_meta; |
||||
db->GetColumnFamilyMetaData(&cf_meta); |
||||
|
||||
std::vector<std::string> input_file_names; |
||||
for (auto level : cf_meta.levels) { |
||||
for (auto file : level.files) { |
||||
if (file.being_compacted) { |
||||
return nullptr; |
||||
} |
||||
input_file_names.push_back(file.name); |
||||
} |
||||
} |
||||
return new CompactionTask( |
||||
db, this, cf_name, input_file_names, |
||||
options_.num_levels - 1, compact_options_, false); |
||||
} |
||||
|
||||
// Schedule the specified compaction task in background.
|
||||
void ScheduleCompaction(CompactionTask* task) override { |
||||
options_.env->Schedule(&FullCompactor::CompactFiles, task); |
||||
} |
||||
|
||||
static void CompactFiles(void* arg) { |
||||
CompactionTask* task = reinterpret_cast<CompactionTask*>(arg); |
||||
assert(task); |
||||
assert(task->db); |
||||
Status s = task->db->CompactFiles( |
||||
task->compact_options, |
||||
task->input_file_names, |
||||
task->output_level); |
||||
printf("CompactFiles() finished with status %s\n", s.ToString().c_str()); |
||||
if (!s.ok() && !s.IsIOError() && task->retry_on_fail) { |
||||
// If a compaction task with its retry_on_fail=true failed,
|
||||
// try to schedule another compaction in case the reason
|
||||
// is not an IO error.
|
||||
CompactionTask* new_task = task->compactor->PickCompaction( |
||||
task->db, task->column_family_name); |
||||
task->compactor->ScheduleCompaction(new_task); |
||||
} |
||||
// release the task
|
||||
delete task; |
||||
} |
||||
|
||||
private: |
||||
Options options_; |
||||
CompactionOptions compact_options_; |
||||
}; |
||||
|
||||
int main() { |
||||
Options options; |
||||
options.create_if_missing = true; |
||||
// Disable RocksDB background compaction.
|
||||
options.compaction_style = kCompactionStyleNone; |
||||
// Small slowdown and stop trigger for experimental purpose.
|
||||
options.level0_slowdown_writes_trigger = 3; |
||||
options.level0_stop_writes_trigger = 5; |
||||
options.IncreaseParallelism(5); |
||||
options.listeners.emplace_back(new FullCompactor(options)); |
||||
|
||||
DB* db = nullptr; |
||||
DestroyDB(kDBPath, options); |
||||
Status s = DB::Open(options, kDBPath, &db); |
||||
assert(s.ok()); |
||||
assert(db); |
||||
|
||||
// if background compaction is not working, write will stall
|
||||
// because of options.level0_stop_writes_trigger
|
||||
for (int i = 1000; i < 99999; ++i) { |
||||
db->Put(WriteOptions(), std::to_string(i), |
||||
std::string(500, 'a' + (i % 26))); |
||||
} |
||||
|
||||
// verify the values are still there
|
||||
std::string value; |
||||
for (int i = 1000; i < 99999; ++i) { |
||||
db->Get(ReadOptions(), std::to_string(i), |
||||
&value); |
||||
assert(value == std::string(500, 'a' + (i % 26))); |
||||
} |
||||
|
||||
// close the db.
|
||||
delete db; |
||||
|
||||
return 0; |
||||
} |
@ -0,0 +1,65 @@ |
||||
// Copyright (c) 2014 The LevelDB Authors. All rights reserved.
|
||||
// Use of this source code is governed by a BSD-style license that can be
|
||||
// found in the LICENSE file. See the AUTHORS file for names of contributors.
|
||||
|
||||
#pragma once |
||||
|
||||
#ifndef ROCKSDB_LITE |
||||
|
||||
#include <string> |
||||
#include "rocksdb/status.h" |
||||
|
||||
namespace rocksdb { |
||||
|
||||
class DB; |
||||
class Status; |
||||
|
||||
// EventListener class contains a set of call-back functions that will
|
||||
// be called when specific RocksDB event happens such as flush. It can
|
||||
// be used as a building block for developing custom features such as
|
||||
// stats-collector or external compaction algorithm.
|
||||
//
|
||||
// Note that call-back functions should not run for an extended period of
|
||||
// time before the function returns, otherwise RocksDB may be blocked.
|
||||
// For example, it is not suggested to do DB::CompactFiles() (as it may
|
||||
// run for a long while) or issue many of DB::Put() (as Put may be blocked
|
||||
// in certain cases) in the same thread in the EventListener callback.
|
||||
// However, doing DB::CompactFiles() and DB::Put() in another thread is
|
||||
// considered safe.
|
||||
//
|
||||
// [Threading] All EventListener callback will be called using the
|
||||
// actual thread that involves in that specific event. For example, it
|
||||
// is the RocksDB background flush thread that does the actual flush to
|
||||
// call EventListener::OnFlushCompleted().
|
||||
class EventListener { |
||||
public: |
||||
// A call-back function to RocksDB which will be called whenever a
|
||||
// registered RocksDB flushes a file. The default implementation is
|
||||
// no-op.
|
||||
//
|
||||
// Note that the this function must be implemented in a way such that
|
||||
// it should not run for an extended period of time before the function
|
||||
// returns. Otherwise, RocksDB may be blocked.
|
||||
//
|
||||
// @param db a pointer to the rocksdb instance which just flushed
|
||||
// a memtable to disk.
|
||||
// @param column_family_id the id of the flushed column family.
|
||||
// @param file_path the path to the newly created file.
|
||||
// @param triggered_writes_slowdown true when rocksdb is currently
|
||||
// slowing-down all writes to prevent creating too many Level 0
|
||||
// files as compaction seems not able to catch up the write request
|
||||
// speed. This indicates that there're too many files in Level 0.
|
||||
// @param triggered_writes_stop true when rocksdb is currently blocking
|
||||
// any writes to prevent creating more L0 files. This indicates that
|
||||
// there're too many files in level 0. Compactions should try to
|
||||
// compact L0 files down to lower levels as soon as possible.
|
||||
virtual void OnFlushCompleted( |
||||
DB* db, const std::string& column_family_name, |
||||
const std::string& file_path, |
||||
bool triggered_writes_slowdown, |
||||
bool triggered_writes_stop) {} |
||||
}; |
||||
|
||||
} // namespace rocksdb
|
||||
|
||||
#endif // ROCKSDB_LITE
|
@ -0,0 +1,90 @@ |
||||
// Copyright (c) 2014, Facebook, Inc. All rights reserved.
|
||||
// This source code is licensed under the BSD-style license found in the
|
||||
// LICENSE file in the root directory of this source tree. An additional grant
|
||||
// of patent rights can be found in the PATENTS file in the same directory.
|
||||
|
||||
#include <limits> |
||||
#include <string> |
||||
#include <vector> |
||||
|
||||
#include "rocksdb/types.h" |
||||
|
||||
#pragma once |
||||
|
||||
namespace rocksdb { |
||||
struct ColumnFamilyMetaData; |
||||
struct LevelMetaData; |
||||
struct SstFileMetaData; |
||||
|
||||
// The metadata that describes a column family.
|
||||
struct ColumnFamilyMetaData { |
||||
ColumnFamilyMetaData() : size(0), name("") {} |
||||
ColumnFamilyMetaData(const std::string& name, uint64_t size, |
||||
const std::vector<LevelMetaData>&& levels) : |
||||
size(size), name(name), levels(levels) {} |
||||
|
||||
// The size of this column family in bytes, which is equal to the sum of
|
||||
// the file size of its "levels".
|
||||
uint64_t size; |
||||
// The number of files in this column family.
|
||||
size_t file_count; |
||||
// The name of the column family.
|
||||
std::string name; |
||||
// The metadata of all levels in this column family.
|
||||
std::vector<LevelMetaData> levels; |
||||
}; |
||||
|
||||
// The metadata that describes a level.
|
||||
struct LevelMetaData { |
||||
LevelMetaData(int level, uint64_t size, |
||||
const std::vector<SstFileMetaData>&& files) : |
||||
level(level), size(size), |
||||
files(files) {} |
||||
|
||||
// The level which this meta data describes.
|
||||
const int level; |
||||
// The size of this level in bytes, which is equal to the sum of
|
||||
// the file size of its "files".
|
||||
const uint64_t size; |
||||
// The metadata of all sst files in this level.
|
||||
const std::vector<SstFileMetaData> files; |
||||
}; |
||||
|
||||
// The metadata that describes a SST file.
|
||||
struct SstFileMetaData { |
||||
SstFileMetaData() {} |
||||
SstFileMetaData(const std::string& file_name, |
||||
const std::string& path, uint64_t size, |
||||
SequenceNumber smallest_seqno, |
||||
SequenceNumber largest_seqno, |
||||
const std::string& smallestkey, |
||||
const std::string& largestkey, |
||||
bool being_compacted) : |
||||
size(size), name(file_name), |
||||
db_path(path), smallest_seqno(smallest_seqno), largest_seqno(largest_seqno), |
||||
smallestkey(smallestkey), largestkey(largestkey), |
||||
being_compacted(being_compacted) {} |
||||
|
||||
// File size in bytes.
|
||||
uint64_t size; |
||||
// The name of the file.
|
||||
std::string name; |
||||
// The full path where the file locates.
|
||||
std::string db_path; |
||||
|
||||
SequenceNumber smallest_seqno; // Smallest sequence number in file.
|
||||
SequenceNumber largest_seqno; // Largest sequence number in file.
|
||||
std::string smallestkey; // Smallest user defined key in the file.
|
||||
std::string largestkey; // Largest user defined key in the file.
|
||||
bool being_compacted; // true if the file is currently being compacted.
|
||||
}; |
||||
|
||||
// The full set of metadata associated with each SST file.
|
||||
struct LiveFileMetaData : SstFileMetaData { |
||||
std::string column_family_name; // Name of the column family
|
||||
int level; // Level at which this file resides.
|
||||
}; |
||||
|
||||
|
||||
|
||||
} // namespace rocksdb
|
Loading…
Reference in new issue