|
|
|
// 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 <algorithm>
|
|
|
|
#include <cstdint>
|
|
|
|
#include <memory>
|
|
|
|
#include <string>
|
|
|
|
#include <vector>
|
|
|
|
|
|
|
|
#include "db/db_impl/db_impl.h"
|
|
|
|
#include "db/job_context.h"
|
|
|
|
#include "db/version_set.h"
|
|
|
|
#include "file/file_util.h"
|
|
|
|
#include "file/filename.h"
|
|
|
|
#include "logging/logging.h"
|
|
|
|
#include "port/port.h"
|
|
|
|
#include "rocksdb/db.h"
|
|
|
|
#include "rocksdb/env.h"
|
|
|
|
#include "rocksdb/metadata.h"
|
|
|
|
#include "rocksdb/types.h"
|
|
|
|
#include "test_util/sync_point.h"
|
|
|
|
#include "util/file_checksum_helper.h"
|
|
|
|
#include "util/mutexlock.h"
|
|
|
|
|
|
|
|
namespace ROCKSDB_NAMESPACE {
|
|
|
|
|
|
|
|
Status DBImpl::FlushForGetLiveFiles() {
|
|
|
|
mutex_.AssertHeld();
|
|
|
|
|
|
|
|
// flush all dirty data to disk.
|
|
|
|
Status status;
|
|
|
|
if (immutable_db_options_.atomic_flush) {
|
|
|
|
mutex_.Unlock();
|
Fix bug of prematurely excluded CF in atomic flush contains unflushed data that should've been included in the atomic flush (#11148)
Summary:
**Context:**
Atomic flush should guarantee recoverability of all data of seqno up to the max seqno of the flush. It achieves this by ensuring all such data are flushed by the time this atomic flush finishes through `SelectColumnFamiliesForAtomicFlush()`. However, our crash test exposed the following case where an excluded CF from an atomic flush contains unflushed data of seqno less than the max seqno of that atomic flush and loses its data with `WriteOptions::DisableWAL=true` in face of a crash right after the atomic flush finishes .
```
./db_stress --preserve_unverified_changes=1 --reopen=0 --acquire_snapshot_one_in=0 --adaptive_readahead=1 --allow_data_in_errors=True --async_io=1 --atomic_flush=1 --avoid_flush_during_recovery=0 --avoid_unnecessary_blocking_io=0 --backup_max_size=104857600 --backup_one_in=0 --batch_protection_bytes_per_key=0 --block_size=16384 --bloom_bits=15 --bottommost_compression_type=none --bytes_per_sync=262144 --cache_index_and_filter_blocks=0 --cache_size=8388608 --cache_type=lru_cache --charge_compression_dictionary_building_buffer=0 --charge_file_metadata=1 --charge_filter_construction=0 --charge_table_reader=0 --checkpoint_one_in=0 --checksum_type=kXXH3 --clear_column_family_one_in=0 --compact_files_one_in=0 --compact_range_one_in=0 --compaction_pri=1 --compaction_ttl=100 --compression_max_dict_buffer_bytes=134217727 --compression_max_dict_bytes=16384 --compression_parallel_threads=1 --compression_type=lz4hc --compression_use_zstd_dict_trainer=0 --compression_zstd_max_train_bytes=0 --continuous_verification_interval=0 --data_block_index_type=0 --db=$db --db_write_buffer_size=1048576 --delpercent=4 --delrangepercent=1 --destroy_db_initially=0 --detect_filter_construct_corruption=0 --disable_wal=1 --enable_compaction_filter=0 --enable_pipelined_write=0 --expected_values_dir=$exp --fail_if_options_file_error=0 --fifo_allow_compaction=0 --file_checksum_impl=none --flush_one_in=0 --format_version=5 --get_current_wal_file_one_in=0 --get_live_files_one_in=100 --get_property_one_in=0 --get_sorted_wal_files_one_in=0 --index_block_restart_interval=2 --index_type=0 --ingest_external_file_one_in=0 --initial_auto_readahead_size=524288 --iterpercent=10 --key_len_percent_dist=1,30,69 --level_compaction_dynamic_level_bytes=True --long_running_snapshots=1 --manual_wal_flush_one_in=100 --mark_for_compaction_one_file_in=0 --max_auto_readahead_size=0 --max_background_compactions=20 --max_bytes_for_level_base=10485760 --max_key=10000 --max_key_len=3 --max_manifest_file_size=1073741824 --max_write_batch_group_size_bytes=64 --max_write_buffer_number=3 --max_write_buffer_size_to_maintain=0 --memtable_prefix_bloom_size_ratio=0.01 --memtable_protection_bytes_per_key=4 --memtable_whole_key_filtering=0 --memtablerep=skip_list --min_write_buffer_number_to_merge=2 --mmap_read=1 --mock_direct_io=False --nooverwritepercent=1 --num_file_reads_for_auto_readahead=0 --open_files=-1 --open_metadata_write_fault_one_in=0 --open_read_fault_one_in=0 --open_write_fault_one_in=0 --ops_per_thread=100000000 --optimize_filters_for_memory=1 --paranoid_file_checks=1 --partition_filters=0 --partition_pinning=3 --pause_background_one_in=0 --periodic_compaction_seconds=100 --prefix_size=8 --prefixpercent=5 --prepopulate_block_cache=0 --preserve_internal_time_seconds=3600 --progress_reports=0 --read_fault_one_in=32 --readahead_size=16384 --readpercent=50 --recycle_log_file_num=0 --ribbon_starting_level=6 --secondary_cache_fault_one_in=0 --set_options_one_in=10000 --snapshot_hold_ops=100000 --sst_file_manager_bytes_per_sec=104857600 --sst_file_manager_bytes_per_truncate=1048576 --stats_dump_period_sec=10 --subcompactions=1 --sync=0 --sync_fault_injection=0 --target_file_size_base=524288 --target_file_size_multiplier=2 --test_batches_snapshots=0 --top_level_index_pinning=0 --unpartitioned_pinning=1 --use_direct_io_for_flush_and_compaction=0 --use_direct_reads=0 --use_full_merge_v1=0 --use_merge=0 --use_multiget=1 --use_put_entity_one_in=0 --user_timestamp_size=0 --value_size_mult=32 --verify_checksum=1 --verify_checksum_one_in=0 --verify_db_one_in=1000 --verify_sst_unique_id_in_manifest=1 --wal_bytes_per_sync=524288 --wal_compression=none --write_buffer_size=524288 --write_dbid_to_manifest=1 --write_fault_one_in=0 --writepercent=30 &
pid=$!
sleep 0.2
sleep 10
kill $pid
sleep 0.2
./db_stress --ops_per_thread=1 --preserve_unverified_changes=1 --reopen=0 --acquire_snapshot_one_in=0 --adaptive_readahead=1 --allow_data_in_errors=True --async_io=1 --atomic_flush=1 --avoid_flush_during_recovery=0 --avoid_unnecessary_blocking_io=0 --backup_max_size=104857600 --backup_one_in=0 --batch_protection_bytes_per_key=0 --block_size=16384 --bloom_bits=15 --bottommost_compression_type=none --bytes_per_sync=262144 --cache_index_and_filter_blocks=0 --cache_size=8388608 --cache_type=lru_cache --charge_compression_dictionary_building_buffer=0 --charge_file_metadata=1 --charge_filter_construction=0 --charge_table_reader=0 --checkpoint_one_in=0 --checksum_type=kXXH3 --clear_column_family_one_in=0 --compact_files_one_in=0 --compact_range_one_in=0 --compaction_pri=1 --compaction_ttl=100 --compression_max_dict_buffer_bytes=134217727 --compression_max_dict_bytes=16384 --compression_parallel_threads=1 --compression_type=lz4hc --compression_use_zstd_dict_trainer=0 --compression_zstd_max_train_bytes=0 --continuous_verification_interval=0 --data_block_index_type=0 --db=$db --db_write_buffer_size=1048576 --delpercent=4 --delrangepercent=1 --destroy_db_initially=0 --detect_filter_construct_corruption=0 --disable_wal=1 --enable_compaction_filter=0 --enable_pipelined_write=0 --expected_values_dir=$exp --fail_if_options_file_error=0 --fifo_allow_compaction=0 --file_checksum_impl=none --flush_one_in=0 --format_version=5 --get_current_wal_file_one_in=0 --get_live_files_one_in=100 --get_property_one_in=0 --get_sorted_wal_files_one_in=0 --index_block_restart_interval=2 --index_type=0 --ingest_external_file_one_in=0 --initial_auto_readahead_size=524288 --iterpercent=10 --key_len_percent_dist=1,30,69 --level_compaction_dynamic_level_bytes=True --long_running_snapshots=1 --manual_wal_flush_one_in=100 --mark_for_compaction_one_file_in=0 --max_auto_readahead_size=0 --max_background_compactions=20 --max_bytes_for_level_base=10485760 --max_key=10000 --max_key_len=3 --max_manifest_file_size=1073741824 --max_write_batch_group_size_bytes=64 --max_write_buffer_number=3 --max_write_buffer_size_to_maintain=0 --memtable_prefix_bloom_size_ratio=0.01 --memtable_protection_bytes_per_key=4 --memtable_whole_key_filtering=0 --memtablerep=skip_list --min_write_buffer_number_to_merge=2 --mmap_read=1 --mock_direct_io=False --nooverwritepercent=1 --num_file_reads_for_auto_readahead=0 --open_files=-1 --open_metadata_write_fault_one_in=0 --open_read_fault_one_in=0 --open_write_fault_one_in=0 --ops_per_thread=100000000 --optimize_filters_for_memory=1 --paranoid_file_checks=1 --partition_filters=0 --partition_pinning=3 --pause_background_one_in=0 --periodic_compaction_seconds=100 --prefix_size=8 --prefixpercent=5 --prepopulate_block_cache=0 --preserve_internal_time_seconds=3600 --progress_reports=0 --read_fault_one_in=32 --readahead_size=16384 --readpercent=50 --recycle_log_file_num=0 --ribbon_starting_level=6 --secondary_cache_fault_one_in=0 --set_options_one_in=10000 --snapshot_hold_ops=100000 --sst_file_manager_bytes_per_sec=104857600 --sst_file_manager_bytes_per_truncate=1048576 --stats_dump_period_sec=10 --subcompactions=1 --sync=0 --sync_fault_injection=0 --target_file_size_base=524288 --target_file_size_multiplier=2 --test_batches_snapshots=0 --top_level_index_pinning=0 --unpartitioned_pinning=1 --use_direct_io_for_flush_and_compaction=0 --use_direct_reads=0 --use_full_merge_v1=0 --use_merge=0 --use_multiget=1 --use_put_entity_one_in=0 --user_timestamp_size=0 --value_size_mult=32 --verify_checksum=1 --verify_checksum_one_in=0 --verify_db_one_in=1000 --verify_sst_unique_id_in_manifest=1 --wal_bytes_per_sync=524288 --wal_compression=none --write_buffer_size=524288 --write_dbid_to_manifest=1 --write_fault_one_in=0 --writepercent=30 &
pid=$!
sleep 0.2
sleep 40
kill $pid
sleep 0.2
Verification failed for column family 6 key 0000000000000239000000000000012B0000000000000138 (56622): value_from_db: , value_from_expected: 4A6331754E4F4C4D42434041464744455A5B58595E5F5C5D5253505156575455, msg: Value not found: NotFound:
Crash-recovery verification failed :(
No writes or ops?
Verification failed :(
```
The bug is due to the following:
- When atomic flush is used, an empty CF is legally [excluded](https://github.com/facebook/rocksdb/blob/7.10.fb/db/db_filesnapshot.cc#L39) in `SelectColumnFamiliesForAtomicFlush` as the first step of `DBImpl::FlushForGetLiveFiles` before [passing](https://github.com/facebook/rocksdb/blob/7.10.fb/db/db_filesnapshot.cc#L42) the included CFDs to `AtomicFlushMemTables`.
- But [later](https://github.com/facebook/rocksdb/blob/7.10.fb/db/db_impl/db_impl_compaction_flush.cc#L2133) in `AtomicFlushMemTables`, `WaitUntilFlushWouldNotStallWrites` will [release the db mutex](https://github.com/facebook/rocksdb/blob/7.10.fb/db/db_impl/db_impl_compaction_flush.cc#L2403), during which data@seqno N can be inserted into the excluded CF and data@seqno M can be inserted into one of the included CFs, where M > N.
- However, data@seqno N in an already-excluded CF is thus excluded from this atomic flush while we seqno N is less than seqno M.
**Summary:**
- Replace `SelectColumnFamiliesForAtomicFlush()`-before-`AtomicFlushMemTables()` with `SelectColumnFamiliesForAtomicFlush()`-after-wait-within-`AtomicFlushMemTables()` so we ensure no write affecting the recoverability of this atomic job (i.e, change to max seqno of this atomic flush or insertion of data with less seqno than the max seqno of the atomic flush to excluded CF) can happen after calling `SelectColumnFamiliesForAtomicFlush()`.
- For above, refactored and clarified comments on `SelectColumnFamiliesForAtomicFlush()` and `AtomicFlushMemTables()` for clearer semantics of passed-in CFDs to atomic-flush
Pull Request resolved: https://github.com/facebook/rocksdb/pull/11148
Test Plan:
- New unit test failed before the fix and passes after
- Make check
- Rehearsal stress test
Reviewed By: ajkr
Differential Revision: D42799871
Pulled By: hx235
fbshipit-source-id: 13636b63e9c25c5895857afc36ea580d57f6d644
2 years ago
|
|
|
status = AtomicFlushMemTables(FlushOptions(), FlushReason::kGetLiveFiles);
|
|
|
|
if (status.IsColumnFamilyDropped()) {
|
|
|
|
status = Status::OK();
|
|
|
|
}
|
|
|
|
mutex_.Lock();
|
|
|
|
} else {
|
|
|
|
for (auto cfd : versions_->GetRefedColumnFamilySet()) {
|
|
|
|
if (cfd->IsDropped()) {
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
mutex_.Unlock();
|
|
|
|
status = FlushMemTable(cfd, FlushOptions(), FlushReason::kGetLiveFiles);
|
|
|
|
TEST_SYNC_POINT("DBImpl::GetLiveFiles:1");
|
|
|
|
TEST_SYNC_POINT("DBImpl::GetLiveFiles:2");
|
|
|
|
mutex_.Lock();
|
|
|
|
if (!status.ok() && !status.IsColumnFamilyDropped()) {
|
|
|
|
break;
|
|
|
|
} else if (status.IsColumnFamilyDropped()) {
|
|
|
|
status = Status::OK();
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return status;
|
|
|
|
}
|
|
|
|
|
|
|
|
Status DBImpl::GetLiveFiles(std::vector<std::string>& ret,
|
|
|
|
uint64_t* manifest_file_size, bool flush_memtable) {
|
|
|
|
*manifest_file_size = 0;
|
|
|
|
|
|
|
|
mutex_.Lock();
|
|
|
|
|
|
|
|
if (flush_memtable) {
|
|
|
|
Status status = FlushForGetLiveFiles();
|
|
|
|
if (!status.ok()) {
|
|
|
|
mutex_.Unlock();
|
|
|
|
ROCKS_LOG_ERROR(immutable_db_options_.info_log, "Cannot Flush data %s\n",
|
|
|
|
status.ToString().c_str());
|
|
|
|
return status;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// Make a set of all of the live table and blob files
|
|
|
|
std::vector<uint64_t> live_table_files;
|
|
|
|
std::vector<uint64_t> live_blob_files;
|
|
|
|
for (auto cfd : *versions_->GetColumnFamilySet()) {
|
|
|
|
if (cfd->IsDropped()) {
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
cfd->current()->AddLiveFiles(&live_table_files, &live_blob_files);
|
|
|
|
}
|
|
|
|
|
|
|
|
ret.clear();
|
|
|
|
ret.reserve(live_table_files.size() + live_blob_files.size() +
|
|
|
|
3); // for CURRENT + MANIFEST + OPTIONS
|
|
|
|
|
|
|
|
// create names of the live files. The names are not absolute
|
|
|
|
// paths, instead they are relative to dbname_.
|
|
|
|
for (const auto& table_file_number : live_table_files) {
|
|
|
|
ret.emplace_back(MakeTableFileName("", table_file_number));
|
|
|
|
}
|
|
|
|
|
|
|
|
for (const auto& blob_file_number : live_blob_files) {
|
|
|
|
ret.emplace_back(BlobFileName("", blob_file_number));
|
|
|
|
}
|
|
|
|
|
|
|
|
ret.emplace_back(CurrentFileName(""));
|
|
|
|
ret.emplace_back(DescriptorFileName("", versions_->manifest_file_number()));
|
|
|
|
// The OPTIONS file number is zero in read-write mode when OPTIONS file
|
|
|
|
// writing failed and the DB was configured with
|
|
|
|
// `fail_if_options_file_error == false`. In read-only mode the OPTIONS file
|
|
|
|
// number is zero when no OPTIONS file exist at all. In those cases we do not
|
|
|
|
// record any OPTIONS file in the live file list.
|
|
|
|
if (versions_->options_file_number() != 0) {
|
|
|
|
ret.emplace_back(OptionsFileName("", versions_->options_file_number()));
|
|
|
|
}
|
|
|
|
|
|
|
|
// find length of manifest file while holding the mutex lock
|
|
|
|
*manifest_file_size = versions_->manifest_file_size();
|
|
|
|
|
|
|
|
mutex_.Unlock();
|
|
|
|
return Status::OK();
|
|
|
|
}
|
|
|
|
|
|
|
|
Status DBImpl::GetSortedWalFiles(VectorLogPtr& files) {
|
|
|
|
// Record tracked WALs as a (minimum) cross-check for directory scan
|
|
|
|
std::vector<uint64_t> required_by_manifest;
|
|
|
|
|
|
|
|
// If caller disabled deletions, this function should return files that are
|
|
|
|
// guaranteed not to be deleted until deletions are re-enabled. We need to
|
|
|
|
// wait for pending purges to finish since WalManager doesn't know which
|
|
|
|
// files are going to be purged. Additional purges won't be scheduled as
|
|
|
|
// long as deletions are disabled (so the below loop must terminate).
|
|
|
|
// Also note that we disable deletions anyway to avoid the case where a
|
|
|
|
// file is deleted in the middle of the scan, causing IO error.
|
|
|
|
Status deletions_disabled = DisableFileDeletions();
|
|
|
|
{
|
|
|
|
InstrumentedMutexLock l(&mutex_);
|
|
|
|
while (pending_purge_obsolete_files_ > 0 || bg_purge_scheduled_ > 0) {
|
|
|
|
bg_cv_.Wait();
|
|
|
|
}
|
|
|
|
|
|
|
|
// Record tracked WALs as a (minimum) cross-check for directory scan
|
|
|
|
const auto& manifest_wals = versions_->GetWalSet().GetWals();
|
|
|
|
required_by_manifest.reserve(manifest_wals.size());
|
|
|
|
for (const auto& wal : manifest_wals) {
|
|
|
|
required_by_manifest.push_back(wal.first);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
Status s = wal_manager_.GetSortedWalFiles(files);
|
|
|
|
|
|
|
|
// DisableFileDeletions / EnableFileDeletions not supported in read-only DB
|
|
|
|
if (deletions_disabled.ok()) {
|
|
|
|
Status s2 = EnableFileDeletions(/*force*/ false);
|
|
|
|
assert(s2.ok());
|
|
|
|
s2.PermitUncheckedError();
|
|
|
|
} else {
|
|
|
|
assert(deletions_disabled.IsNotSupported());
|
|
|
|
}
|
|
|
|
|
|
|
|
if (s.ok()) {
|
|
|
|
// Verify includes those required by manifest (one sorted list is superset
|
|
|
|
// of the other)
|
|
|
|
auto required = required_by_manifest.begin();
|
|
|
|
auto included = files.begin();
|
|
|
|
|
|
|
|
while (required != required_by_manifest.end()) {
|
|
|
|
if (included == files.end() || *required < (*included)->LogNumber()) {
|
|
|
|
// FAIL - did not find
|
|
|
|
return Status::Corruption(
|
|
|
|
"WAL file " + std::to_string(*required) +
|
|
|
|
" required by manifest but not in directory list");
|
|
|
|
}
|
|
|
|
if (*required == (*included)->LogNumber()) {
|
|
|
|
++required;
|
|
|
|
++included;
|
|
|
|
} else {
|
|
|
|
assert(*required > (*included)->LogNumber());
|
|
|
|
++included;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
return s;
|
|
|
|
}
|
|
|
|
|
|
|
|
Status DBImpl::GetCurrentWalFile(std::unique_ptr<LogFile>* current_log_file) {
|
|
|
|
uint64_t current_logfile_number;
|
|
|
|
{
|
|
|
|
InstrumentedMutexLock l(&mutex_);
|
|
|
|
current_logfile_number = logfile_number_;
|
|
|
|
}
|
|
|
|
|
|
|
|
return wal_manager_.GetLiveWalFile(current_logfile_number, current_log_file);
|
|
|
|
}
|
|
|
|
|
|
|
|
Status DBImpl::GetLiveFilesStorageInfo(
|
|
|
|
const LiveFilesStorageInfoOptions& opts,
|
|
|
|
std::vector<LiveFileStorageInfo>* files) {
|
|
|
|
// To avoid returning partial results, only move results to files on success.
|
|
|
|
assert(files);
|
|
|
|
files->clear();
|
|
|
|
std::vector<LiveFileStorageInfo> results;
|
|
|
|
|
|
|
|
// NOTE: This implementation was largely migrated from Checkpoint.
|
|
|
|
|
|
|
|
Status s;
|
|
|
|
VectorLogPtr live_wal_files;
|
|
|
|
bool flush_memtable = true;
|
|
|
|
if (!immutable_db_options_.allow_2pc) {
|
|
|
|
if (opts.wal_size_for_flush == std::numeric_limits<uint64_t>::max()) {
|
|
|
|
flush_memtable = false;
|
|
|
|
} else if (opts.wal_size_for_flush > 0) {
|
|
|
|
// If the outstanding log files are small, we skip the flush.
|
|
|
|
s = GetSortedWalFiles(live_wal_files);
|
|
|
|
|
|
|
|
if (!s.ok()) {
|
|
|
|
return s;
|
|
|
|
}
|
|
|
|
|
|
|
|
// Don't flush column families if total log size is smaller than
|
|
|
|
// log_size_for_flush. We copy the log files instead.
|
|
|
|
// We may be able to cover 2PC case too.
|
|
|
|
uint64_t total_wal_size = 0;
|
|
|
|
for (auto& wal : live_wal_files) {
|
|
|
|
total_wal_size += wal->SizeFileBytes();
|
|
|
|
}
|
|
|
|
if (total_wal_size < opts.wal_size_for_flush) {
|
|
|
|
flush_memtable = false;
|
|
|
|
}
|
|
|
|
live_wal_files.clear();
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// This is a modified version of GetLiveFiles, to get access to more
|
|
|
|
// metadata.
|
|
|
|
mutex_.Lock();
|
|
|
|
if (flush_memtable) {
|
|
|
|
Status status = FlushForGetLiveFiles();
|
|
|
|
if (!status.ok()) {
|
|
|
|
mutex_.Unlock();
|
|
|
|
ROCKS_LOG_ERROR(immutable_db_options_.info_log, "Cannot Flush data %s\n",
|
|
|
|
status.ToString().c_str());
|
|
|
|
return status;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// Make a set of all of the live table and blob files
|
|
|
|
for (auto cfd : *versions_->GetColumnFamilySet()) {
|
|
|
|
if (cfd->IsDropped()) {
|
|
|
|
continue;
|
|
|
|
}
|
|
|
|
VersionStorageInfo& vsi = *cfd->current()->storage_info();
|
|
|
|
auto& cf_paths = cfd->ioptions()->cf_paths;
|
|
|
|
|
|
|
|
auto GetDir = [&](size_t path_id) {
|
|
|
|
// Matching TableFileName() behavior
|
|
|
|
if (path_id >= cf_paths.size()) {
|
|
|
|
assert(false);
|
|
|
|
return cf_paths.back().path;
|
|
|
|
} else {
|
|
|
|
return cf_paths[path_id].path;
|
|
|
|
}
|
|
|
|
};
|
|
|
|
|
|
|
|
for (int level = 0; level < vsi.num_levels(); ++level) {
|
|
|
|
const auto& level_files = vsi.LevelFiles(level);
|
|
|
|
for (const auto& meta : level_files) {
|
|
|
|
assert(meta);
|
|
|
|
|
|
|
|
results.emplace_back();
|
|
|
|
LiveFileStorageInfo& info = results.back();
|
|
|
|
|
|
|
|
info.relative_filename = MakeTableFileName(meta->fd.GetNumber());
|
|
|
|
info.directory = GetDir(meta->fd.GetPathId());
|
|
|
|
info.file_number = meta->fd.GetNumber();
|
|
|
|
info.file_type = kTableFile;
|
|
|
|
info.size = meta->fd.GetFileSize();
|
|
|
|
if (opts.include_checksum_info) {
|
|
|
|
info.file_checksum_func_name = meta->file_checksum_func_name;
|
|
|
|
info.file_checksum = meta->file_checksum;
|
|
|
|
if (info.file_checksum_func_name.empty()) {
|
|
|
|
info.file_checksum_func_name = kUnknownFileChecksumFuncName;
|
|
|
|
info.file_checksum = kUnknownFileChecksum;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
info.temperature = meta->temperature;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
const auto& blob_files = vsi.GetBlobFiles();
|
Use a sorted vector instead of a map to store blob file metadata (#9526)
Summary:
The patch replaces `std::map` with a sorted `std::vector` for
`VersionStorageInfo::blob_files_` and preallocates the space
for the `vector` before saving the `BlobFileMetaData` into the
new `VersionStorageInfo` in `VersionBuilder::Rep::SaveBlobFilesTo`.
These changes reduce the time the DB mutex is held while
saving new `Version`s, and using a sorted `vector` also makes
lookups faster thanks to better memory locality.
In addition, the patch introduces helper methods
`VersionStorageInfo::GetBlobFileMetaData` and
`VersionStorageInfo::GetBlobFileMetaDataLB` that can be used by
clients to perform lookups in the `vector`, and does some general
cleanup in the parts of code where blob file metadata are used.
Pull Request resolved: https://github.com/facebook/rocksdb/pull/9526
Test Plan:
Ran `make check` and the crash test script for a while.
Performance was tested using a load-optimized benchmark (`fillseq` with vector memtable, no WAL) and small file sizes so that a significant number of files are produced:
```
numactl --interleave=all ./db_bench --benchmarks=fillseq --allow_concurrent_memtable_write=false --level0_file_num_compaction_trigger=4 --level0_slowdown_writes_trigger=20 --level0_stop_writes_trigger=30 --max_background_jobs=8 --max_write_buffer_number=8 --db=/data/ltamasi-dbbench --wal_dir=/data/ltamasi-dbbench --num=800000000 --num_levels=8 --key_size=20 --value_size=400 --block_size=8192 --cache_size=51539607552 --cache_numshardbits=6 --compression_max_dict_bytes=0 --compression_ratio=0.5 --compression_type=lz4 --bytes_per_sync=8388608 --cache_index_and_filter_blocks=1 --cache_high_pri_pool_ratio=0.5 --benchmark_write_rate_limit=0 --write_buffer_size=16777216 --target_file_size_base=16777216 --max_bytes_for_level_base=67108864 --verify_checksum=1 --delete_obsolete_files_period_micros=62914560 --max_bytes_for_level_multiplier=8 --statistics=0 --stats_per_interval=1 --stats_interval_seconds=20 --histogram=1 --memtablerep=skip_list --bloom_bits=10 --open_files=-1 --subcompactions=1 --compaction_style=0 --min_level_to_compress=3 --level_compaction_dynamic_level_bytes=true --pin_l0_filter_and_index_blocks_in_cache=1 --soft_pending_compaction_bytes_limit=167503724544 --hard_pending_compaction_bytes_limit=335007449088 --min_level_to_compress=0 --use_existing_db=0 --sync=0 --threads=1 --memtablerep=vector --allow_concurrent_memtable_write=false --disable_wal=1 --enable_blob_files=1 --blob_file_size=16777216 --min_blob_size=0 --blob_compression_type=lz4 --enable_blob_garbage_collection=1 --seed=<some value>
```
Final statistics before the patch:
```
Cumulative writes: 0 writes, 700M keys, 0 commit groups, 0.0 writes per commit group, ingest: 284.62 GB, 121.27 MB/s
Interval writes: 0 writes, 334K keys, 0 commit groups, 0.0 writes per commit group, ingest: 139.28 MB, 72.46 MB/s
```
With the patch:
```
Cumulative writes: 0 writes, 760M keys, 0 commit groups, 0.0 writes per commit group, ingest: 308.66 GB, 131.52 MB/s
Interval writes: 0 writes, 445K keys, 0 commit groups, 0.0 writes per commit group, ingest: 185.35 MB, 93.15 MB/s
```
Total time to complete the benchmark is 2611 seconds with the patch, down from 2986 secs.
Reviewed By: riversand963
Differential Revision: D34082728
Pulled By: ltamasi
fbshipit-source-id: fc598abf676dce436734d06bb9d2d99a26a004fc
3 years ago
|
|
|
for (const auto& meta : blob_files) {
|
|
|
|
assert(meta);
|
|
|
|
|
|
|
|
results.emplace_back();
|
|
|
|
LiveFileStorageInfo& info = results.back();
|
|
|
|
|
|
|
|
info.relative_filename = BlobFileName(meta->GetBlobFileNumber());
|
Use a sorted vector instead of a map to store blob file metadata (#9526)
Summary:
The patch replaces `std::map` with a sorted `std::vector` for
`VersionStorageInfo::blob_files_` and preallocates the space
for the `vector` before saving the `BlobFileMetaData` into the
new `VersionStorageInfo` in `VersionBuilder::Rep::SaveBlobFilesTo`.
These changes reduce the time the DB mutex is held while
saving new `Version`s, and using a sorted `vector` also makes
lookups faster thanks to better memory locality.
In addition, the patch introduces helper methods
`VersionStorageInfo::GetBlobFileMetaData` and
`VersionStorageInfo::GetBlobFileMetaDataLB` that can be used by
clients to perform lookups in the `vector`, and does some general
cleanup in the parts of code where blob file metadata are used.
Pull Request resolved: https://github.com/facebook/rocksdb/pull/9526
Test Plan:
Ran `make check` and the crash test script for a while.
Performance was tested using a load-optimized benchmark (`fillseq` with vector memtable, no WAL) and small file sizes so that a significant number of files are produced:
```
numactl --interleave=all ./db_bench --benchmarks=fillseq --allow_concurrent_memtable_write=false --level0_file_num_compaction_trigger=4 --level0_slowdown_writes_trigger=20 --level0_stop_writes_trigger=30 --max_background_jobs=8 --max_write_buffer_number=8 --db=/data/ltamasi-dbbench --wal_dir=/data/ltamasi-dbbench --num=800000000 --num_levels=8 --key_size=20 --value_size=400 --block_size=8192 --cache_size=51539607552 --cache_numshardbits=6 --compression_max_dict_bytes=0 --compression_ratio=0.5 --compression_type=lz4 --bytes_per_sync=8388608 --cache_index_and_filter_blocks=1 --cache_high_pri_pool_ratio=0.5 --benchmark_write_rate_limit=0 --write_buffer_size=16777216 --target_file_size_base=16777216 --max_bytes_for_level_base=67108864 --verify_checksum=1 --delete_obsolete_files_period_micros=62914560 --max_bytes_for_level_multiplier=8 --statistics=0 --stats_per_interval=1 --stats_interval_seconds=20 --histogram=1 --memtablerep=skip_list --bloom_bits=10 --open_files=-1 --subcompactions=1 --compaction_style=0 --min_level_to_compress=3 --level_compaction_dynamic_level_bytes=true --pin_l0_filter_and_index_blocks_in_cache=1 --soft_pending_compaction_bytes_limit=167503724544 --hard_pending_compaction_bytes_limit=335007449088 --min_level_to_compress=0 --use_existing_db=0 --sync=0 --threads=1 --memtablerep=vector --allow_concurrent_memtable_write=false --disable_wal=1 --enable_blob_files=1 --blob_file_size=16777216 --min_blob_size=0 --blob_compression_type=lz4 --enable_blob_garbage_collection=1 --seed=<some value>
```
Final statistics before the patch:
```
Cumulative writes: 0 writes, 700M keys, 0 commit groups, 0.0 writes per commit group, ingest: 284.62 GB, 121.27 MB/s
Interval writes: 0 writes, 334K keys, 0 commit groups, 0.0 writes per commit group, ingest: 139.28 MB, 72.46 MB/s
```
With the patch:
```
Cumulative writes: 0 writes, 760M keys, 0 commit groups, 0.0 writes per commit group, ingest: 308.66 GB, 131.52 MB/s
Interval writes: 0 writes, 445K keys, 0 commit groups, 0.0 writes per commit group, ingest: 185.35 MB, 93.15 MB/s
```
Total time to complete the benchmark is 2611 seconds with the patch, down from 2986 secs.
Reviewed By: riversand963
Differential Revision: D34082728
Pulled By: ltamasi
fbshipit-source-id: fc598abf676dce436734d06bb9d2d99a26a004fc
3 years ago
|
|
|
info.directory = GetDir(/* path_id */ 0);
|
|
|
|
info.file_number = meta->GetBlobFileNumber();
|
|
|
|
info.file_type = kBlobFile;
|
|
|
|
info.size = meta->GetBlobFileSize();
|
|
|
|
if (opts.include_checksum_info) {
|
|
|
|
info.file_checksum_func_name = meta->GetChecksumMethod();
|
|
|
|
info.file_checksum = meta->GetChecksumValue();
|
|
|
|
if (info.file_checksum_func_name.empty()) {
|
|
|
|
info.file_checksum_func_name = kUnknownFileChecksumFuncName;
|
|
|
|
info.file_checksum = kUnknownFileChecksum;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
// TODO?: info.temperature
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// Capture some final info before releasing mutex
|
|
|
|
const uint64_t manifest_number = versions_->manifest_file_number();
|
|
|
|
const uint64_t manifest_size = versions_->manifest_file_size();
|
|
|
|
const uint64_t options_number = versions_->options_file_number();
|
|
|
|
const uint64_t options_size = versions_->options_file_size_;
|
|
|
|
const uint64_t min_log_num = MinLogNumberToKeep();
|
|
|
|
|
|
|
|
mutex_.Unlock();
|
|
|
|
|
|
|
|
std::string manifest_fname = DescriptorFileName(manifest_number);
|
|
|
|
{ // MANIFEST
|
|
|
|
results.emplace_back();
|
|
|
|
LiveFileStorageInfo& info = results.back();
|
|
|
|
|
|
|
|
info.relative_filename = manifest_fname;
|
|
|
|
info.directory = GetName();
|
|
|
|
info.file_number = manifest_number;
|
|
|
|
info.file_type = kDescriptorFile;
|
|
|
|
info.size = manifest_size;
|
|
|
|
info.trim_to_size = true;
|
|
|
|
if (opts.include_checksum_info) {
|
|
|
|
info.file_checksum_func_name = kUnknownFileChecksumFuncName;
|
|
|
|
info.file_checksum = kUnknownFileChecksum;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
{ // CURRENT
|
|
|
|
results.emplace_back();
|
|
|
|
LiveFileStorageInfo& info = results.back();
|
|
|
|
|
|
|
|
info.relative_filename = kCurrentFileName;
|
|
|
|
info.directory = GetName();
|
|
|
|
info.file_type = kCurrentFile;
|
|
|
|
// CURRENT could be replaced so we have to record the contents as needed.
|
|
|
|
info.replacement_contents = manifest_fname + "\n";
|
|
|
|
info.size = manifest_fname.size() + 1;
|
|
|
|
if (opts.include_checksum_info) {
|
|
|
|
info.file_checksum_func_name = kUnknownFileChecksumFuncName;
|
|
|
|
info.file_checksum = kUnknownFileChecksum;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// The OPTIONS file number is zero in read-write mode when OPTIONS file
|
|
|
|
// writing failed and the DB was configured with
|
|
|
|
// `fail_if_options_file_error == false`. In read-only mode the OPTIONS file
|
|
|
|
// number is zero when no OPTIONS file exist at all. In those cases we do not
|
|
|
|
// record any OPTIONS file in the live file list.
|
|
|
|
if (options_number != 0) {
|
|
|
|
results.emplace_back();
|
|
|
|
LiveFileStorageInfo& info = results.back();
|
|
|
|
|
|
|
|
info.relative_filename = OptionsFileName(options_number);
|
|
|
|
info.directory = GetName();
|
|
|
|
info.file_number = options_number;
|
|
|
|
info.file_type = kOptionsFile;
|
|
|
|
info.size = options_size;
|
|
|
|
if (opts.include_checksum_info) {
|
|
|
|
info.file_checksum_func_name = kUnknownFileChecksumFuncName;
|
|
|
|
info.file_checksum = kUnknownFileChecksum;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// Some legacy testing stuff TODO: carefully clean up obsolete parts
|
|
|
|
TEST_SYNC_POINT("CheckpointImpl::CreateCheckpoint:FlushDone");
|
|
|
|
|
|
|
|
TEST_SYNC_POINT("CheckpointImpl::CreateCheckpoint:SavedLiveFiles1");
|
|
|
|
TEST_SYNC_POINT("CheckpointImpl::CreateCheckpoint:SavedLiveFiles2");
|
|
|
|
|
|
|
|
if (s.ok()) {
|
|
|
|
// To maximize the effectiveness of track_and_verify_wals_in_manifest,
|
|
|
|
// sync WAL when it is enabled.
|
|
|
|
s = FlushWAL(
|
|
|
|
immutable_db_options_.track_and_verify_wals_in_manifest /* sync */);
|
|
|
|
if (s.IsNotSupported()) { // read-only DB or similar
|
|
|
|
s = Status::OK();
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
TEST_SYNC_POINT("CheckpointImpl::CreateCustomCheckpoint:AfterGetLive1");
|
|
|
|
TEST_SYNC_POINT("CheckpointImpl::CreateCustomCheckpoint:AfterGetLive2");
|
|
|
|
|
|
|
|
// If we have more than one column family, we also need to get WAL files.
|
|
|
|
if (s.ok()) {
|
|
|
|
s = GetSortedWalFiles(live_wal_files);
|
|
|
|
}
|
|
|
|
if (!s.ok()) {
|
|
|
|
return s;
|
|
|
|
}
|
|
|
|
|
|
|
|
size_t wal_size = live_wal_files.size();
|
|
|
|
|
|
|
|
ROCKS_LOG_INFO(immutable_db_options_.info_log,
|
|
|
|
"Number of log files %" ROCKSDB_PRIszt, live_wal_files.size());
|
|
|
|
|
|
|
|
// Link WAL files. Copy exact size of last one because it is the only one
|
|
|
|
// that has changes after the last flush.
|
|
|
|
auto wal_dir = immutable_db_options_.GetWalDir();
|
|
|
|
for (size_t i = 0; s.ok() && i < wal_size; ++i) {
|
|
|
|
if ((live_wal_files[i]->Type() == kAliveLogFile) &&
|
|
|
|
(!flush_memtable || live_wal_files[i]->LogNumber() >= min_log_num)) {
|
|
|
|
results.emplace_back();
|
|
|
|
LiveFileStorageInfo& info = results.back();
|
|
|
|
auto f = live_wal_files[i]->PathName();
|
|
|
|
assert(!f.empty() && f[0] == '/');
|
|
|
|
info.relative_filename = f.substr(1);
|
|
|
|
info.directory = wal_dir;
|
|
|
|
info.file_number = live_wal_files[i]->LogNumber();
|
|
|
|
info.file_type = kWalFile;
|
|
|
|
info.size = live_wal_files[i]->SizeFileBytes();
|
|
|
|
// Only last should need to be trimmed
|
|
|
|
info.trim_to_size = (i + 1 == wal_size);
|
|
|
|
if (opts.include_checksum_info) {
|
|
|
|
info.file_checksum_func_name = kUnknownFileChecksumFuncName;
|
|
|
|
info.file_checksum = kUnknownFileChecksum;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if (s.ok()) {
|
|
|
|
// Only move results to output on success.
|
|
|
|
*files = std::move(results);
|
|
|
|
}
|
|
|
|
return s;
|
|
|
|
}
|
|
|
|
|
|
|
|
} // namespace ROCKSDB_NAMESPACE
|