diff --git a/dbms/src/Interpreters/AsynchronousMetrics.cpp b/dbms/src/Interpreters/AsynchronousMetrics.cpp index e08725ea91d..6a49a8410ae 100644 --- a/dbms/src/Interpreters/AsynchronousMetrics.cpp +++ b/dbms/src/Interpreters/AsynchronousMetrics.cpp @@ -15,8 +15,10 @@ #include #include #include +#include #include #include +#include #include #include #include @@ -125,27 +127,62 @@ static void calculateMaxAndSum(Max & max, Sum & sum, T x) FileUsageStatistics AsynchronousMetrics::getPageStorageFileUsage() { - RUNTIME_ASSERT(!(context.getSharedContextDisagg()->isDisaggregatedComputeMode() && context.getSharedContextDisagg()->use_autoscaler)); - // Get from RegionPersister - auto & tmt = context.getTMTContext(); - auto & kvstore = tmt.getKVStore(); - FileUsageStatistics usage = kvstore->getFileUsageStatistics(); - - // Get the blob file status from all PS V3 instances - if (auto global_storage_pool = context.getGlobalStoragePool(); global_storage_pool != nullptr) + FileUsageStatistics usage; + switch (context.getSharedContextDisagg()->disaggregated_mode) { - const auto log_usage = global_storage_pool->log_storage->getFileUsageStatistics(); - const auto meta_usage = global_storage_pool->meta_storage->getFileUsageStatistics(); - const auto data_usage = global_storage_pool->data_storage->getFileUsageStatistics(); + case DisaggregatedMode::None: + { + if (auto uni_ps = context.tryGetWriteNodePageStorage(); uni_ps != nullptr) + { + /// When format_version=5 is enabled, then all data are stored in the `uni_ps` + usage.merge(uni_ps->getFileUsageStatistics()); + } + else + { + /// When format_version < 5, then there are multiple PageStorage instances - usage.merge(log_usage) - .merge(meta_usage) - .merge(data_usage); - } + // Get from RegionPersister + auto & tmt = context.getTMTContext(); + auto & kvstore = tmt.getKVStore(); + usage = kvstore->getFileUsageStatistics(); + + // Get the blob file status from all PS V3 instances + if (auto global_storage_pool = context.getGlobalStoragePool(); global_storage_pool != nullptr) + { + const auto log_usage = global_storage_pool->log_storage->getFileUsageStatistics(); + const auto meta_usage = global_storage_pool->meta_storage->getFileUsageStatistics(); + const auto data_usage = global_storage_pool->data_storage->getFileUsageStatistics(); - if (auto ps_cache = context.getSharedContextDisagg()->rn_page_cache_storage; ps_cache != nullptr) + usage.merge(log_usage) + .merge(meta_usage) + .merge(data_usage); + } + } + break; + } + case DisaggregatedMode::Storage: + { + // disagg write node, all data are stored in the `uni_ps` + if (auto uni_ps = context.getWriteNodePageStorage(); uni_ps != nullptr) + { + usage.merge(uni_ps->getFileUsageStatistics()); + } + break; + } + case DisaggregatedMode::Compute: { - usage.merge(ps_cache->getUniversalPageStorage()->getFileUsageStatistics()); + // disagg compute node without auto-scaler, the proxy data are stored in the `uni_ps` + if (auto uni_ps = context.getWriteNodePageStorage(); uni_ps != nullptr) + { + usage.merge(uni_ps->getFileUsageStatistics()); + } + // disagg compute node, all cache page data are stored in the `ps_cache` + if (auto ps_cache = context.getSharedContextDisagg()->rn_page_cache_storage; ps_cache != nullptr) + { + usage.merge(ps_cache->getUniversalPageStorage()->getFileUsageStatistics()); + } + break; + } } return usage; @@ -206,7 +243,6 @@ void AsynchronousMetrics::update() set("MaxDTBackgroundTasksLength", max_dt_background_tasks_length); } - if (!(context.getSharedContextDisagg()->isDisaggregatedComputeMode() && context.getSharedContextDisagg()->use_autoscaler)) { const FileUsageStatistics usage = getPageStorageFileUsage(); set("BlobFileNums", usage.total_file_num); @@ -217,6 +253,15 @@ void AsynchronousMetrics::update() set("PagesInMem", usage.num_pages); } + if (context.getSharedContextDisagg()->isDisaggregatedStorageMode()) + { + auto & tmt = context.getTMTContext(); + if (auto s3_gc_owner = tmt.getS3GCOwnerManager(); s3_gc_owner->isOwner()) + { + GET_METRIC(tiflash_storage_s3_gc_status, type_owner).Set(1.0); + } + } + #if USE_MIMALLOC #define MI_STATS_SET(X) set("mimalloc." #X, X) @@ -256,7 +301,7 @@ void AsynchronousMetrics::update() M("background_thread.num_runs", uint64_t) \ M("background_thread.run_interval", uint64_t) -#define GET_METRIC(NAME, TYPE) \ +#define GET_JEMALLOC_METRIC(NAME, TYPE) \ do \ { \ TYPE value{}; \ @@ -265,9 +310,9 @@ void AsynchronousMetrics::update() set("jemalloc." NAME, value); \ } while (0); - FOR_EACH_METRIC(GET_METRIC); + FOR_EACH_METRIC(GET_JEMALLOC_METRIC); -#undef GET_METRIC +#undef GET_JEMALLOC_METRIC #undef FOR_EACH_METRIC } #endif diff --git a/dbms/src/Storages/Page/PageConstants.h b/dbms/src/Storages/Page/PageConstants.h index 0e5e4a091a3..f8b0364702e 100644 --- a/dbms/src/Storages/Page/PageConstants.h +++ b/dbms/src/Storages/Page/PageConstants.h @@ -24,10 +24,15 @@ static constexpr UInt64 GB = MB * 1024; enum class StorageType { + Unknown = 0, Log = 1, Data = 2, Meta = 3, KVStore = 4, + RaftEngine = 5, + KVEngine = 6, + + _MAX_STORAGE_TYPE_, // NOLINT(bugprone-reserved-identifier) }; enum class PageStorageRunMode : UInt8 diff --git a/dbms/src/Storages/Page/V3/CheckpointFile/CPDataFileStat.h b/dbms/src/Storages/Page/V3/CheckpointFile/CPDataFileStat.h index 7c5bf352d23..1b20c262b8a 100644 --- a/dbms/src/Storages/Page/V3/CheckpointFile/CPDataFileStat.h +++ b/dbms/src/Storages/Page/V3/CheckpointFile/CPDataFileStat.h @@ -26,12 +26,6 @@ namespace DB::PS::V3 { -struct CPDataWriteStats -{ - bool has_new_data = false; - size_t incremental_data_bytes = 0; - size_t compact_data_bytes = 0; -}; using RemoteFileValidSizes = std::unordered_map; diff --git a/dbms/src/Storages/Page/V3/CheckpointFile/CPDumpStat.cpp b/dbms/src/Storages/Page/V3/CheckpointFile/CPDumpStat.cpp new file mode 100644 index 00000000000..3e1da69d23b --- /dev/null +++ b/dbms/src/Storages/Page/V3/CheckpointFile/CPDumpStat.cpp @@ -0,0 +1,78 @@ +// Copyright 2023 PingCAP, Ltd. +// +// Licensed under the Apache License, Version 2.0 (the "License"); +// you may not use this file except in compliance with the License. +// You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, software +// distributed under the License is distributed on an "AS IS" BASIS, +// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. +// See the License for the specific language governing permissions and +// limitations under the License. + +#include +#include +#include +#include + +namespace DB::PS::V3 +{ + +void SetMetrics(const CPDataDumpStats & stats) +{ + for (size_t i = 0; i < static_cast(DB::StorageType::_MAX_STORAGE_TYPE_); ++i) + { + auto type = static_cast(i); + switch (type) + { + case DB::StorageType::Unknown: + { + GET_METRIC(tiflash_storage_checkpoint_keys_by_types, type_unknown).Increment(stats.num_keys[i]); + GET_METRIC(tiflash_storage_checkpoint_flow_by_types, type_unknown).Increment(stats.num_bytes[i]); + break; + } + case DB::StorageType::RaftEngine: + { + GET_METRIC(tiflash_storage_checkpoint_keys_by_types, type_raftengine).Increment(stats.num_keys[i]); + GET_METRIC(tiflash_storage_checkpoint_flow_by_types, type_raftengine).Increment(stats.num_bytes[i]); + break; + } + case DB::StorageType::KVEngine: + { + GET_METRIC(tiflash_storage_checkpoint_keys_by_types, type_kvengine).Increment(stats.num_keys[i]); + GET_METRIC(tiflash_storage_checkpoint_flow_by_types, type_kvengine).Increment(stats.num_bytes[i]); + break; + } + case DB::StorageType::KVStore: + { + GET_METRIC(tiflash_storage_checkpoint_keys_by_types, type_kvstore).Increment(stats.num_keys[i]); + GET_METRIC(tiflash_storage_checkpoint_flow_by_types, type_kvstore).Increment(stats.num_bytes[i]); + break; + } + case DB::StorageType::Data: + { + GET_METRIC(tiflash_storage_checkpoint_keys_by_types, type_data).Increment(stats.num_keys[i]); + GET_METRIC(tiflash_storage_checkpoint_flow_by_types, type_data).Increment(stats.num_bytes[i]); + break; + } + case DB::StorageType::Log: + { + GET_METRIC(tiflash_storage_checkpoint_keys_by_types, type_log).Increment(stats.num_keys[i]); + GET_METRIC(tiflash_storage_checkpoint_flow_by_types, type_log).Increment(stats.num_bytes[i]); + break; + } + case DB::StorageType::Meta: + { + GET_METRIC(tiflash_storage_checkpoint_keys_by_types, type_meta).Increment(stats.num_keys[i]); + GET_METRIC(tiflash_storage_checkpoint_flow_by_types, type_meta).Increment(stats.num_bytes[i]); + break; + } + default: + __builtin_unreachable(); + } + } +} + +} // namespace DB::PS::V3 diff --git a/dbms/src/Storages/Page/V3/CheckpointFile/CPDumpStat.h b/dbms/src/Storages/Page/V3/CheckpointFile/CPDumpStat.h new file mode 100644 index 00000000000..c9f4a9e5fff --- /dev/null +++ b/dbms/src/Storages/Page/V3/CheckpointFile/CPDumpStat.h @@ -0,0 +1,102 @@ +// Copyright 2023 PingCAP, Ltd. +// +// Licensed under the Apache License, Version 2.0 (the "License"); +// you may not use this file except in compliance with the License. +// You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, software +// distributed under the License is distributed on an "AS IS" BASIS, +// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. +// See the License for the specific language governing permissions and +// limitations under the License. + +#pragma once + +#include +#include + +#include + +namespace DB::PS::V3 +{ + +struct CPDataDumpStats +{ + bool has_new_data = false; + + size_t incremental_data_bytes = 0; + size_t compact_data_bytes = 0; + + std::array(StorageType::_MAX_STORAGE_TYPE_)> num_keys{}; + std::array(StorageType::_MAX_STORAGE_TYPE_)> num_bytes{}; + + // Total number of records in this checkpoint + size_t num_records = 0; + // Number of Pages that already uploaded to S3 + // and is not changed in this checkpoint + size_t num_pages_unchanged = 0; + // Number of Pages that already uploaded to S3 + // but picked by compaction in this checkpoint + size_t num_pages_compact = 0; + // Number of incremental Pages since last checkpoint + size_t num_pages_incremental = 0; + // Number of ExternalPages + size_t num_ext_pages = 0; + // Number of RefPages + size_t num_ref_pages = 0; + // Number of delete records + size_t num_delete_records = 0; + // Number of other records other than Pages/ExternalPages + size_t num_other_records = 0; +}; + +void SetMetrics(const CPDataDumpStats & stats); + +} // namespace DB::PS::V3 + +template <> +struct fmt::formatter +{ + static constexpr auto parse(format_parse_context & ctx) { return ctx.begin(); } + + template + auto format(const DB::PS::V3::CPDataDumpStats & value, FormatContext & ctx) const -> decltype(ctx.out()) + { + auto it = format_to( + ctx.out(), + "CPDataDumpStats{{" + "incremental_data_bytes={} compact_data_bytes={}" + " n_records{{total={}" + " pages_unchanged={} pages_compact={} pages_incremental={} ext_pages={} ref_pages={}" + " delete={} other={}}}", + value.incremental_data_bytes, + value.compact_data_bytes, + value.num_records, + value.num_pages_unchanged, + value.num_pages_compact, + value.num_pages_incremental, + value.num_ext_pages, + value.num_ref_pages, + value.num_delete_records, + value.num_other_records); + it = format_to(it, " types["); + for (size_t i = 0; i < static_cast(DB::StorageType::_MAX_STORAGE_TYPE_); ++i) + { + if (i != 0) + it = format_to(it, " "); + it = format_to( + it, + "{{type={} keys={} bytes={}}}", + magic_enum::enum_name(static_cast(i)), + value.num_keys[i], + value.num_bytes[i]); + } + return format_to( + it, + "]" // end of "keys" + "}}" // end of "CPDataDumpStats" + ); + } +}; diff --git a/dbms/src/Storages/Page/V3/CheckpointFile/CPFilesWriter.cpp b/dbms/src/Storages/Page/V3/CheckpointFile/CPFilesWriter.cpp index 6e4c0a02b70..c6f9a77ce50 100644 --- a/dbms/src/Storages/Page/V3/CheckpointFile/CPFilesWriter.cpp +++ b/dbms/src/Storages/Page/V3/CheckpointFile/CPFilesWriter.cpp @@ -12,8 +12,11 @@ // See the License for the specific language governing permissions and // limitations under the License. +#include #include +#include #include +#include #include #include @@ -62,7 +65,7 @@ void CPFilesWriter::writePrefix(const CPFilesWriter::PrefixInfo & info) write_stage = WriteStage::WritingEdits; } -CPDataWriteStats CPFilesWriter::writeEditsAndApplyCheckpointInfo( +CPDataDumpStats CPFilesWriter::writeEditsAndApplyCheckpointInfo( universal::PageEntriesEdit & edits, const std::unordered_set & file_ids_to_compact) { @@ -72,14 +75,28 @@ CPDataWriteStats CPFilesWriter::writeEditsAndApplyCheckpointInfo( if (records.empty()) return {.has_new_data = false}; - CPDataWriteStats write_down_stats; + CPDataDumpStats write_down_stats; + for (size_t i = 0; i < static_cast(StorageType::_MAX_STORAGE_TYPE_); ++i) + { + write_down_stats.num_keys[i] = 0; + write_down_stats.num_bytes[i] = 0; + } + std::unordered_map compact_stats; bool last_page_is_raft_data = true; // 1. Iterate all edits, find these entry edits without the checkpoint info // and collect the lock files from applied entries. + write_down_stats.num_records = records.size(); for (auto & rec_edit : records) { + StorageType id_storage_type = StorageType::Unknown; + { + id_storage_type = UniversalPageIdFormat::getUniversalPageIdType(rec_edit.page_id); + write_down_stats.num_keys[static_cast(id_storage_type)] += 1; + write_down_stats.num_bytes[static_cast(id_storage_type)] += rec_edit.entry.size; + } + if (rec_edit.type == EditRecordType::VAR_EXTERNAL) { RUNTIME_CHECK_MSG( @@ -90,11 +107,26 @@ CPDataWriteStats CPFilesWriter::writeEditsAndApplyCheckpointInfo( rec_edit); // for example, the s3 fullpath of external id locked_files.emplace(*rec_edit.entry.checkpoint_info.data_location.data_file_id); + write_down_stats.num_ext_pages += 1; continue; } if (rec_edit.type != EditRecordType::VAR_ENTRY) + { + if (rec_edit.type == EditRecordType::VAR_REF) + { + write_down_stats.num_ref_pages += 1; + } + else if (rec_edit.type == EditRecordType::VAR_DELETE) + { + write_down_stats.num_delete_records += 1; + } + else + { + write_down_stats.num_other_records += 1; + } continue; + } bool is_compaction = false; if (rec_edit.entry.checkpoint_info.has_value()) @@ -104,6 +136,7 @@ CPDataWriteStats CPFilesWriter::writeEditsAndApplyCheckpointInfo( { // for example, the s3 fullpath that was written in the previous uploaded CheckpointDataFile locked_files.emplace(file_id); + write_down_stats.num_pages_unchanged += 1; continue; } // else we rewrite this entry data to the data file generated by this checkpoint, so that @@ -113,7 +146,7 @@ CPDataWriteStats CPFilesWriter::writeEditsAndApplyCheckpointInfo( compact_stats.try_emplace(file_id, 0).first->second += rec_edit.entry.size; } - bool current_page_is_raft_data = rec_edit.page_id.isRaftData(); + bool current_page_is_raft_data = (id_storage_type == StorageType::RaftEngine); if (current_write_size > 0 // If current_write_size is 0, data_writer is a empty file, not need to create a new one. && (current_page_is_raft_data != last_page_is_raft_data // Data type changed || (max_data_file_size != 0 && current_write_size >= max_data_file_size))) // or reach size limit. @@ -143,10 +176,12 @@ CPDataWriteStats CPFilesWriter::writeEditsAndApplyCheckpointInfo( if (is_compaction) { write_down_stats.compact_data_bytes += rec_edit.entry.size; + write_down_stats.num_pages_compact += 1; } else { write_down_stats.incremental_data_bytes += rec_edit.entry.size; + write_down_stats.num_pages_incremental += 1; } } diff --git a/dbms/src/Storages/Page/V3/CheckpointFile/CPFilesWriter.h b/dbms/src/Storages/Page/V3/CheckpointFile/CPFilesWriter.h index d69f5ec2e49..97a4b7fa163 100644 --- a/dbms/src/Storages/Page/V3/CheckpointFile/CPFilesWriter.h +++ b/dbms/src/Storages/Page/V3/CheckpointFile/CPFilesWriter.h @@ -17,6 +17,7 @@ #include #include #include +#include #include #include #include @@ -80,7 +81,7 @@ class CPFilesWriter : private boost::noncopyable * * You must call `writeSuffix` finally, if you don't plan to write edits anymore. */ - CPDataWriteStats writeEditsAndApplyCheckpointInfo( + CPDataDumpStats writeEditsAndApplyCheckpointInfo( universal::PageEntriesEdit & edit, const std::unordered_set & file_ids_to_compact = {}); diff --git a/dbms/src/Storages/Page/V3/Universal/UniversalPageId.h b/dbms/src/Storages/Page/V3/Universal/UniversalPageId.h index 3d6266d6906..d2247af98bd 100644 --- a/dbms/src/Storages/Page/V3/Universal/UniversalPageId.h +++ b/dbms/src/Storages/Page/V3/Universal/UniversalPageId.h @@ -70,13 +70,7 @@ class UniversalPageId final friend bool operator==(const String & lhs, const UniversalPageId & rhs); - bool isRaftData() const - { - return !id.empty() && id[0] == raft_prefix; - } - private: - static constexpr char raft_prefix = 0x01; String id; }; diff --git a/dbms/src/Storages/Page/V3/Universal/UniversalPageIdFormatImpl.h b/dbms/src/Storages/Page/V3/Universal/UniversalPageIdFormatImpl.h index 1b89faef8b0..c87c3dbd980 100644 --- a/dbms/src/Storages/Page/V3/Universal/UniversalPageIdFormatImpl.h +++ b/dbms/src/Storages/Page/V3/Universal/UniversalPageIdFormatImpl.h @@ -18,6 +18,7 @@ #include #include #include +#include #include #include #include @@ -92,12 +93,15 @@ struct UniversalPageIdFormat return toFullPageId(getSubPrefix(StorageType::KVStore), region_id); } + static constexpr char RAFT_PREFIX = 0x01; + static constexpr char KV_PREFIX = 0x02; + // data is in kv engine, so it is prepended by KV_PREFIX // KV_PREFIX LOCAL_PREFIX REGION_RAFT_PREFIX region_id APPLY_STATE_SUFFIX static UniversalPageId toRaftApplyStateKeyInKVEngine(UInt64 region_id) { WriteBufferFromOwnString buff; - writeChar(0x02, buff); + writeChar(KV_PREFIX, buff); writeChar(0x01, buff); writeChar(0x02, buff); encodeUInt64(region_id, buff); @@ -110,7 +114,7 @@ struct UniversalPageIdFormat static UniversalPageId toRegionLocalStateKeyInKVEngine(UInt64 region_id) { WriteBufferFromOwnString buff; - writeChar(0x02, buff); + writeChar(KV_PREFIX, buff); writeChar(0x01, buff); writeChar(0x03, buff); encodeUInt64(region_id, buff); @@ -122,7 +126,7 @@ struct UniversalPageIdFormat static String toFullRaftLogPrefix(UInt64 region_id) { WriteBufferFromOwnString buff; - writeChar(0x01, buff); + writeChar(RAFT_PREFIX, buff); writeChar(0x01, buff); writeChar(0x02, buff); encodeUInt64(region_id, buff); @@ -134,7 +138,7 @@ struct UniversalPageIdFormat static String toFullRaftLogScanEnd(UInt64 region_id) { WriteBufferFromOwnString buff; - writeChar(0x01, buff); + writeChar(RAFT_PREFIX, buff); writeChar(0x01, buff); writeChar(0x02, buff); encodeUInt64(region_id, buff); @@ -148,7 +152,7 @@ struct UniversalPageIdFormat static String getStoreIdentIdInKVEngine() { WriteBufferFromOwnString buff; - writeChar(0x02, buff); + writeChar(KV_PREFIX, buff); writeChar(0x01, buff); writeChar(0x01, buff); return buff.releaseStr(); @@ -158,7 +162,7 @@ struct UniversalPageIdFormat static String getStoreIdentId() { WriteBufferFromOwnString buff; - writeChar(0x01, buff); + writeChar(RAFT_PREFIX, buff); writeChar(0x01, buff); writeChar(0x01, buff); return buff.releaseStr(); @@ -190,6 +194,43 @@ struct UniversalPageIdFormat return page_id_without_keyspace.starts_with(getSubPrefix(type)); } + static inline StorageType getUniversalPageIdType(const UniversalPageId & page_id) + { + if (page_id.empty()) + return StorageType::Unknown; + + const auto & page_id_str = page_id.asStr(); + if (page_id_str[0] == RAFT_PREFIX) + { + return StorageType::RaftEngine; + } + else if (page_id_str[0] == KV_PREFIX) + { + return StorageType::KVEngine; + } + else + { + auto page_id_without_keyspace = TiKVKeyspaceID::removeKeyspaceID(std::string_view(page_id_str.data(), page_id_str.size())); + if (page_id_without_keyspace.starts_with(getSubPrefix(StorageType::Log))) + { + return StorageType::Log; + } + if (page_id_without_keyspace.starts_with(getSubPrefix(StorageType::Data))) + { + return StorageType::Data; + } + if (page_id_without_keyspace.starts_with(getSubPrefix(StorageType::Meta))) + { + return StorageType::Meta; + } + if (page_id_without_keyspace.starts_with(getSubPrefix(StorageType::KVStore))) + { + return StorageType::KVStore; + } + } + return StorageType::Unknown; + } + private: static inline void encodeUInt64(const UInt64 x, WriteBuffer & ss) { diff --git a/dbms/src/Storages/Page/V3/Universal/UniversalPageStorage.cpp b/dbms/src/Storages/Page/V3/Universal/UniversalPageStorage.cpp index fc3afe8cfee..705803939e3 100644 --- a/dbms/src/Storages/Page/V3/Universal/UniversalPageStorage.cpp +++ b/dbms/src/Storages/Page/V3/Universal/UniversalPageStorage.cpp @@ -420,7 +420,7 @@ bool UniversalPageStorage::canSkipCheckpoint() const return snap->sequence == last_checkpoint_sequence; } -PS::V3::CPDataWriteStats UniversalPageStorage::dumpIncrementalCheckpoint(const UniversalPageStorage::DumpCheckpointOptions & options) +PS::V3::CPDataDumpStats UniversalPageStorage::dumpIncrementalCheckpoint(const UniversalPageStorage::DumpCheckpointOptions & options) { std::scoped_lock lock(checkpoint_mu); Stopwatch sw; @@ -508,23 +508,25 @@ PS::V3::CPDataWriteStats UniversalPageStorage::dumpIncrementalCheckpoint(const U // New checkpoint infos are attached in `writeEditsAndApplyCheckpointInfo`. page_directory->copyCheckpointInfoFromEdit(edit_from_mem); } + auto copy_checkpoint_info_seconds = sw.elapsedMillisecondsFromLastTime() / 1000.0; last_checkpoint_sequence = snap->sequence; GET_METRIC(tiflash_storage_checkpoint_seconds, type_dump_checkpoint_snapshot).Observe(dump_snapshot_seconds); GET_METRIC(tiflash_storage_checkpoint_seconds, type_dump_checkpoint_data).Observe(dump_data_seconds); GET_METRIC(tiflash_storage_checkpoint_seconds, type_upload_checkpoint).Observe(upload_seconds); - LOG_DEBUG(log, - "Checkpoint result: files={}, dump_snapshot={:.3f}s, dump_data={:.3f}s, upload={:.3f}s, " - "total={:.3f}s, sequence={}, incremental_data_bytes={}, compact_data_bytes={}", - data_file_paths, - dump_snapshot_seconds, - dump_data_seconds, - upload_seconds, - sw.elapsedSeconds(), - sequence, - write_stats.incremental_data_bytes, - write_stats.compact_data_bytes); + GET_METRIC(tiflash_storage_checkpoint_seconds, type_copy_checkpoint_info).Observe(copy_checkpoint_info_seconds); + LOG_INFO(log, + "Checkpoint result: files={} dump_snapshot={:.3f}s dump_data={:.3f}s upload={:.3f}s copy_checkpoint_info={:.3f}s " + "total={:.3f}s sequence={} {}", + data_file_paths, + dump_snapshot_seconds, + dump_data_seconds, + upload_seconds, + copy_checkpoint_info_seconds, + sw.elapsedSeconds(), + sequence, + write_stats); return write_stats; } diff --git a/dbms/src/Storages/Page/V3/Universal/UniversalPageStorage.h b/dbms/src/Storages/Page/V3/Universal/UniversalPageStorage.h index 75eddfef87e..d720b1ff008 100644 --- a/dbms/src/Storages/Page/V3/Universal/UniversalPageStorage.h +++ b/dbms/src/Storages/Page/V3/Universal/UniversalPageStorage.h @@ -25,6 +25,7 @@ #include #include #include +#include #include #include #include @@ -198,7 +199,7 @@ class UniversalPageStorage final UInt64 max_edit_records_per_part = 100000; }; - PS::V3::CPDataWriteStats dumpIncrementalCheckpoint(const DumpCheckpointOptions & options); + PS::V3::CPDataDumpStats dumpIncrementalCheckpoint(const DumpCheckpointOptions & options); PS::V3::CPDataFilesStatCache::CacheMap getRemoteDataFilesStatCache() const {