You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pegasus.apache.org by wa...@apache.org on 2023/06/05 04:35:10 UTC

[incubator-pegasus] 32/32: feat(new_metrics): migrate server-level metrics of rocksdb (#1506)

This is an automated email from the ASF dual-hosted git repository.

wangdan pushed a commit to branch migrate-metrics-dev
in repository https://gitbox.apache.org/repos/asf/incubator-pegasus.git

commit cfcb7bfad8f4b69687c10887db23e2719e4d77dd
Author: Dan Wang <wa...@apache.org>
AuthorDate: Fri Jun 2 22:00:53 2023 +0800

    feat(new_metrics): migrate server-level metrics of rocksdb (#1506)
    
    https://github.com/apache/incubator-pegasus/issues/1505
    
    All of the server-level metrics of rocksdb are migrated to the new framework,
    including 2 metrics: the memory usage of block cache, and the through bytes
    per second that go through the rate limiter which takes control of the write rate
    of flush and compaction.
    
    This migration are somewhat different from previous ones:
    
    - firstly, both metrics are declared as static variables;
    - then, both variables need to be defined globally;
    - also, both metrics have to be initialized by assignment operator, rather than in
       the member initializer lists of the constructor of user class.
    
    To support all of these new features, some new macros are introduced.
---
 src/server/pegasus_server_impl.cpp      | 13 +++-------
 src/server/pegasus_server_impl.h        |  8 +++---
 src/server/pegasus_server_impl_init.cpp | 34 ++++++++++++------------
 src/utils/metrics.h                     | 46 ++++++++++++++++++++++++---------
 4 files changed, 57 insertions(+), 44 deletions(-)

diff --git a/src/server/pegasus_server_impl.cpp b/src/server/pegasus_server_impl.cpp
index c8d3ed601..3343ffb9d 100644
--- a/src/server/pegasus_server_impl.cpp
+++ b/src/server/pegasus_server_impl.cpp
@@ -56,8 +56,6 @@
 #include "pegasus_const.h"
 #include "pegasus_rpc_types.h"
 #include "pegasus_server_write.h"
-#include "perf_counter/perf_counter.h"
-#include "perf_counter/perf_counter_wrapper.h"
 #include "replica_admin_types.h"
 #include "rrdb/rrdb.code.definition.h"
 #include "rrdb/rrdb_types.h"
@@ -139,8 +137,8 @@ int64_t pegasus_server_impl::_rocksdb_limiter_last_total_through;
 std::shared_ptr<rocksdb::Cache> pegasus_server_impl::_s_block_cache;
 std::shared_ptr<rocksdb::WriteBufferManager> pegasus_server_impl::_s_write_buffer_manager;
 ::dsn::task_ptr pegasus_server_impl::_update_server_rdb_stat;
-::dsn::perf_counter_wrapper pegasus_server_impl::_pfc_rdb_block_cache_mem_usage;
-::dsn::perf_counter_wrapper pegasus_server_impl::_pfc_rdb_write_limiter_rate_bytes;
+METRIC_VAR_DEFINE_gauge_int64(rdb_block_cache_mem_usage_bytes, pegasus_server_impl);
+METRIC_VAR_DEFINE_gauge_int64(rdb_write_rate_limiter_through_bytes_per_sec, pegasus_server_impl);
 const std::string pegasus_server_impl::COMPRESSION_HEADER = "per_level:";
 const std::string pegasus_server_impl::DATA_COLUMN_FAMILY_NAME = "default";
 const std::string pegasus_server_impl::META_COLUMN_FAMILY_NAME = "pegasus_meta_cf";
@@ -1865,7 +1863,6 @@ void pegasus_server_impl::cancel_background_work(bool wait)
         METRIC_VAR_SET(rdb_memtable_mem_usage_bytes, 0);
         METRIC_VAR_SET(rdb_block_cache_hit_count, 0);
         METRIC_VAR_SET(rdb_block_cache_total_count, 0);
-        _pfc_rdb_block_cache_mem_usage->set(0);
     }
 
     LOG_INFO_PREFIX("close app succeed, clear_state = {}", clear_state ? "true" : "false");
@@ -2533,19 +2530,17 @@ void pegasus_server_impl::update_replica_rocksdb_statistics()
 
 void pegasus_server_impl::update_server_rocksdb_statistics()
 {
-    // Update _pfc_rdb_block_cache_mem_usage
     if (_s_block_cache) {
         uint64_t val = _s_block_cache->GetUsage();
-        _pfc_rdb_block_cache_mem_usage->set(val);
+        METRIC_VAR_SET(rdb_block_cache_mem_usage_bytes, val);
     }
 
-    // Update _pfc_rdb_write_limiter_rate_bytes
     if (_s_rate_limiter) {
         uint64_t current_total_through = _s_rate_limiter->GetTotalBytesThrough();
         uint64_t through_bytes_per_sec =
             (current_total_through - _rocksdb_limiter_last_total_through) /
             kServerStatUpdateTimeSec.count();
-        _pfc_rdb_write_limiter_rate_bytes->set(through_bytes_per_sec);
+        METRIC_VAR_SET(rdb_write_rate_limiter_through_bytes_per_sec, through_bytes_per_sec);
         _rocksdb_limiter_last_total_through = current_total_through;
     }
 }
diff --git a/src/server/pegasus_server_impl.h b/src/server/pegasus_server_impl.h
index 6a7817017..04aff8149 100644
--- a/src/server/pegasus_server_impl.h
+++ b/src/server/pegasus_server_impl.h
@@ -69,7 +69,6 @@ class WriteBufferManager;
 namespace dsn {
 class blob;
 class message_ex;
-class perf_counter_wrapper;
 class rpc_address;
 
 namespace replication {
@@ -544,10 +543,9 @@ private:
     METRIC_VAR_DECLARE_counter(abnormal_read_requests);
     METRIC_VAR_DECLARE_counter(throttling_rejected_read_requests);
 
-    // rocksdb internal statistics
-    // server level
-    static ::dsn::perf_counter_wrapper _pfc_rdb_write_limiter_rate_bytes;
-    static ::dsn::perf_counter_wrapper _pfc_rdb_block_cache_mem_usage;
+    // Server-level metrics for rocksdb.
+    METRIC_VAR_DECLARE_gauge_int64(rdb_block_cache_mem_usage_bytes, static);
+    METRIC_VAR_DECLARE_gauge_int64(rdb_write_rate_limiter_through_bytes_per_sec, static);
 
     // Replica-level metrics for rocksdb.
     METRIC_VAR_DECLARE_gauge_int64(rdb_total_sst_files);
diff --git a/src/server/pegasus_server_impl_init.cpp b/src/server/pegasus_server_impl_init.cpp
index cd51f0575..c04636514 100644
--- a/src/server/pegasus_server_impl_init.cpp
+++ b/src/server/pegasus_server_impl_init.cpp
@@ -40,8 +40,6 @@
 #include "pegasus_event_listener.h"
 #include "pegasus_server_impl.h"
 #include "pegasus_value_schema.h"
-#include "perf_counter/perf_counter.h"
-#include "perf_counter/perf_counter_wrapper.h"
 #include "replica_admin_types.h"
 #include "runtime/api_layer1.h"
 #include "runtime/rpc/rpc_address.h"
@@ -129,7 +127,7 @@ METRIC_DEFINE_gauge_int64(replica,
 METRIC_DEFINE_gauge_int64(replica,
                           rdb_total_sst_size_mb,
                           dsn::metric_unit::kMegaBytes,
-                          "The total size of rocksdb sst files in MB");
+                          "The total size of rocksdb sst files");
 
 METRIC_DEFINE_gauge_int64(replica,
                           rdb_estimated_keys,
@@ -139,12 +137,12 @@ METRIC_DEFINE_gauge_int64(replica,
 METRIC_DEFINE_gauge_int64(replica,
                           rdb_index_and_filter_blocks_mem_usage_bytes,
                           dsn::metric_unit::kBytes,
-                          "The memory usage of rocksdb index and filter blocks in bytes");
+                          "The memory usage of rocksdb index and filter blocks");
 
 METRIC_DEFINE_gauge_int64(replica,
                           rdb_memtable_mem_usage_bytes,
                           dsn::metric_unit::kBytes,
-                          "The memory usage of rocksdb memtables in bytes");
+                          "The memory usage of rocksdb memtables");
 
 METRIC_DEFINE_gauge_int64(replica,
                           rdb_block_cache_hit_count,
@@ -238,6 +236,17 @@ METRIC_DEFINE_gauge_int64(replica,
                           "The number of times full bloom filter has not avoided the reads and "
                           "data actually exist, used by rocksdb");
 
+METRIC_DEFINE_gauge_int64(server,
+                          rdb_block_cache_mem_usage_bytes,
+                          dsn::metric_unit::kBytes,
+                          "The memory usage of rocksdb block cache");
+
+METRIC_DEFINE_gauge_int64(server,
+                          rdb_write_rate_limiter_through_bytes_per_sec,
+                          dsn::metric_unit::kBytesPerSec,
+                          "The through bytes per second that go through the rate limiter which "
+                          "takes control of the write rate of flush and compaction of rocksdb");
+
 namespace pegasus {
 namespace server {
 
@@ -820,19 +829,8 @@ pegasus_server_impl::pegasus_server_impl(dsn::replication::replica *r)
     // them only once.
     static std::once_flag flag;
     std::call_once(flag, [&]() {
-        _pfc_rdb_block_cache_mem_usage.init_global_counter(
-            "replica",
-            "app.pegasus",
-            "rdb.block_cache.memory_usage",
-            COUNTER_TYPE_NUMBER,
-            "statistic the memory usage of rocksdb block cache");
-
-        _pfc_rdb_write_limiter_rate_bytes.init_global_counter(
-            "replica",
-            "app.pegasus",
-            "rdb.write_limiter_rate_bytes",
-            COUNTER_TYPE_NUMBER,
-            "statistic the through bytes of rocksdb write rate limiter");
+        METRIC_VAR_ASSIGN_server(rdb_block_cache_mem_usage_bytes);
+        METRIC_VAR_ASSIGN_server(rdb_write_rate_limiter_through_bytes_per_sec);
     });
 }
 
diff --git a/src/utils/metrics.h b/src/utils/metrics.h
index 11f34b56c..2ecc4d305 100644
--- a/src/utils/metrics.h
+++ b/src/utils/metrics.h
@@ -101,9 +101,9 @@ class error_code;
         {#entity_type, dsn::metric_type::kGauge, #name, unit, desc, ##__VA_ARGS__})
 
 // There are 2 kinds of counters:
-// - `counter` is the general type of counter that is implemented by striped_long_adder, which can
+// * `counter` is the general type of counter that is implemented by striped_long_adder, which can
 //   achieve high performance while consuming less memory if it's not updated very frequently.
-// - `concurrent_counter` uses concurrent_long_adder as the underlying implementation. It has
+// * `concurrent_counter` uses concurrent_long_adder as the underlying implementation. It has
 //   higher performance while consuming more memory if it's updated very frequently.
 // See also include/dsn/utility/long_adder.h for details.
 #define METRIC_DEFINE_counter(entity_type, name, unit, desc, ...)                                  \
@@ -144,7 +144,7 @@ class error_code;
 #define METRIC_DECLARE_percentile_double(name)                                                     \
     extern dsn::floating_percentile_prototype<double> METRIC_##name
 
-// Following METRIC_*VAR* macros are introduced so that:
+// Following METRIC_VAR* macros are introduced so that:
 // * only need to use prototype name to operate each metric variable;
 // * uniformly name each variable in user class;
 // * differentiate operations on metrics significantly from main logic, improving code readability.
@@ -155,16 +155,37 @@ class error_code;
 // instead of a single fixed argument to represent a type.
 #define METRIC_VAR_NAME(name) _metric_##name
 #define METRIC_VAR_DECLARE(name, ...) __VA_ARGS__ METRIC_VAR_NAME(name)
-#define METRIC_VAR_DECLARE_gauge_int64(name) METRIC_VAR_DECLARE(name, dsn::gauge_ptr<int64_t>)
-#define METRIC_VAR_DECLARE_counter(name)                                                           \
-    METRIC_VAR_DECLARE(name, dsn::counter_ptr<dsn::striped_long_adder, false>)
-#define METRIC_VAR_DECLARE_percentile_int64(name)                                                  \
-    METRIC_VAR_DECLARE(name, dsn::percentile_ptr<int64_t>)
-
-// Initialize a metric variable in user class.
-#define METRIC_VAR_INIT(name, entity, ...)                                                         \
-    METRIC_VAR_NAME(name)(METRIC_##name.instantiate(entity##_metric_entity(), ##__VA_ARGS__))
+
+// Variadic arguments are possible qualifiers for the variable, such as `static`.
+#define METRIC_VAR_DECLARE_gauge_int64(name, ...)                                                  \
+    METRIC_VAR_DECLARE(name, __VA_ARGS__ dsn::gauge_ptr<int64_t>)
+#define METRIC_VAR_DECLARE_counter(name, ...)                                                      \
+    METRIC_VAR_DECLARE(name, __VA_ARGS__ dsn::counter_ptr<dsn::striped_long_adder, false>)
+#define METRIC_VAR_DECLARE_percentile_int64(name, ...)                                             \
+    METRIC_VAR_DECLARE(name, __VA_ARGS__ dsn::percentile_ptr<int64_t>)
+
+// Macro METRIC_VAR_DEFINE* are used for the metric that is a static member of a class:
+// * `clazz` is the name of the class;
+// * variadic arguments are possible qualifiers for the variable.
+#define METRIC_VAR_DEFINE(name, clazz, ...) __VA_ARGS__ clazz::METRIC_VAR_NAME(name)
+#define METRIC_VAR_DEFINE_gauge_int64(name, clazz, ...)                                            \
+    METRIC_VAR_DEFINE(name, clazz, __VA_ARGS__ dsn::gauge_ptr<int64_t>)
+#define METRIC_VAR_DEFINE_counter(name, clazz, ...)                                                \
+    METRIC_VAR_DEFINE(name, clazz, __VA_ARGS__ dsn::counter_ptr<dsn::striped_long_adder, false>)
+#define METRIC_VAR_DEFINE_percentile_int64(name, clazz, ...)                                       \
+    METRIC_VAR_DEFINE(name, clazz, __VA_ARGS__ dsn::percentile_ptr<int64_t>)
+
+// Initialize a metric variable in user class:
+// * macros METRIC_VAR_INIT* could be used to initialize metric variables in member initializer
+//   lists of the constructor of user class;
+// * macros METRIC_VAR_ASSIGN* could be used to initialize metric variables by assignment operator
+//   (=).
+#define METRIC_VAR_INSTANTIATE(name, entity, op, ...)                                              \
+    METRIC_VAR_NAME(name) op(METRIC_##name.instantiate(entity##_metric_entity(), ##__VA_ARGS__))
+#define METRIC_VAR_ASSIGN(name, entity, ...) METRIC_VAR_INSTANTIATE(name, entity, =, ##__VA_ARGS__)
+#define METRIC_VAR_INIT(name, entity, ...) METRIC_VAR_INSTANTIATE(name, entity, , ##__VA_ARGS__)
 #define METRIC_VAR_INIT_replica(name, ...) METRIC_VAR_INIT(name, replica, ##__VA_ARGS__)
+#define METRIC_VAR_ASSIGN_server(name, ...) METRIC_VAR_ASSIGN(name, server, ##__VA_ARGS__)
 #define METRIC_VAR_INIT_server(name, ...) METRIC_VAR_INIT(name, server, ##__VA_ARGS__)
 #define METRIC_VAR_INIT_disk(name, ...) METRIC_VAR_INIT(name, disk, ##__VA_ARGS__)
 #define METRIC_VAR_INIT_table(name, ...) METRIC_VAR_INIT(name, table, ##__VA_ARGS__)
@@ -668,6 +689,7 @@ ENUM_END(metric_type)
     DEF(Seconds)                                                                                   \
     DEF(Bytes)                                                                                     \
     DEF(MegaBytes)                                                                                 \
+    DEF(BytesPerSec)                                                                               \
     DEF(CapacityUnits)                                                                             \
     DEF(Percent)                                                                                   \
     DEF(Replicas)                                                                                  \


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pegasus.apache.org
For additional commands, e-mail: commits-help@pegasus.apache.org