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

yihua pushed a commit to branch branch-0.x
in repository https://gitbox.apache.org/repos/asf/hudi.git

commit 027575259934e798ba7a9605886f9e4e2828f41b
Author: Lokesh Jain <[email protected]>
AuthorDate: Thu Apr 11 13:32:02 2024 +0530

    [HUDI-7391] HoodieMetadataMetrics should use Metrics instance for metrics 
registry (#10635)
    
    Currently HoodieMetadataMetrics stores metrics in memory and these metrics 
are not pushed by the metric reporters. The metric reporters are configured 
within Metrics instance. List of changes in the PR:
    
    Metrics related classes have been moved from hudi-client-common to 
hudi-common.
    HoodieMetadataMetrics now uses Metrics class so that all the reporters can 
be supported with it.
    Some gaps in configs which are added in HoodieMetadataWriteUtils
    Some metrics related apis and functionality has been moved to 
HoodieMetricsConfig. The HoodieWriteConfig APIs now delegate to 
HoodieMetricsConfig for the functionality.
---
 hudi-client/hudi-client-common/pom.xml             |  46 -----
 .../lock/metrics/HoodieLockMetrics.java            |   2 +-
 .../org/apache/hudi/config/HoodieWriteConfig.java  |  98 +++++-----
 .../hudi/metadata/HoodieMetadataWriteUtils.java    |   9 +-
 .../org/apache/hudi/metrics/HoodieMetrics.java     |   2 +-
 .../cloudwatch/CloudWatchMetricsReporter.java      |  29 ++-
 .../table/action/index/RunIndexActionExecutor.java |   3 +-
 .../hudi/metrics/TestHoodieConsoleMetrics.java     |  16 +-
 .../hudi/metrics/TestHoodieGraphiteMetrics.java    |  22 ++-
 .../apache/hudi/metrics/TestHoodieJmxMetrics.java  |  19 +-
 .../org/apache/hudi/metrics/TestHoodieMetrics.java |  17 +-
 .../hudi/metrics/TestMetricsReporterFactory.java   |  20 +-
 .../cloudwatch/TestCloudWatchMetricsReporter.java  |  27 ++-
 .../datadog/TestDatadogMetricsReporter.java        |  60 +++---
 .../org/apache/hudi/metrics/m3/TestM3Metrics.java  |  54 +++---
 .../metrics/prometheus/TestPrometheusReporter.java |  19 +-
 .../prometheus/TestPushGateWayReporter.java        |  52 +++---
 .../FlinkHoodieBackedTableMetadataWriter.java      |   4 +-
 .../JavaHoodieBackedTableMetadataWriter.java       |   4 +-
 .../hudi/client/TestJavaHoodieBackedMetadata.java  |  21 ++-
 .../SparkHoodieBackedTableMetadataWriter.java      |   2 +-
 .../functional/TestHoodieBackedMetadata.java       |  18 +-
 hudi-common/pom.xml                                |  47 +++++
 .../metrics/HoodieMetricsCloudWatchConfig.java     |   0
 .../hudi/config/metrics/HoodieMetricsConfig.java   | 201 +++++++++++++++++++++
 .../config/metrics/HoodieMetricsDatadogConfig.java |   0
 .../metrics/HoodieMetricsGraphiteConfig.java       |   0
 .../config/metrics/HoodieMetricsJmxConfig.java     |   0
 .../hudi/config/metrics/HoodieMetricsM3Config.java |   0
 .../metrics/HoodieMetricsPrometheusConfig.java     |   0
 .../apache/hudi/metadata/BaseTableMetadata.java    |   4 +-
 .../hudi/metadata/HoodieMetadataMetrics.java       |  21 ++-
 .../hudi/metrics/ConsoleMetricsReporter.java       |   0
 .../java/org/apache/hudi/metrics/HoodieGauge.java  |   0
 .../hudi/metrics/InMemoryMetricsReporter.java      |   0
 .../apache/hudi/metrics/JmxMetricsReporter.java    |   4 +-
 .../org/apache/hudi/metrics/JmxReporterServer.java |   0
 .../java/org/apache/hudi/metrics/MetricUtils.java  |   0
 .../main/java/org/apache/hudi/metrics/Metrics.java |  43 +++--
 .../hudi/metrics/MetricsGraphiteReporter.java      |  16 +-
 .../org/apache/hudi/metrics/MetricsReporter.java   |   0
 .../hudi/metrics/MetricsReporterFactory.java       |  27 ++-
 .../apache/hudi/metrics/MetricsReporterType.java   |   0
 .../custom/CustomizableMetricsReporter.java        |   0
 .../hudi/metrics/datadog/DatadogHttpClient.java    |   0
 .../metrics/datadog/DatadogMetricsReporter.java    |   4 +-
 .../hudi/metrics/datadog/DatadogReporter.java      |   0
 .../apache/hudi/metrics/m3/M3MetricsReporter.java  |  16 +-
 .../hudi/metrics/m3/M3ScopeReporterAdaptor.java    |   0
 .../metrics/prometheus/PrometheusReporter.java     |  10 +-
 .../prometheus/PushGatewayMetricsReporter.java     |  18 +-
 .../metrics/prometheus/PushGatewayReporter.java    |   0
 .../AbstractUserDefinedMetricsReporter.java        |   0
 .../deltastreamer/HoodieDeltaStreamerMetrics.java  |   8 +-
 .../ingestion/HoodieIngestionMetrics.java          |   7 +-
 .../utilities/streamer/HoodieStreamerMetrics.java  |   5 +
 .../apache/hudi/utilities/streamer/StreamSync.java |   2 +-
 57 files changed, 642 insertions(+), 335 deletions(-)

diff --git a/hudi-client/hudi-client-common/pom.xml 
b/hudi-client/hudi-client-common/pom.xml
index 6caccd0b0a6..022f5d6faa0 100644
--- a/hudi-client/hudi-client-common/pom.xml
+++ b/hudi-client/hudi-client-common/pom.xml
@@ -85,52 +85,6 @@
       <version>0.2.2</version>
     </dependency>
 
-    <!-- Dropwizard Metrics -->
-    <dependency>
-      <groupId>io.dropwizard.metrics</groupId>
-      <artifactId>metrics-graphite</artifactId>
-      <exclusions>
-        <exclusion>
-          <groupId>com.rabbitmq</groupId>
-          <artifactId>*</artifactId>
-        </exclusion>
-      </exclusions>
-    </dependency>
-    <dependency>
-      <groupId>io.dropwizard.metrics</groupId>
-      <artifactId>metrics-core</artifactId>
-    </dependency>
-    <dependency>
-      <groupId>io.dropwizard.metrics</groupId>
-      <artifactId>metrics-jmx</artifactId>
-    </dependency>
-    <dependency>
-      <groupId>io.prometheus</groupId>
-      <artifactId>simpleclient</artifactId>
-    </dependency>
-    <dependency>
-      <groupId>io.prometheus</groupId>
-      <artifactId>simpleclient_httpserver</artifactId>
-    </dependency>
-    <dependency>
-      <groupId>io.prometheus</groupId>
-      <artifactId>simpleclient_dropwizard</artifactId>
-    </dependency>
-    <dependency>
-      <groupId>io.prometheus</groupId>
-      <artifactId>simpleclient_pushgateway</artifactId>
-    </dependency>
-    <dependency>
-      <groupId>com.uber.m3</groupId>
-      <artifactId>tally-m3</artifactId>
-      <version>${tally.version}</version>
-    </dependency>
-    <dependency>
-      <groupId>com.uber.m3</groupId>
-      <artifactId>tally-core</artifactId>
-      <version>${tally.version}</version>
-    </dependency>
-
     <!-- Lock -->
     <dependency>
       <groupId>org.apache.curator</groupId>
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/client/transaction/lock/metrics/HoodieLockMetrics.java
 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/client/transaction/lock/metrics/HoodieLockMetrics.java
index 25603d5655c..bbf3d6876d8 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/client/transaction/lock/metrics/HoodieLockMetrics.java
+++ 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/client/transaction/lock/metrics/HoodieLockMetrics.java
@@ -54,7 +54,7 @@ public class HoodieLockMetrics {
     this.writeConfig = writeConfig;
 
     if (isMetricsEnabled) {
-      metrics = Metrics.getInstance(writeConfig);
+      metrics = Metrics.getInstance(writeConfig.getMetricsConfig());
       MetricRegistry registry = metrics.getRegistry();
 
       lockAttempts = 
registry.counter(getMetricsName(LOCK_ACQUIRE_ATTEMPTS_COUNTER_NAME));
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/HoodieWriteConfig.java
 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/HoodieWriteConfig.java
index 558aba5b17b..e8f327faecb 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/HoodieWriteConfig.java
+++ 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/HoodieWriteConfig.java
@@ -57,13 +57,10 @@ import org.apache.hudi.common.util.StringUtils;
 import org.apache.hudi.common.util.VisibleForTesting;
 import org.apache.hudi.common.util.queue.DisruptorWaitStrategyType;
 import org.apache.hudi.common.util.queue.ExecutorType;
-import org.apache.hudi.config.metrics.HoodieMetricsCloudWatchConfig;
 import org.apache.hudi.config.metrics.HoodieMetricsConfig;
-import org.apache.hudi.config.metrics.HoodieMetricsDatadogConfig;
 import org.apache.hudi.config.metrics.HoodieMetricsGraphiteConfig;
 import org.apache.hudi.config.metrics.HoodieMetricsJmxConfig;
 import org.apache.hudi.config.metrics.HoodieMetricsM3Config;
-import org.apache.hudi.config.metrics.HoodieMetricsPrometheusConfig;
 import org.apache.hudi.exception.HoodieNotSupportedException;
 import org.apache.hudi.execution.bulkinsert.BulkInsertSortMode;
 import org.apache.hudi.index.HoodieIndex;
@@ -99,7 +96,6 @@ import java.util.Locale;
 import java.util.Map;
 import java.util.Objects;
 import java.util.Properties;
-import java.util.function.Supplier;
 import java.util.stream.Collectors;
 
 import static org.apache.hudi.common.util.ValidationUtils.checkArgument;
@@ -766,6 +762,7 @@ public class HoodieWriteConfig extends HoodieConfig {
   private FileSystemViewStorageConfig viewStorageConfig;
   private HoodiePayloadConfig hoodiePayloadConfig;
   private HoodieMetadataConfig metadataConfig;
+  private HoodieMetricsConfig metricsConfig;
   private HoodieMetaserverConfig metaserverConfig;
   private HoodieTableServiceManagerConfig tableServiceManagerConfig;
   private HoodieCommonConfig commonConfig;
@@ -1160,6 +1157,7 @@ public class HoodieWriteConfig extends HoodieConfig {
     this.viewStorageConfig = clientSpecifiedViewStorageConfig;
     this.hoodiePayloadConfig = 
HoodiePayloadConfig.newBuilder().fromProperties(newProps).build();
     this.metadataConfig = 
HoodieMetadataConfig.newBuilder().fromProperties(props).build();
+    this.metricsConfig = 
HoodieMetricsConfig.newBuilder().fromProperties(props).build();
     this.metaserverConfig = 
HoodieMetaserverConfig.newBuilder().fromProperties(props).build();
     this.tableServiceManagerConfig = 
HoodieTableServiceManagerConfig.newBuilder().fromProperties(props).build();
     this.commonConfig = 
HoodieCommonConfig.newBuilder().fromProperties(props).build();
@@ -2140,172 +2138,162 @@ public class HoodieWriteConfig extends HoodieConfig {
    * metrics properties.
    */
   public boolean isMetricsOn() {
-    return getBoolean(HoodieMetricsConfig.TURN_METRICS_ON);
+    return metricsConfig.isMetricsOn();
   }
 
   /**
    * metrics properties.
    */
   public boolean isCompactionLogBlockMetricsOn() {
-    return 
getBoolean(HoodieMetricsConfig.TURN_METRICS_COMPACTION_LOG_BLOCKS_ON);
+    return metricsConfig.isCompactionLogBlockMetricsOn();
   }
 
   public boolean isExecutorMetricsEnabled() {
-    return Boolean.parseBoolean(
-        getStringOrDefault(HoodieMetricsConfig.EXECUTOR_METRICS_ENABLE, 
"false"));
+    return metricsConfig.isExecutorMetricsEnabled();
   }
 
   public boolean isLockingMetricsEnabled() {
-    return getBoolean(HoodieMetricsConfig.LOCK_METRICS_ENABLE);
+    return metricsConfig.isLockingMetricsEnabled();
   }
 
   public MetricsReporterType getMetricsReporterType() {
-    return 
MetricsReporterType.valueOf(getString(HoodieMetricsConfig.METRICS_REPORTER_TYPE_VALUE));
+    return metricsConfig.getMetricsReporterType();
   }
 
   public String getGraphiteServerHost() {
-    return getString(HoodieMetricsGraphiteConfig.GRAPHITE_SERVER_HOST_NAME);
+    return metricsConfig.getGraphiteServerHost();
   }
 
   public int getGraphiteServerPort() {
-    return getInt(HoodieMetricsGraphiteConfig.GRAPHITE_SERVER_PORT_NUM);
+    return metricsConfig.getGraphiteServerPort();
   }
 
   public String getGraphiteMetricPrefix() {
-    return getString(HoodieMetricsGraphiteConfig.GRAPHITE_METRIC_PREFIX_VALUE);
+    return metricsConfig.getGraphiteMetricPrefix();
   }
 
   public int getGraphiteReportPeriodSeconds() {
-    return 
getInt(HoodieMetricsGraphiteConfig.GRAPHITE_REPORT_PERIOD_IN_SECONDS);
+    return metricsConfig.getGraphiteReportPeriodSeconds();
   }
 
   public String getM3ServerHost() {
-    return getString(HoodieMetricsM3Config.M3_SERVER_HOST_NAME);
+    return metricsConfig.getM3ServerHost();
   }
 
   public int getM3ServerPort() {
-    return getInt(HoodieMetricsM3Config.M3_SERVER_PORT_NUM);
+    return metricsConfig.getM3ServerPort();
   }
 
   public String getM3Tags() {
-    return getString(HoodieMetricsM3Config.M3_TAGS);
+    return metricsConfig.getM3Tags();
   }
 
   public String getM3Env() {
-    return getString(HoodieMetricsM3Config.M3_ENV);
+    return metricsConfig.getM3Env();
   }
 
   public String getM3Service() {
-    return getString(HoodieMetricsM3Config.M3_SERVICE);
+    return metricsConfig.getM3Service();
   }
 
   public String getJmxHost() {
-    return getString(HoodieMetricsJmxConfig.JMX_HOST_NAME);
+    return metricsConfig.getJmxHost();
   }
 
   public String getJmxPort() {
-    return getString(HoodieMetricsJmxConfig.JMX_PORT_NUM);
+    return metricsConfig.getJmxPort();
   }
 
   public int getDatadogReportPeriodSeconds() {
-    return getInt(HoodieMetricsDatadogConfig.REPORT_PERIOD_IN_SECONDS);
+    return metricsConfig.getDatadogReportPeriodSeconds();
   }
 
   public ApiSite getDatadogApiSite() {
-    return 
ApiSite.valueOf(getString(HoodieMetricsDatadogConfig.API_SITE_VALUE));
+    return metricsConfig.getDatadogApiSite();
   }
 
   public String getDatadogApiKey() {
-    if (props.containsKey(HoodieMetricsDatadogConfig.API_KEY.key())) {
-      return getString(HoodieMetricsDatadogConfig.API_KEY);
-
-    } else {
-      Supplier<String> apiKeySupplier = ReflectionUtils.loadClass(
-          getString(HoodieMetricsDatadogConfig.API_KEY_SUPPLIER));
-      return apiKeySupplier.get();
-    }
+    return metricsConfig.getDatadogApiKey();
   }
 
   public boolean getDatadogApiKeySkipValidation() {
-    return getBoolean(HoodieMetricsDatadogConfig.API_KEY_SKIP_VALIDATION);
+    return metricsConfig.getDatadogApiKeySkipValidation();
   }
 
   public int getDatadogApiTimeoutSeconds() {
-    return getInt(HoodieMetricsDatadogConfig.API_TIMEOUT_IN_SECONDS);
+    return metricsConfig.getDatadogApiTimeoutSeconds();
   }
 
   public String getDatadogMetricPrefix() {
-    return getString(HoodieMetricsDatadogConfig.METRIC_PREFIX_VALUE);
+    return metricsConfig.getDatadogMetricPrefix();
   }
 
   public String getDatadogMetricHost() {
-    return getString(HoodieMetricsDatadogConfig.METRIC_HOST_NAME);
+    return metricsConfig.getDatadogMetricHost();
   }
 
   public List<String> getDatadogMetricTags() {
-    return Arrays.stream(getStringOrDefault(
-        HoodieMetricsDatadogConfig.METRIC_TAG_VALUES, 
",").split("\\s*,\\s*")).collect(Collectors.toList());
+    return metricsConfig.getDatadogMetricTags();
   }
 
   public int getCloudWatchReportPeriodSeconds() {
-    return getInt(HoodieMetricsCloudWatchConfig.REPORT_PERIOD_SECONDS);
+    return metricsConfig.getCloudWatchReportPeriodSeconds();
   }
 
   public String getCloudWatchMetricPrefix() {
-    return getString(HoodieMetricsCloudWatchConfig.METRIC_PREFIX);
+    return metricsConfig.getCloudWatchMetricPrefix();
   }
 
   public String getCloudWatchMetricNamespace() {
-    return getString(HoodieMetricsCloudWatchConfig.METRIC_NAMESPACE);
+    return metricsConfig.getCloudWatchMetricNamespace();
   }
 
   public int getCloudWatchMaxDatumsPerRequest() {
-    return getInt(HoodieMetricsCloudWatchConfig.MAX_DATUMS_PER_REQUEST);
+    return metricsConfig.getCloudWatchMaxDatumsPerRequest();
   }
 
   public String getMetricReporterClassName() {
-    return getString(HoodieMetricsConfig.METRICS_REPORTER_CLASS_NAME);
+    return metricsConfig.getMetricReporterClassName();
   }
 
   public int getPrometheusPort() {
-    return getInt(HoodieMetricsPrometheusConfig.PROMETHEUS_PORT_NUM);
+    return metricsConfig.getPrometheusPort();
   }
 
   public String getPushGatewayHost() {
-    return getString(HoodieMetricsPrometheusConfig.PUSHGATEWAY_HOST_NAME);
+    return metricsConfig.getPushGatewayHost();
   }
 
   public int getPushGatewayPort() {
-    return getInt(HoodieMetricsPrometheusConfig.PUSHGATEWAY_PORT_NUM);
+    return metricsConfig.getPushGatewayPort();
   }
 
   public int getPushGatewayReportPeriodSeconds() {
-    return 
getInt(HoodieMetricsPrometheusConfig.PUSHGATEWAY_REPORT_PERIOD_IN_SECONDS);
+    return metricsConfig.getPushGatewayReportPeriodSeconds();
   }
 
   public boolean getPushGatewayDeleteOnShutdown() {
-    return 
getBoolean(HoodieMetricsPrometheusConfig.PUSHGATEWAY_DELETE_ON_SHUTDOWN_ENABLE);
+    return metricsConfig.getPushGatewayDeleteOnShutdown();
   }
 
   public String getPushGatewayJobName() {
-    return getString(HoodieMetricsPrometheusConfig.PUSHGATEWAY_JOBNAME);
+    return metricsConfig.getPushGatewayJobName();
   }
 
   public String getPushGatewayLabels() {
-    return getString(HoodieMetricsPrometheusConfig.PUSHGATEWAY_LABELS);
+    return metricsConfig.getPushGatewayLabels();
   }
 
   public boolean getPushGatewayRandomJobNameSuffix() {
-    return 
getBoolean(HoodieMetricsPrometheusConfig.PUSHGATEWAY_RANDOM_JOBNAME_SUFFIX);
+    return metricsConfig.getPushGatewayRandomJobNameSuffix();
   }
 
   public String getMetricReporterMetricsNamePrefix() {
-    // Metrics prefixes should not have a dot as this is usually a separator
-    return 
getStringOrDefault(HoodieMetricsConfig.METRICS_REPORTER_PREFIX).replaceAll("\\.",
 "_");
+    return metricsConfig.getMetricReporterMetricsNamePrefix();
   }
 
   public String getMetricReporterFileBasedConfigs() {
-    return 
getStringOrDefault(HoodieMetricsConfig.METRICS_REPORTER_FILE_BASED_CONFIGS_PATH);
+    return metricsConfig.getMetricReporterFileBasedConfigs();
   }
 
   /**
@@ -2360,6 +2348,10 @@ public class HoodieWriteConfig extends HoodieConfig {
     return metadataConfig;
   }
 
+  public HoodieMetricsConfig getMetricsConfig() {
+    return metricsConfig;
+  }
+
   public HoodieTableServiceManagerConfig getTableServiceManagerConfig() {
     return tableServiceManagerConfig;
   }
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieMetadataWriteUtils.java
 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieMetadataWriteUtils.java
index 48cfb46b49f..dfad3b13c11 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieMetadataWriteUtils.java
+++ 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieMetadataWriteUtils.java
@@ -187,14 +187,14 @@ public class HoodieMetadataWriteUtils {
               .build());
           break;
         case PROMETHEUS_PUSHGATEWAY:
-          HoodieMetricsPrometheusConfig prometheusConfig = 
HoodieMetricsPrometheusConfig.newBuilder()
+          HoodieMetricsPrometheusConfig pushGatewayConfig = 
HoodieMetricsPrometheusConfig.newBuilder()
               .withPushgatewayJobname(writeConfig.getPushGatewayJobName())
               
.withPushgatewayRandomJobnameSuffix(writeConfig.getPushGatewayRandomJobNameSuffix())
               .withPushgatewayLabels(writeConfig.getPushGatewayLabels())
               
.withPushgatewayReportPeriodInSeconds(String.valueOf(writeConfig.getPushGatewayReportPeriodSeconds()))
               .withPushgatewayHostName(writeConfig.getPushGatewayHost())
               
.withPushgatewayPortNum(writeConfig.getPushGatewayPort()).build();
-          builder.withProperties(prometheusConfig.getProps());
+          builder.withProperties(pushGatewayConfig.getProps());
           break;
         case M3:
           HoodieMetricsM3Config m3Config = HoodieMetricsM3Config.newBuilder()
@@ -223,6 +223,11 @@ public class HoodieMetadataWriteUtils {
           builder.withProperties(datadogConfig.build().getProps());
           break;
         case PROMETHEUS:
+          HoodieMetricsPrometheusConfig prometheusConfig = 
HoodieMetricsPrometheusConfig.newBuilder()
+              .withPushgatewayLabels(writeConfig.getPushGatewayLabels())
+              .withPrometheusPortNum(writeConfig.getPrometheusPort()).build();
+          builder.withProperties(prometheusConfig.getProps());
+          break;
         case CONSOLE:
         case INMEMORY:
         case CLOUDWATCH:
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/HoodieMetrics.java
 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/HoodieMetrics.java
index feca84a5e73..efb9be2414b 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/HoodieMetrics.java
+++ 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/HoodieMetrics.java
@@ -92,7 +92,7 @@ public class HoodieMetrics {
     this.config = config;
     this.tableName = config.getTableName();
     if (config.isMetricsOn()) {
-      metrics = Metrics.getInstance(config);
+      metrics = Metrics.getInstance(config.getMetricsConfig());
       this.rollbackTimerName = getMetricsName("timer", 
HoodieTimeline.ROLLBACK_ACTION);
       this.cleanTimerName = getMetricsName("timer", 
HoodieTimeline.CLEAN_ACTION);
       this.commitTimerName = getMetricsName("timer", 
HoodieTimeline.COMMIT_ACTION);
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/cloudwatch/CloudWatchMetricsReporter.java
 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/cloudwatch/CloudWatchMetricsReporter.java
index d05632b9bbf..68e4951f74f 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/cloudwatch/CloudWatchMetricsReporter.java
+++ 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/cloudwatch/CloudWatchMetricsReporter.java
@@ -20,6 +20,7 @@ package org.apache.hudi.metrics.cloudwatch;
 
 import org.apache.hudi.aws.cloudwatch.CloudWatchReporter;
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.MetricsReporter;
 
 import com.codahale.metrics.MetricRegistry;
@@ -37,33 +38,41 @@ public class CloudWatchMetricsReporter extends 
MetricsReporter {
   private static final Logger LOG = 
LoggerFactory.getLogger(CloudWatchMetricsReporter.class);
 
   private final MetricRegistry registry;
-  private final HoodieWriteConfig config;
+  private final HoodieMetricsConfig metricsConfig;
   private final CloudWatchReporter reporter;
 
-  public CloudWatchMetricsReporter(HoodieWriteConfig config, MetricRegistry 
registry) {
-    this.config = config;
+  public CloudWatchMetricsReporter(HoodieWriteConfig writeConfig, 
MetricRegistry registry) {
+    this(writeConfig.getMetricsConfig(), registry);
+  }
+
+  CloudWatchMetricsReporter(HoodieWriteConfig writeConfig, MetricRegistry 
registry, CloudWatchReporter reporter) {
+    this(writeConfig.getMetricsConfig(), registry, reporter);
+  }
+
+  public CloudWatchMetricsReporter(HoodieMetricsConfig metricsConfig, 
MetricRegistry registry) {
+    this.metricsConfig = metricsConfig;
     this.registry = registry;
     this.reporter = createCloudWatchReporter();
   }
 
-  CloudWatchMetricsReporter(HoodieWriteConfig config, MetricRegistry registry, 
CloudWatchReporter reporter) {
-    this.config = config;
+  CloudWatchMetricsReporter(HoodieMetricsConfig metricsConfig, MetricRegistry 
registry, CloudWatchReporter reporter) {
+    this.metricsConfig = metricsConfig;
     this.registry = registry;
     this.reporter = reporter;
   }
 
   private CloudWatchReporter createCloudWatchReporter() {
     return CloudWatchReporter.forRegistry(registry)
-        .prefixedWith(config.getCloudWatchMetricPrefix())
-        .namespace(config.getCloudWatchMetricNamespace())
-        .maxDatumsPerRequest(config.getCloudWatchMaxDatumsPerRequest())
-        .build(config.getProps());
+        .prefixedWith(metricsConfig.getCloudWatchMetricPrefix())
+        .namespace(metricsConfig.getCloudWatchMetricNamespace())
+        .maxDatumsPerRequest(metricsConfig.getCloudWatchMaxDatumsPerRequest())
+        .build(metricsConfig.getProps());
   }
 
   @Override
   public void start() {
     LOG.info("Starting CloudWatch Metrics Reporter.");
-    reporter.start(config.getCloudWatchReportPeriodSeconds(), 
TimeUnit.SECONDS);
+    reporter.start(metricsConfig.getCloudWatchReportPeriodSeconds(), 
TimeUnit.SECONDS);
   }
 
   @Override
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/table/action/index/RunIndexActionExecutor.java
 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/table/action/index/RunIndexActionExecutor.java
index 2f006965417..cb29173db63 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/table/action/index/RunIndexActionExecutor.java
+++ 
b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/table/action/index/RunIndexActionExecutor.java
@@ -24,7 +24,6 @@ import org.apache.hudi.avro.model.HoodieIndexPartitionInfo;
 import org.apache.hudi.avro.model.HoodieIndexPlan;
 import org.apache.hudi.client.transaction.TransactionManager;
 import org.apache.hudi.common.engine.HoodieEngineContext;
-import org.apache.hudi.common.metrics.Registry;
 import org.apache.hudi.common.table.HoodieTableConfig;
 import org.apache.hudi.common.table.HoodieTableMetaClient;
 import org.apache.hudi.common.table.timeline.HoodieInstant;
@@ -100,7 +99,7 @@ public class RunIndexActionExecutor<T, I, K, O> extends 
BaseActionExecutor<T, I,
     super(context, config, table, instantTime);
     this.txnManager = new TransactionManager(config, 
table.getMetaClient().getFs());
     if (config.getMetadataConfig().enableMetrics()) {
-      this.metrics = Option.of(new 
HoodieMetadataMetrics(Registry.getRegistry("HoodieIndexer")));
+      this.metrics = Option.of(new 
HoodieMetadataMetrics(config.getMetricsConfig()));
     } else {
       this.metrics = Option.empty();
     }
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieConsoleMetrics.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieConsoleMetrics.java
index 4a0de10512e..43748e96833 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieConsoleMetrics.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieConsoleMetrics.java
@@ -19,6 +19,8 @@
 package org.apache.hudi.metrics;
 
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
+
 import org.junit.jupiter.api.AfterEach;
 import org.junit.jupiter.api.BeforeEach;
 import org.junit.jupiter.api.Test;
@@ -35,17 +37,19 @@ import static org.mockito.Mockito.when;
 public class TestHoodieConsoleMetrics {
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieWriteConfig writeConfig;
+  @Mock
+  HoodieMetricsConfig metricsConfig;
   HoodieMetrics hoodieMetrics;
   Metrics metrics;
 
   @BeforeEach
   public void start() {
-    when(config.getTableName()).thenReturn("console_metrics_test");
-    when(config.isMetricsOn()).thenReturn(true);
-    
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.CONSOLE);
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
-    hoodieMetrics = new HoodieMetrics(config);
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.CONSOLE);
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
+    hoodieMetrics = new HoodieMetrics(writeConfig);
     metrics = hoodieMetrics.getMetrics();
   }
 
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieGraphiteMetrics.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieGraphiteMetrics.java
index dc1d0ae0cf5..63a6704b02f 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieGraphiteMetrics.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieGraphiteMetrics.java
@@ -20,6 +20,8 @@ package org.apache.hudi.metrics;
 
 import org.apache.hudi.common.testutils.NetworkTestUtils;
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
+
 import org.junit.jupiter.api.AfterEach;
 import org.junit.jupiter.api.Test;
 import org.junit.jupiter.api.extension.ExtendWith;
@@ -38,7 +40,9 @@ import static org.mockito.Mockito.when;
 public class TestHoodieGraphiteMetrics {
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieWriteConfig writeConfig;
+  @Mock
+  HoodieMetricsConfig metricsConfig;
   HoodieMetrics hoodieMetrics;
   Metrics metrics;
 
@@ -49,14 +53,14 @@ public class TestHoodieGraphiteMetrics {
 
   @Test
   public void testRegisterGauge() {
-    when(config.isMetricsOn()).thenReturn(true);
-    when(config.getTableName()).thenReturn("table1");
-    
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.GRAPHITE);
-    when(config.getGraphiteServerHost()).thenReturn("localhost");
-    
when(config.getGraphiteServerPort()).thenReturn(NetworkTestUtils.nextFreePort());
-    when(config.getGraphiteReportPeriodSeconds()).thenReturn(30);
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
-    hoodieMetrics = new HoodieMetrics(config);
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.GRAPHITE);
+    when(metricsConfig.getGraphiteServerHost()).thenReturn("localhost");
+    
when(metricsConfig.getGraphiteServerPort()).thenReturn(NetworkTestUtils.nextFreePort());
+    when(metricsConfig.getGraphiteReportPeriodSeconds()).thenReturn(30);
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
+    hoodieMetrics = new HoodieMetrics(writeConfig);
     metrics = hoodieMetrics.getMetrics();
     metrics.registerGauge("graphite_metric", 123L);
     assertEquals("123", metrics.getRegistry().getGauges()
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieJmxMetrics.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieJmxMetrics.java
index a2ec03263a7..3b776c104cd 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieJmxMetrics.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieJmxMetrics.java
@@ -20,6 +20,7 @@ package org.apache.hudi.metrics;
 
 import org.apache.hudi.common.testutils.NetworkTestUtils;
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 
 import org.junit.jupiter.api.AfterEach;
 import org.junit.jupiter.api.BeforeEach;
@@ -40,19 +41,21 @@ import static org.mockito.Mockito.when;
 public class TestHoodieJmxMetrics {
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieWriteConfig writeConfig;
+  @Mock
+  HoodieMetricsConfig metricsConfig;
   HoodieMetrics hoodieMetrics;
   Metrics metrics;
 
   @BeforeEach
   void setup() {
-    when(config.isMetricsOn()).thenReturn(true);
-    when(config.getTableName()).thenReturn("foo");
-    when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.JMX);
-    when(config.getJmxHost()).thenReturn("localhost");
-    
when(config.getJmxPort()).thenReturn(String.valueOf(NetworkTestUtils.nextFreePort()));
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
-    hoodieMetrics = new HoodieMetrics(config);
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.JMX);
+    when(metricsConfig.getJmxHost()).thenReturn("localhost");
+    
when(metricsConfig.getJmxPort()).thenReturn(String.valueOf(NetworkTestUtils.nextFreePort()));
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
+    hoodieMetrics = new HoodieMetrics(writeConfig);
     metrics = hoodieMetrics.getMetrics();
   }
 
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieMetrics.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieMetrics.java
index f305c9d1776..8c34931d93e 100755
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieMetrics.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestHoodieMetrics.java
@@ -22,6 +22,7 @@ import org.apache.hudi.common.model.HoodieCommitMetadata;
 import org.apache.hudi.common.util.Option;
 import org.apache.hudi.common.util.collection.Pair;
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 
 import com.codahale.metrics.Timer;
 import org.junit.jupiter.api.AfterEach;
@@ -44,17 +45,19 @@ import static org.mockito.Mockito.when;
 public class TestHoodieMetrics {
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieWriteConfig writeConfig;
+  @Mock
+  HoodieMetricsConfig metricsConfig;
   HoodieMetrics hoodieMetrics;
   Metrics metrics;
 
   @BeforeEach
   void setUp() {
-    when(config.isMetricsOn()).thenReturn(true);
-    when(config.getTableName()).thenReturn("raw_table");
-    
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.INMEMORY);
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
-    hoodieMetrics = new HoodieMetrics(config);
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.INMEMORY);
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
+    hoodieMetrics = new HoodieMetrics(writeConfig);
     metrics = hoodieMetrics.getMetrics();
   }
 
@@ -143,7 +146,7 @@ public class TestHoodieMetrics {
       when(metadata.getTotalCorruptLogBlocks()).thenReturn(randomValue + 15);
       when(metadata.getTotalRollbackLogBlocks()).thenReturn(randomValue + 16);
       
when(metadata.getMinAndMaxEventTime()).thenReturn(Pair.of(Option.empty(), 
Option.empty()));
-      when(config.isCompactionLogBlockMetricsOn()).thenReturn(true);
+      when(writeConfig.isCompactionLogBlockMetricsOn()).thenReturn(true);
 
       hoodieMetrics.updateCommitMetrics(randomValue + 17, commitTimer.stop(), 
metadata, action);
 
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestMetricsReporterFactory.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestMetricsReporterFactory.java
index a44443d9bd5..dd0ada87693 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestMetricsReporterFactory.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/TestMetricsReporterFactory.java
@@ -20,7 +20,7 @@
 package org.apache.hudi.metrics;
 
 import org.apache.hudi.common.config.TypedProperties;
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.exception.HoodieException;
 import org.apache.hudi.metrics.custom.CustomizableMetricsReporter;
 
@@ -41,27 +41,27 @@ import static org.mockito.Mockito.when;
 public class TestMetricsReporterFactory {
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieMetricsConfig metricsConfig;
 
   @Mock
   MetricRegistry registry;
 
   @Test
   public void metricsReporterFactoryShouldReturnReporter() {
-    
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.INMEMORY);
-    MetricsReporter reporter = MetricsReporterFactory.createReporter(config, 
registry).get();
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.INMEMORY);
+    MetricsReporter reporter = 
MetricsReporterFactory.createReporter(metricsConfig, registry).get();
     assertTrue(reporter instanceof InMemoryMetricsReporter);
   }
 
   @Test
   public void metricsReporterFactoryShouldReturnUserDefinedReporter() {
-    
when(config.getMetricReporterClassName()).thenReturn(DummyMetricsReporter.class.getName());
+    
when(metricsConfig.getMetricReporterClassName()).thenReturn(DummyMetricsReporter.class.getName());
 
     TypedProperties props = new TypedProperties();
     props.setProperty("testKey", "testValue");
 
-    when(config.getProps()).thenReturn(props);
-    MetricsReporter reporter = MetricsReporterFactory.createReporter(config, 
registry).get();
+    when(metricsConfig.getProps()).thenReturn(props);
+    MetricsReporter reporter = 
MetricsReporterFactory.createReporter(metricsConfig, registry).get();
     assertTrue(reporter instanceof CustomizableMetricsReporter);
     assertEquals(props, ((DummyMetricsReporter) reporter).getProps());
     assertEquals(registry, ((DummyMetricsReporter) reporter).getRegistry());
@@ -69,9 +69,9 @@ public class TestMetricsReporterFactory {
 
   @Test
   public void 
metricsReporterFactoryShouldThrowExceptionWhenMetricsReporterClassIsIllegal() {
-    
when(config.getMetricReporterClassName()).thenReturn(IllegalTestMetricsReporter.class.getName());
-    when(config.getProps()).thenReturn(new TypedProperties());
-    assertThrows(HoodieException.class, () -> 
MetricsReporterFactory.createReporter(config, registry));
+    
when(metricsConfig.getMetricReporterClassName()).thenReturn(IllegalTestMetricsReporter.class.getName());
+    when(metricsConfig.getProps()).thenReturn(new TypedProperties());
+    assertThrows(HoodieException.class, () -> 
MetricsReporterFactory.createReporter(metricsConfig, registry));
   }
 
   public static class DummyMetricsReporter extends CustomizableMetricsReporter 
{
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/cloudwatch/TestCloudWatchMetricsReporter.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/cloudwatch/TestCloudWatchMetricsReporter.java
index 7901d802465..4b1aaffbf86 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/cloudwatch/TestCloudWatchMetricsReporter.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/cloudwatch/TestCloudWatchMetricsReporter.java
@@ -19,7 +19,9 @@
 package org.apache.hudi.metrics.cloudwatch;
 
 import org.apache.hudi.aws.cloudwatch.CloudWatchReporter;
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
+import org.apache.hudi.metrics.MetricsReporterFactory;
+import org.apache.hudi.metrics.MetricsReporterType;
 
 import com.codahale.metrics.MetricRegistry;
 import org.junit.jupiter.api.Test;
@@ -27,8 +29,11 @@ import org.junit.jupiter.api.extension.ExtendWith;
 import org.mockito.Mock;
 import org.mockito.junit.jupiter.MockitoExtension;
 
+import java.lang.reflect.InvocationTargetException;
+import java.util.Arrays;
 import java.util.concurrent.TimeUnit;
 
+import static org.junit.jupiter.api.Assertions.assertTrue;
 import static org.mockito.Mockito.times;
 import static org.mockito.Mockito.verify;
 import static org.mockito.Mockito.when;
@@ -37,7 +42,7 @@ import static org.mockito.Mockito.when;
 public class TestCloudWatchMetricsReporter {
 
   @Mock
-  private HoodieWriteConfig config;
+  private HoodieMetricsConfig metricsConfig;
 
   @Mock
   private MetricRegistry registry;
@@ -47,8 +52,8 @@ public class TestCloudWatchMetricsReporter {
 
   @Test
   public void testReporter() {
-    when(config.getCloudWatchReportPeriodSeconds()).thenReturn(30);
-    CloudWatchMetricsReporter metricsReporter = new 
CloudWatchMetricsReporter(config, registry, reporter);
+    when(metricsConfig.getCloudWatchReportPeriodSeconds()).thenReturn(30);
+    CloudWatchMetricsReporter metricsReporter = new 
CloudWatchMetricsReporter(metricsConfig, registry, reporter);
 
     metricsReporter.start();
     verify(reporter, times(1)).start(30, TimeUnit.SECONDS);
@@ -59,4 +64,18 @@ public class TestCloudWatchMetricsReporter {
     metricsReporter.stop();
     verify(reporter, times(1)).stop();
   }
+
+  @Test
+  public void testReporterViaReporterFactory() {
+    try {
+      
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.CLOUDWATCH);
+      // MetricsReporterFactory uses reflection to create 
CloudWatchMetricsReporter
+      // This test verifies that reflection is working well and is able to 
invoke the CloudWatchMetricsReporter constructor
+      MetricsReporterFactory.createReporter(metricsConfig, registry).get();
+    } catch (Exception e) {
+      assertTrue(e.getCause() instanceof InvocationTargetException);
+      assertTrue(Arrays.stream(((InvocationTargetException) 
e.getCause()).getTargetException().getStackTrace()).anyMatch(
+          ste -> 
ste.toString().contains("org.apache.hudi.aws.cloudwatch.CloudWatchReporter.getAmazonCloudWatchClient")));
+    }
+  }
 }
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/datadog/TestDatadogMetricsReporter.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/datadog/TestDatadogMetricsReporter.java
index 16120fe2f24..55637a241e2 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/datadog/TestDatadogMetricsReporter.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/datadog/TestDatadogMetricsReporter.java
@@ -19,6 +19,7 @@
 package org.apache.hudi.metrics.datadog;
 
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.HoodieMetrics;
 import org.apache.hudi.metrics.Metrics;
 import org.apache.hudi.metrics.MetricsReporterType;
@@ -43,7 +44,9 @@ import static org.mockito.Mockito.when;
 public class TestDatadogMetricsReporter {
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieWriteConfig writeConfig;
+  @Mock
+  HoodieMetricsConfig metricsConfig;
   HoodieMetrics hoodieMetrics;
   Metrics metrics;
 
@@ -59,14 +62,15 @@ public class TestDatadogMetricsReporter {
 
   @Test
   public void instantiationShouldFailWhenNoApiKey() {
-    when(config.isMetricsOn()).thenReturn(true);
-    when(config.getTableName()).thenReturn("table1");
-    
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.DATADOG);
-    when(config.getDatadogApiKey()).thenReturn("");
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.DATADOG);
+    when(metricsConfig.getDatadogApiKey()).thenReturn("");
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
 
     Throwable t = assertThrows(IllegalStateException.class, () -> {
-      hoodieMetrics = new HoodieMetrics(config);
+      hoodieMetrics = new HoodieMetrics(writeConfig);
       metrics = hoodieMetrics.getMetrics();
     });
     assertEquals("Datadog cannot be initialized: API key is null or empty.", 
t.getMessage());
@@ -74,14 +78,15 @@ public class TestDatadogMetricsReporter {
 
   @Test
   public void instantiationShouldFailWhenNoMetricPrefix() {
-    when(config.isMetricsOn()).thenReturn(true);
-    when(config.getTableName()).thenReturn("table1");
-    
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.DATADOG);
-    when(config.getDatadogApiKey()).thenReturn("foo");
-    when(config.getDatadogMetricPrefix()).thenReturn("");
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.DATADOG);
+    when(metricsConfig.getDatadogApiKey()).thenReturn("foo");
+    when(metricsConfig.getDatadogMetricPrefix()).thenReturn("");
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
     Throwable t = assertThrows(IllegalStateException.class, () -> {
-      hoodieMetrics = new HoodieMetrics(config);
+      hoodieMetrics = new HoodieMetrics(writeConfig);
       metrics = hoodieMetrics.getMetrics();
     });
     assertEquals("Datadog cannot be initialized: Metric prefix is null or 
empty.", t.getMessage());
@@ -89,20 +94,21 @@ public class TestDatadogMetricsReporter {
 
   @Test
   public void instantiationShouldSucceed() {
-    when(config.isMetricsOn()).thenReturn(true);
-    when(config.getTableName()).thenReturn("table1");
-    
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.DATADOG);
-    when(config.getDatadogApiSite()).thenReturn(ApiSite.EU);
-    when(config.getDatadogApiKey()).thenReturn("foo");
-    when(config.getDatadogApiKeySkipValidation()).thenReturn(true);
-    when(config.getDatadogMetricPrefix()).thenReturn("bar");
-    when(config.getDatadogMetricHost()).thenReturn("foo");
-    when(config.getDatadogMetricTags()).thenReturn(Arrays.asList("baz", 
"foo"));
-    when(config.getDatadogReportPeriodSeconds()).thenReturn(10);
-    when(config.getMetricReporterMetricsNamePrefix()).thenReturn("");
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.DATADOG);
+    when(metricsConfig.getDatadogApiSite()).thenReturn(ApiSite.EU);
+    when(metricsConfig.getDatadogApiKey()).thenReturn("foo");
+    when(metricsConfig.getDatadogApiKeySkipValidation()).thenReturn(true);
+    when(metricsConfig.getDatadogMetricPrefix()).thenReturn("bar");
+    when(metricsConfig.getDatadogMetricHost()).thenReturn("foo");
+    when(metricsConfig.getDatadogMetricTags()).thenReturn(Arrays.asList("baz", 
"foo"));
+    when(metricsConfig.getDatadogReportPeriodSeconds()).thenReturn(10);
+    when(metricsConfig.getMetricReporterMetricsNamePrefix()).thenReturn("");
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
     assertDoesNotThrow(() -> {
-      hoodieMetrics = new HoodieMetrics(config);
+      hoodieMetrics = new HoodieMetrics(writeConfig);
       metrics = hoodieMetrics.getMetrics();
     });
   }
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/m3/TestM3Metrics.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/m3/TestM3Metrics.java
index e7299d706b8..65c4b1d4aba 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/m3/TestM3Metrics.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/m3/TestM3Metrics.java
@@ -25,6 +25,7 @@ import static org.mockito.Mockito.when;
 import java.util.UUID;
 import org.apache.hudi.common.testutils.NetworkTestUtils;
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.HoodieMetrics;
 import org.apache.hudi.metrics.Metrics;
 import org.apache.hudi.metrics.MetricsReporterType;
@@ -38,27 +39,30 @@ import org.mockito.junit.jupiter.MockitoExtension;
 public class TestM3Metrics {
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieWriteConfig writeConfig;
+  @Mock
+  HoodieMetricsConfig metricsConfig;
   HoodieMetrics hoodieMetrics;
   Metrics metrics;
 
   @BeforeEach
   public void start() {
-    when(config.isMetricsOn()).thenReturn(true);
-    when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.M3);
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.M3);
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
   }
 
   @Test
   public void testRegisterGauge() {
-    when(config.getM3ServerHost()).thenReturn("localhost");
-    when(config.getM3ServerPort()).thenReturn(NetworkTestUtils.nextFreePort());
-    when(config.getTableName()).thenReturn("raw_table");
-    when(config.getM3Env()).thenReturn("dev");
-    when(config.getM3Service()).thenReturn("hoodie");
-    when(config.getM3Tags()).thenReturn("tag1=value1,tag2=value2");
-    when(config.getMetricReporterMetricsNamePrefix()).thenReturn("");
-    hoodieMetrics = new HoodieMetrics(config);
+    when(writeConfig.getTableName()).thenReturn("raw_table");
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    when(metricsConfig.getM3ServerHost()).thenReturn("localhost");
+    
when(metricsConfig.getM3ServerPort()).thenReturn(NetworkTestUtils.nextFreePort());
+    when(metricsConfig.getM3Env()).thenReturn("dev");
+    when(metricsConfig.getM3Service()).thenReturn("hoodie");
+    when(metricsConfig.getM3Tags()).thenReturn("tag1=value1,tag2=value2");
+    when(metricsConfig.getMetricReporterMetricsNamePrefix()).thenReturn("");
+    hoodieMetrics = new HoodieMetrics(writeConfig);
     metrics = hoodieMetrics.getMetrics();
     metrics.registerGauge("metric1", 123L);
     assertEquals("123", 
metrics.getRegistry().getGauges().get("metric1").getValue().toString());
@@ -67,14 +71,16 @@ public class TestM3Metrics {
 
   @Test
   public void testEmptyM3Tags() {
-    when(config.getM3ServerHost()).thenReturn("localhost");
-    when(config.getM3ServerPort()).thenReturn(NetworkTestUtils.nextFreePort());
-    when(config.getTableName()).thenReturn("raw_table");
-    when(config.getM3Env()).thenReturn("dev");
-    when(config.getM3Service()).thenReturn("hoodie");
-    when(config.getM3Tags()).thenReturn("");
-    when(config.getMetricReporterMetricsNamePrefix()).thenReturn("");
-    hoodieMetrics = new HoodieMetrics(config);
+    when(writeConfig.getTableName()).thenReturn("raw_table");
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    when(metricsConfig.getM3ServerHost()).thenReturn("localhost");
+    
when(metricsConfig.getM3ServerPort()).thenReturn(NetworkTestUtils.nextFreePort());
+    when(metricsConfig.getM3Env()).thenReturn("dev");
+    when(metricsConfig.getM3Service()).thenReturn("hoodie");
+    when(metricsConfig.getM3Tags()).thenReturn("");
+    when(metricsConfig.getMetricReporterMetricsNamePrefix()).thenReturn("");
+    hoodieMetrics = new HoodieMetrics(writeConfig);
     metrics = hoodieMetrics.getMetrics();
     metrics.registerGauge("metric1", 123L);
     assertEquals("123", 
metrics.getRegistry().getGauges().get("metric1").getValue().toString());
@@ -83,10 +89,12 @@ public class TestM3Metrics {
 
   @Test
   public void testInvalidM3Tags() {
-    when(config.getTableName()).thenReturn("raw_table");
-    when(config.getMetricReporterMetricsNamePrefix()).thenReturn("");
+    when(writeConfig.getTableName()).thenReturn("raw_table");
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    when(metricsConfig.getMetricReporterMetricsNamePrefix()).thenReturn("");
     assertThrows(RuntimeException.class, () -> {
-      hoodieMetrics = new HoodieMetrics(config);
+      hoodieMetrics = new HoodieMetrics(writeConfig);
     });
   }
 }
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/prometheus/TestPrometheusReporter.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/prometheus/TestPrometheusReporter.java
index 4e94ece52c9..9ad2b8388a2 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/prometheus/TestPrometheusReporter.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/prometheus/TestPrometheusReporter.java
@@ -19,6 +19,7 @@
 package org.apache.hudi.metrics.prometheus;
 
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.HoodieMetrics;
 import org.apache.hudi.metrics.Metrics;
 import org.apache.hudi.metrics.MetricsReporterType;
@@ -38,7 +39,9 @@ import static org.mockito.Mockito.when;
 public class TestPrometheusReporter {
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieWriteConfig writeConfig;
+  @Mock
+  HoodieMetricsConfig metricsConfig;
   HoodieMetrics hoodieMetrics;
   Metrics metrics;
 
@@ -51,14 +54,14 @@ public class TestPrometheusReporter {
 
   @Test
   public void testRegisterGauge() {
-    when(config.isMetricsOn()).thenReturn(true);
-    when(config.getTableName()).thenReturn("foo");
-    
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.PROMETHEUS);
-    when(config.getPrometheusPort()).thenReturn(9090);
-    when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.PROMETHEUS);
+    when(metricsConfig.getPrometheusPort()).thenReturn(9090);
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
     assertDoesNotThrow(() -> {
-      new HoodieMetrics(config);
-      hoodieMetrics = new HoodieMetrics(config);
+      new HoodieMetrics(writeConfig);
+      hoodieMetrics = new HoodieMetrics(writeConfig);
       metrics = hoodieMetrics.getMetrics();
     });
   }
diff --git 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/prometheus/TestPushGateWayReporter.java
 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/prometheus/TestPushGateWayReporter.java
index 27f7c5a8345..aa1c3f06b6f 100644
--- 
a/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/prometheus/TestPushGateWayReporter.java
+++ 
b/hudi-client/hudi-client-common/src/test/java/org/apache/hudi/metrics/prometheus/TestPushGateWayReporter.java
@@ -20,6 +20,7 @@ package org.apache.hudi.metrics.prometheus;
 
 import org.apache.hudi.common.util.Option;
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.HoodieMetrics;
 import org.apache.hudi.metrics.MetricUtils;
 import org.apache.hudi.metrics.Metrics;
@@ -56,7 +57,9 @@ public class TestPushGateWayReporter {
   static final URL PROP_FILE_DATADOG_URL = 
TestPushGateWayReporter.class.getClassLoader().getResource("datadog.properties");
 
   @Mock
-  HoodieWriteConfig config;
+  HoodieWriteConfig writeConfig;
+  @Mock
+  HoodieMetricsConfig metricsConfig;
 
   HoodieMetrics hoodieMetrics;
   Metrics metrics;
@@ -70,10 +73,12 @@ public class TestPushGateWayReporter {
 
   @Test
   public void testRegisterGauge() {
-    when(config.isMetricsOn()).thenReturn(true);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    configureDefaultReporter();
 
     assertDoesNotThrow(() -> {
-      hoodieMetrics = new HoodieMetrics(config);
+      hoodieMetrics = new HoodieMetrics(writeConfig);
       metrics = hoodieMetrics.getMetrics();
     });
 
@@ -85,21 +90,20 @@ public class TestPushGateWayReporter {
   @ParameterizedTest
   @ValueSource(booleans = {true, false})
   public void testMultiReporter(boolean addDefaultReporter) throws 
IOException, InterruptedException, URISyntaxException {
+    when(writeConfig.getMetricsConfig()).thenReturn(metricsConfig);
+    when(writeConfig.isMetricsOn()).thenReturn(true);
 
     String propPrometheusPath = 
Objects.requireNonNull(PROP_FILE_PROMETHEUS_URL).toURI().getPath();
     String propDatadogPath = 
Objects.requireNonNull(PROP_FILE_DATADOG_URL).toURI().getPath();
     if (addDefaultReporter) {
-      when(config.isMetricsOn()).thenReturn(true);
-      
when(config.getMetricsReporterType()).thenReturn(MetricsReporterType.PROMETHEUS_PUSHGATEWAY);
-      when(config.getPushGatewayReportPeriodSeconds()).thenReturn(30);
+      configureDefaultReporter();
     } else {
-      when(config.getBasePath()).thenReturn("s3://test" + UUID.randomUUID());
-      
when(config.getMetricReporterMetricsNamePrefix()).thenReturn(TestPushGateWayReporter.class.getSimpleName());
-      
when(config.getMetricReporterFileBasedConfigs()).thenReturn(propPrometheusPath 
+ "," + propDatadogPath);
-      when(config.isMetricsOn()).thenReturn(true);
+      when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
+      
when(metricsConfig.getMetricReporterMetricsNamePrefix()).thenReturn(TestPushGateWayReporter.class.getSimpleName());
+      
when(metricsConfig.getMetricReporterFileBasedConfigs()).thenReturn(propPrometheusPath
 + "," + propDatadogPath);
     }
 
-    hoodieMetrics = new HoodieMetrics(config);
+    hoodieMetrics = new HoodieMetrics(writeConfig);
     metrics = hoodieMetrics.getMetrics();
 
     Map<String, Long> metricsMap = new HashMap<>();
@@ -123,29 +127,29 @@ public class TestPushGateWayReporter {
     PushGatewayMetricsReporter reporter;
     Map<String, String> labels;
 
-    when(config.getPushGatewayLabels()).thenReturn("hudi:prometheus");
-    reporter = new PushGatewayMetricsReporter(config, null);
+    when(metricsConfig.getPushGatewayLabels()).thenReturn("hudi:prometheus");
+    reporter = new PushGatewayMetricsReporter(metricsConfig, null);
     labels = reporter.getLabels();
     assertEquals(1, labels.size());
     assertTrue(labels.containsKey("hudi"));
     assertTrue(labels.containsValue("prometheus"));
 
-    when(config.getPushGatewayLabels()).thenReturn("hudi:prome:theus");
-    reporter = new PushGatewayMetricsReporter(config, null);
+    when(metricsConfig.getPushGatewayLabels()).thenReturn("hudi:prome:theus");
+    reporter = new PushGatewayMetricsReporter(metricsConfig, null);
     labels = reporter.getLabels();
     assertEquals(1, labels.size());
     assertTrue(labels.containsKey("hudi"));
     assertTrue(labels.containsValue("prome:theus"));
 
-    when(config.getPushGatewayLabels()).thenReturn("hudiprometheus");
-    reporter = new PushGatewayMetricsReporter(config, null);
+    when(metricsConfig.getPushGatewayLabels()).thenReturn("hudiprometheus");
+    reporter = new PushGatewayMetricsReporter(metricsConfig, null);
     labels = reporter.getLabels();
     assertEquals(1, labels.size());
     assertTrue(labels.containsKey("hudiprometheus"));
     assertTrue(labels.containsValue(""));
 
-    
when(config.getPushGatewayLabels()).thenReturn("hudi1:prometheus,hudi2:prometheus");
-    reporter = new PushGatewayMetricsReporter(config, null);
+    
when(metricsConfig.getPushGatewayLabels()).thenReturn("hudi1:prometheus,hudi2:prometheus");
+    reporter = new PushGatewayMetricsReporter(metricsConfig, null);
     labels = reporter.getLabels();
     assertEquals(2, labels.size());
     assertTrue(labels.containsKey("hudi1"));
@@ -153,11 +157,17 @@ public class TestPushGateWayReporter {
     assertTrue(labels.containsValue("prometheus"));
 
     try {
-      
when(config.getPushGatewayLabels()).thenReturn("hudi:prometheus,hudi:prom");
-      reporter = new PushGatewayMetricsReporter(config, null);
+      
when(metricsConfig.getPushGatewayLabels()).thenReturn("hudi:prometheus,hudi:prom");
+      reporter = new PushGatewayMetricsReporter(metricsConfig, null);
       fail("Should fail");
     } catch (IllegalStateException e) {
       assertTrue(e.getMessage().contains("Multiple values {prometheus, prom} 
for same key"));
     }
   }
+
+  private void configureDefaultReporter() {
+    when(metricsConfig.getBasePath()).thenReturn("s3://test" + 
UUID.randomUUID());
+    
when(metricsConfig.getMetricsReporterType()).thenReturn(MetricsReporterType.PROMETHEUS_PUSHGATEWAY);
+    when(metricsConfig.getPushGatewayReportPeriodSeconds()).thenReturn(30);
+  }
 }
diff --git 
a/hudi-client/hudi-flink-client/src/main/java/org/apache/hudi/metadata/FlinkHoodieBackedTableMetadataWriter.java
 
b/hudi-client/hudi-flink-client/src/main/java/org/apache/hudi/metadata/FlinkHoodieBackedTableMetadataWriter.java
index 61999c44b6e..bafee7295c3 100644
--- 
a/hudi-client/hudi-flink-client/src/main/java/org/apache/hudi/metadata/FlinkHoodieBackedTableMetadataWriter.java
+++ 
b/hudi-client/hudi-flink-client/src/main/java/org/apache/hudi/metadata/FlinkHoodieBackedTableMetadataWriter.java
@@ -23,7 +23,6 @@ import org.apache.hudi.client.HoodieFlinkWriteClient;
 import org.apache.hudi.client.WriteStatus;
 import org.apache.hudi.common.data.HoodieData;
 import org.apache.hudi.common.engine.HoodieEngineContext;
-import org.apache.hudi.common.metrics.Registry;
 import org.apache.hudi.common.model.HoodieFailedWritesCleaningPolicy;
 import org.apache.hudi.common.model.HoodieRecord;
 import org.apache.hudi.common.table.HoodieTableMetaClient;
@@ -87,8 +86,7 @@ public class FlinkHoodieBackedTableMetadataWriter extends 
HoodieBackedTableMetad
   protected void initRegistry() {
     if (metadataWriteConfig.isMetricsOn()) {
       // should support executor metrics
-      Registry registry = Registry.getRegistry("HoodieMetadata");
-      this.metrics = Option.of(new HoodieMetadataMetrics(registry));
+      this.metrics = Option.of(new 
HoodieMetadataMetrics(metadataWriteConfig.getMetricsConfig()));
     } else {
       this.metrics = Option.empty();
     }
diff --git 
a/hudi-client/hudi-java-client/src/main/java/org/apache/hudi/metadata/JavaHoodieBackedTableMetadataWriter.java
 
b/hudi-client/hudi-java-client/src/main/java/org/apache/hudi/metadata/JavaHoodieBackedTableMetadataWriter.java
index f16392378c8..cca1b883882 100644
--- 
a/hudi-client/hudi-java-client/src/main/java/org/apache/hudi/metadata/JavaHoodieBackedTableMetadataWriter.java
+++ 
b/hudi-client/hudi-java-client/src/main/java/org/apache/hudi/metadata/JavaHoodieBackedTableMetadataWriter.java
@@ -22,7 +22,6 @@ import org.apache.hudi.client.BaseHoodieWriteClient;
 import org.apache.hudi.client.HoodieJavaWriteClient;
 import org.apache.hudi.common.data.HoodieData;
 import org.apache.hudi.common.engine.HoodieEngineContext;
-import org.apache.hudi.common.metrics.Registry;
 import org.apache.hudi.common.model.HoodieFailedWritesCleaningPolicy;
 import org.apache.hudi.common.model.HoodieRecord;
 import org.apache.hudi.common.table.timeline.HoodieActiveTimeline;
@@ -74,8 +73,7 @@ public class JavaHoodieBackedTableMetadataWriter extends 
HoodieBackedTableMetada
   @Override
   protected void initRegistry() {
     if (metadataWriteConfig.isMetricsOn()) {
-      Registry registry = Registry.getRegistry("HoodieMetadata");
-      this.metrics = Option.of(new HoodieMetadataMetrics(registry));
+      this.metrics = Option.of(new 
HoodieMetadataMetrics(metadataWriteConfig.getMetricsConfig()));
     } else {
       this.metrics = Option.empty();
     }
diff --git 
a/hudi-client/hudi-java-client/src/test/java/org/apache/hudi/client/TestJavaHoodieBackedMetadata.java
 
b/hudi-client/hudi-java-client/src/test/java/org/apache/hudi/client/TestJavaHoodieBackedMetadata.java
index 8e1bbc84b4b..22f46e58f62 100644
--- 
a/hudi-client/hudi-java-client/src/test/java/org/apache/hudi/client/TestJavaHoodieBackedMetadata.java
+++ 
b/hudi-client/hudi-java-client/src/test/java/org/apache/hudi/client/TestJavaHoodieBackedMetadata.java
@@ -33,7 +33,6 @@ import 
org.apache.hudi.common.config.SerializableConfiguration;
 import org.apache.hudi.common.engine.HoodieEngineContext;
 import org.apache.hudi.common.fs.ConsistencyGuardConfig;
 import org.apache.hudi.common.fs.FSUtils;
-import org.apache.hudi.common.metrics.Registry;
 import org.apache.hudi.common.model.FileSlice;
 import org.apache.hudi.common.model.HoodieAvroRecord;
 import org.apache.hudi.common.model.HoodieBaseFile;
@@ -98,6 +97,7 @@ import org.apache.hudi.metadata.HoodieTableMetadata;
 import org.apache.hudi.metadata.HoodieTableMetadataUtil;
 import org.apache.hudi.metadata.JavaHoodieBackedTableMetadataWriter;
 import org.apache.hudi.metadata.MetadataPartitionType;
+import org.apache.hudi.metrics.Metrics;
 import org.apache.hudi.storage.StoragePath;
 import org.apache.hudi.table.HoodieJavaTable;
 import org.apache.hudi.table.HoodieTable;
@@ -2340,7 +2340,8 @@ public class TestJavaHoodieBackedMetadata extends 
TestHoodieMetadataBase {
     init(HoodieTableType.COPY_ON_WRITE, false);
     HoodieEngineContext engineContext = new 
HoodieJavaEngineContext(hadoopConf);
 
-    try (HoodieJavaWriteClient client = new 
HoodieJavaWriteClient(engineContext, getWriteConfigBuilder(true, true, 
true).build())) {
+    HoodieWriteConfig writeConfig = getWriteConfigBuilder(true, true, 
true).build();
+    try (HoodieJavaWriteClient client = new 
HoodieJavaWriteClient(engineContext, writeConfig)) {
       // Write
       String newCommitTime = HoodieActiveTimeline.createNewInstantTime();
       List<HoodieRecord> records = dataGen.generateInserts(newCommitTime, 20);
@@ -2349,15 +2350,15 @@ public class TestJavaHoodieBackedMetadata extends 
TestHoodieMetadataBase {
       assertNoWriteErrors(writeStatuses);
       validateMetadata(client);
 
-      Registry metricsRegistry = Registry.getRegistry("HoodieMetadata");
-      
assertTrue(metricsRegistry.getAllCounts().containsKey(HoodieMetadataMetrics.INITIALIZE_STR
 + ".count"));
-      
assertTrue(metricsRegistry.getAllCounts().containsKey(HoodieMetadataMetrics.INITIALIZE_STR
 + ".totalDuration"));
-      
assertTrue(metricsRegistry.getAllCounts().get(HoodieMetadataMetrics.INITIALIZE_STR
 + ".count") >= 1L);
+      Metrics metrics = Metrics.getInstance(writeConfig.getMetricsConfig());
+      
assertTrue(metrics.getRegistry().getGauges().containsKey(HoodieMetadataMetrics.INITIALIZE_STR
 + ".count"));
+      
assertTrue(metrics.getRegistry().getGauges().containsKey(HoodieMetadataMetrics.INITIALIZE_STR
 + ".totalDuration"));
+      assertTrue((Long) 
metrics.getRegistry().getGauges().get(HoodieMetadataMetrics.INITIALIZE_STR + 
".count").getValue() >= 1L);
       final String prefix = FILES.getPartitionPath() + ".";
-      assertTrue(metricsRegistry.getAllCounts().containsKey(prefix + 
HoodieMetadataMetrics.STAT_COUNT_BASE_FILES));
-      assertTrue(metricsRegistry.getAllCounts().containsKey(prefix + 
HoodieMetadataMetrics.STAT_COUNT_LOG_FILES));
-      assertTrue(metricsRegistry.getAllCounts().containsKey(prefix + 
HoodieMetadataMetrics.STAT_TOTAL_BASE_FILE_SIZE));
-      assertTrue(metricsRegistry.getAllCounts().containsKey(prefix + 
HoodieMetadataMetrics.STAT_TOTAL_LOG_FILE_SIZE));
+      assertTrue(metrics.getRegistry().getGauges().containsKey(prefix + 
HoodieMetadataMetrics.STAT_COUNT_BASE_FILES));
+      assertTrue(metrics.getRegistry().getGauges().containsKey(prefix + 
HoodieMetadataMetrics.STAT_COUNT_LOG_FILES));
+      assertTrue(metrics.getRegistry().getGauges().containsKey(prefix + 
HoodieMetadataMetrics.STAT_TOTAL_BASE_FILE_SIZE));
+      assertTrue(metrics.getRegistry().getGauges().containsKey(prefix + 
HoodieMetadataMetrics.STAT_TOTAL_LOG_FILE_SIZE));
     }
   }
 
diff --git 
a/hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/metadata/SparkHoodieBackedTableMetadataWriter.java
 
b/hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/metadata/SparkHoodieBackedTableMetadataWriter.java
index 15b527a0fe3..d6e964e7faf 100644
--- 
a/hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/metadata/SparkHoodieBackedTableMetadataWriter.java
+++ 
b/hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/metadata/SparkHoodieBackedTableMetadataWriter.java
@@ -106,7 +106,7 @@ public class SparkHoodieBackedTableMetadataWriter extends 
HoodieBackedTableMetad
       } else {
         registry = Registry.getRegistry("HoodieMetadata");
       }
-      this.metrics = Option.of(new HoodieMetadataMetrics(registry));
+      this.metrics = Option.of(new 
HoodieMetadataMetrics(metadataWriteConfig.getMetricsConfig()));
     } else {
       this.metrics = Option.empty();
     }
diff --git 
a/hudi-client/hudi-spark-client/src/test/java/org/apache/hudi/client/functional/TestHoodieBackedMetadata.java
 
b/hudi-client/hudi-spark-client/src/test/java/org/apache/hudi/client/functional/TestHoodieBackedMetadata.java
index ba78f18efae..6cc474676de 100644
--- 
a/hudi-client/hudi-spark-client/src/test/java/org/apache/hudi/client/functional/TestHoodieBackedMetadata.java
+++ 
b/hudi-client/hudi-spark-client/src/test/java/org/apache/hudi/client/functional/TestHoodieBackedMetadata.java
@@ -33,7 +33,6 @@ import org.apache.hudi.common.config.LockConfiguration;
 import org.apache.hudi.common.config.SerializableConfiguration;
 import org.apache.hudi.common.fs.ConsistencyGuardConfig;
 import org.apache.hudi.common.fs.FSUtils;
-import org.apache.hudi.common.metrics.Registry;
 import org.apache.hudi.common.model.FileSlice;
 import org.apache.hudi.common.model.HoodieAvroRecord;
 import org.apache.hudi.common.model.HoodieBaseFile;
@@ -101,6 +100,7 @@ import org.apache.hudi.metadata.HoodieTableMetadata;
 import org.apache.hudi.metadata.HoodieTableMetadataUtil;
 import org.apache.hudi.metadata.MetadataPartitionType;
 import org.apache.hudi.metadata.SparkHoodieBackedTableMetadataWriter;
+import org.apache.hudi.metrics.Metrics;
 import org.apache.hudi.storage.StoragePath;
 import org.apache.hudi.table.HoodieSparkTable;
 import org.apache.hudi.table.HoodieTable;
@@ -2981,15 +2981,15 @@ public class TestHoodieBackedMetadata extends 
TestHoodieMetadataBase {
       assertNoWriteErrors(writeStatuses);
       validateMetadata(client);
 
-      Registry metricsRegistry = Registry.getRegistry("HoodieMetadata");
-      
assertTrue(metricsRegistry.getAllCounts().containsKey(HoodieMetadataMetrics.INITIALIZE_STR
 + ".count"));
-      
assertTrue(metricsRegistry.getAllCounts().containsKey(HoodieMetadataMetrics.INITIALIZE_STR
 + ".totalDuration"));
-      
assertTrue(metricsRegistry.getAllCounts().get(HoodieMetadataMetrics.INITIALIZE_STR
 + ".count") >= 1L);
+      Metrics metrics = Metrics.getInstance(writeConfig.getMetricsConfig());
+      
assertTrue(metrics.getRegistry().getGauges().containsKey(HoodieMetadataMetrics.INITIALIZE_STR
 + ".count"));
+      
assertTrue(metrics.getRegistry().getGauges().containsKey(HoodieMetadataMetrics.INITIALIZE_STR
 + ".totalDuration"));
+      assertTrue((Long) 
metrics.getRegistry().getGauges().get(HoodieMetadataMetrics.INITIALIZE_STR + 
".count").getValue() >= 1L);
       final String prefix = FILES.getPartitionPath() + ".";
-      assertTrue(metricsRegistry.getAllCounts().containsKey(prefix + 
HoodieMetadataMetrics.STAT_COUNT_BASE_FILES));
-      assertTrue(metricsRegistry.getAllCounts().containsKey(prefix + 
HoodieMetadataMetrics.STAT_COUNT_LOG_FILES));
-      assertTrue(metricsRegistry.getAllCounts().containsKey(prefix + 
HoodieMetadataMetrics.STAT_TOTAL_BASE_FILE_SIZE));
-      assertTrue(metricsRegistry.getAllCounts().containsKey(prefix + 
HoodieMetadataMetrics.STAT_TOTAL_LOG_FILE_SIZE));
+      assertTrue(metrics.getRegistry().getGauges().containsKey(prefix + 
HoodieMetadataMetrics.STAT_COUNT_BASE_FILES));
+      assertTrue(metrics.getRegistry().getGauges().containsKey(prefix + 
HoodieMetadataMetrics.STAT_COUNT_LOG_FILES));
+      assertTrue(metrics.getRegistry().getGauges().containsKey(prefix + 
HoodieMetadataMetrics.STAT_TOTAL_BASE_FILE_SIZE));
+      assertTrue(metrics.getRegistry().getGauges().containsKey(prefix + 
HoodieMetadataMetrics.STAT_TOTAL_LOG_FILE_SIZE));
     }
   }
 
diff --git a/hudi-common/pom.xml b/hudi-common/pom.xml
index 3cb5bcc233e..6e2aee560f4 100644
--- a/hudi-common/pom.xml
+++ b/hudi-common/pom.xml
@@ -316,5 +316,52 @@
       <artifactId>disruptor</artifactId>
       <version>${disruptor.version}</version>
     </dependency>
+
+    <!-- Dropwizard Metrics -->
+    <dependency>
+          <groupId>io.dropwizard.metrics</groupId>
+          <artifactId>metrics-graphite</artifactId>
+          <exclusions>
+              <exclusion>
+                  <groupId>com.rabbitmq</groupId>
+                  <artifactId>*</artifactId>
+              </exclusion>
+          </exclusions>
+      </dependency>
+      <dependency>
+          <groupId>io.dropwizard.metrics</groupId>
+          <artifactId>metrics-core</artifactId>
+      </dependency>
+      <dependency>
+          <groupId>io.dropwizard.metrics</groupId>
+          <artifactId>metrics-jmx</artifactId>
+      </dependency>
+      <dependency>
+          <groupId>io.prometheus</groupId>
+          <artifactId>simpleclient</artifactId>
+      </dependency>
+      <dependency>
+          <groupId>io.prometheus</groupId>
+          <artifactId>simpleclient_httpserver</artifactId>
+      </dependency>
+      <dependency>
+          <groupId>io.prometheus</groupId>
+          <artifactId>simpleclient_dropwizard</artifactId>
+      </dependency>
+      <dependency>
+          <groupId>io.prometheus</groupId>
+          <artifactId>simpleclient_pushgateway</artifactId>
+      </dependency>
+    <dependency>
+      <groupId>com.uber.m3</groupId>
+      <artifactId>tally-m3</artifactId>
+      <version>${tally.version}</version>
+    </dependency>
+    <dependency>
+      <groupId>com.uber.m3</groupId>
+      <artifactId>tally-core</artifactId>
+      <version>${tally.version}</version>
+    </dependency>
+
   </dependencies>
 </project>
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsCloudWatchConfig.java
 
b/hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsCloudWatchConfig.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsCloudWatchConfig.java
rename to 
hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsCloudWatchConfig.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsConfig.java
 
b/hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsConfig.java
similarity index 60%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsConfig.java
rename to 
hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsConfig.java
index 328619f5e9c..6ad389c05d7 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsConfig.java
+++ 
b/hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsConfig.java
@@ -21,17 +21,25 @@ package org.apache.hudi.config.metrics;
 import org.apache.hudi.common.config.ConfigClassProperty;
 import org.apache.hudi.common.config.ConfigGroups;
 import org.apache.hudi.common.config.ConfigProperty;
+import org.apache.hudi.common.config.HoodieCommonConfig;
 import org.apache.hudi.common.config.HoodieConfig;
 import org.apache.hudi.common.table.HoodieTableConfig;
 import org.apache.hudi.common.util.Option;
+import org.apache.hudi.common.util.ReflectionUtils;
 import org.apache.hudi.metrics.MetricsReporterType;
+import org.apache.hudi.metrics.datadog.DatadogHttpClient;
 
 import javax.annotation.concurrent.Immutable;
 
 import java.io.File;
 import java.io.FileReader;
 import java.io.IOException;
+import java.io.InputStream;
+import java.util.Arrays;
+import java.util.List;
 import java.util.Properties;
+import java.util.function.Supplier;
+import java.util.stream.Collectors;
 
 /**
  * Fetch the configurations used by the Metrics system.
@@ -156,6 +164,185 @@ public class HoodieMetricsConfig extends HoodieConfig {
     return new Builder();
   }
 
+  /**
+   * base properties.
+   */
+  public String getBasePath() {
+    return getString(HoodieCommonConfig.BASE_PATH);
+  }
+
+  /**
+   * metrics properties.
+   */
+  public boolean isMetricsOn() {
+    return getBoolean(HoodieMetricsConfig.TURN_METRICS_ON);
+  }
+
+  /**
+   * metrics properties.
+   */
+  public boolean isCompactionLogBlockMetricsOn() {
+    return 
getBoolean(HoodieMetricsConfig.TURN_METRICS_COMPACTION_LOG_BLOCKS_ON);
+  }
+
+  public boolean isExecutorMetricsEnabled() {
+    return Boolean.parseBoolean(
+        getStringOrDefault(HoodieMetricsConfig.EXECUTOR_METRICS_ENABLE, 
"false"));
+  }
+
+  public boolean isLockingMetricsEnabled() {
+    return getBoolean(HoodieMetricsConfig.LOCK_METRICS_ENABLE);
+  }
+
+  public MetricsReporterType getMetricsReporterType() {
+    return 
MetricsReporterType.valueOf(getString(HoodieMetricsConfig.METRICS_REPORTER_TYPE_VALUE));
+  }
+
+  public String getGraphiteServerHost() {
+    return getString(HoodieMetricsGraphiteConfig.GRAPHITE_SERVER_HOST_NAME);
+  }
+
+  public int getGraphiteServerPort() {
+    return getInt(HoodieMetricsGraphiteConfig.GRAPHITE_SERVER_PORT_NUM);
+  }
+
+  public String getGraphiteMetricPrefix() {
+    return getString(HoodieMetricsGraphiteConfig.GRAPHITE_METRIC_PREFIX_VALUE);
+  }
+
+  public int getGraphiteReportPeriodSeconds() {
+    return 
getInt(HoodieMetricsGraphiteConfig.GRAPHITE_REPORT_PERIOD_IN_SECONDS);
+  }
+
+  public String getM3ServerHost() {
+    return getString(HoodieMetricsM3Config.M3_SERVER_HOST_NAME);
+  }
+
+  public int getM3ServerPort() {
+    return getInt(HoodieMetricsM3Config.M3_SERVER_PORT_NUM);
+  }
+
+  public String getM3Tags() {
+    return getString(HoodieMetricsM3Config.M3_TAGS);
+  }
+
+  public String getM3Env() {
+    return getString(HoodieMetricsM3Config.M3_ENV);
+  }
+
+  public String getM3Service() {
+    return getString(HoodieMetricsM3Config.M3_SERVICE);
+  }
+
+  public String getJmxHost() {
+    return getString(HoodieMetricsJmxConfig.JMX_HOST_NAME);
+  }
+
+  public String getJmxPort() {
+    return getString(HoodieMetricsJmxConfig.JMX_PORT_NUM);
+  }
+
+  public int getDatadogReportPeriodSeconds() {
+    return getInt(HoodieMetricsDatadogConfig.REPORT_PERIOD_IN_SECONDS);
+  }
+
+  public DatadogHttpClient.ApiSite getDatadogApiSite() {
+    return 
DatadogHttpClient.ApiSite.valueOf(getString(HoodieMetricsDatadogConfig.API_SITE_VALUE));
+  }
+
+  public String getDatadogApiKey() {
+    if (props.containsKey(HoodieMetricsDatadogConfig.API_KEY.key())) {
+      return getString(HoodieMetricsDatadogConfig.API_KEY);
+
+    } else {
+      Supplier<String> apiKeySupplier = ReflectionUtils.loadClass(
+          getString(HoodieMetricsDatadogConfig.API_KEY_SUPPLIER));
+      return apiKeySupplier.get();
+    }
+  }
+
+  public boolean getDatadogApiKeySkipValidation() {
+    return getBoolean(HoodieMetricsDatadogConfig.API_KEY_SKIP_VALIDATION);
+  }
+
+  public int getDatadogApiTimeoutSeconds() {
+    return getInt(HoodieMetricsDatadogConfig.API_TIMEOUT_IN_SECONDS);
+  }
+
+  public String getDatadogMetricPrefix() {
+    return getString(HoodieMetricsDatadogConfig.METRIC_PREFIX_VALUE);
+  }
+
+  public String getDatadogMetricHost() {
+    return getString(HoodieMetricsDatadogConfig.METRIC_HOST_NAME);
+  }
+
+  public List<String> getDatadogMetricTags() {
+    return Arrays.stream(getStringOrDefault(
+        HoodieMetricsDatadogConfig.METRIC_TAG_VALUES, 
",").split("\\s*,\\s*")).collect(Collectors.toList());
+  }
+
+  public int getCloudWatchReportPeriodSeconds() {
+    return getInt(HoodieMetricsCloudWatchConfig.REPORT_PERIOD_SECONDS);
+  }
+
+  public String getCloudWatchMetricPrefix() {
+    return getString(HoodieMetricsCloudWatchConfig.METRIC_PREFIX);
+  }
+
+  public String getCloudWatchMetricNamespace() {
+    return getString(HoodieMetricsCloudWatchConfig.METRIC_NAMESPACE);
+  }
+
+  public int getCloudWatchMaxDatumsPerRequest() {
+    return getInt(HoodieMetricsCloudWatchConfig.MAX_DATUMS_PER_REQUEST);
+  }
+
+  public String getMetricReporterClassName() {
+    return getString(HoodieMetricsConfig.METRICS_REPORTER_CLASS_NAME);
+  }
+
+  public int getPrometheusPort() {
+    return getInt(HoodieMetricsPrometheusConfig.PROMETHEUS_PORT_NUM);
+  }
+
+  public String getPushGatewayHost() {
+    return getString(HoodieMetricsPrometheusConfig.PUSHGATEWAY_HOST_NAME);
+  }
+
+  public int getPushGatewayPort() {
+    return getInt(HoodieMetricsPrometheusConfig.PUSHGATEWAY_PORT_NUM);
+  }
+
+  public int getPushGatewayReportPeriodSeconds() {
+    return 
getInt(HoodieMetricsPrometheusConfig.PUSHGATEWAY_REPORT_PERIOD_IN_SECONDS);
+  }
+
+  public boolean getPushGatewayDeleteOnShutdown() {
+    return 
getBoolean(HoodieMetricsPrometheusConfig.PUSHGATEWAY_DELETE_ON_SHUTDOWN_ENABLE);
+  }
+
+  public String getPushGatewayJobName() {
+    return getString(HoodieMetricsPrometheusConfig.PUSHGATEWAY_JOBNAME);
+  }
+
+  public String getPushGatewayLabels() {
+    return getString(HoodieMetricsPrometheusConfig.PUSHGATEWAY_LABELS);
+  }
+
+  public boolean getPushGatewayRandomJobNameSuffix() {
+    return 
getBoolean(HoodieMetricsPrometheusConfig.PUSHGATEWAY_RANDOM_JOBNAME_SUFFIX);
+  }
+
+  public String getMetricReporterMetricsNamePrefix() {
+    // Metrics prefixes should not have a dot as this is usually a separator
+    return 
getStringOrDefault(HoodieMetricsConfig.METRICS_REPORTER_PREFIX).replaceAll("\\.",
 "_");
+  }
+
+  public String getMetricReporterFileBasedConfigs() {
+    return 
getStringOrDefault(HoodieMetricsConfig.METRICS_REPORTER_FILE_BASED_CONFIGS_PATH);
+  }
+
   public static class Builder {
 
     private final HoodieMetricsConfig hoodieMetricsConfig = new 
HoodieMetricsConfig();
@@ -167,6 +354,15 @@ public class HoodieMetricsConfig extends HoodieConfig {
       }
     }
 
+    public Builder fromInputStream(InputStream inputStream) throws IOException 
{
+      try {
+        this.hoodieMetricsConfig.getProps().load(inputStream);
+        return this;
+      } finally {
+        inputStream.close();
+      }
+    }
+
     public Builder fromProperties(Properties props) {
       this.hoodieMetricsConfig.getProps().putAll(props);
       return this;
@@ -182,6 +378,11 @@ public class HoodieMetricsConfig extends HoodieConfig {
       return this;
     }
 
+    public Builder withPath(String basePath) {
+      hoodieMetricsConfig.setValue(HoodieCommonConfig.BASE_PATH, basePath);
+      return this;
+    }
+
     public Builder withReporterType(String reporterType) {
       hoodieMetricsConfig.setValue(METRICS_REPORTER_TYPE_VALUE, reporterType);
       return this;
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsDatadogConfig.java
 
b/hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsDatadogConfig.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsDatadogConfig.java
rename to 
hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsDatadogConfig.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsGraphiteConfig.java
 
b/hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsGraphiteConfig.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsGraphiteConfig.java
rename to 
hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsGraphiteConfig.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsJmxConfig.java
 
b/hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsJmxConfig.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsJmxConfig.java
rename to 
hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsJmxConfig.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsM3Config.java
 
b/hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsM3Config.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsM3Config.java
rename to 
hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsM3Config.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsPrometheusConfig.java
 
b/hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsPrometheusConfig.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsPrometheusConfig.java
rename to 
hudi-common/src/main/java/org/apache/hudi/config/metrics/HoodieMetricsPrometheusConfig.java
diff --git 
a/hudi-common/src/main/java/org/apache/hudi/metadata/BaseTableMetadata.java 
b/hudi-common/src/main/java/org/apache/hudi/metadata/BaseTableMetadata.java
index ccb0968b169..4702b8db056 100644
--- a/hudi-common/src/main/java/org/apache/hudi/metadata/BaseTableMetadata.java
+++ b/hudi-common/src/main/java/org/apache/hudi/metadata/BaseTableMetadata.java
@@ -27,7 +27,6 @@ import org.apache.hudi.common.config.HoodieMetadataConfig;
 import org.apache.hudi.common.engine.HoodieEngineContext;
 import org.apache.hudi.common.engine.HoodieLocalEngineContext;
 import org.apache.hudi.common.fs.FSUtils;
-import org.apache.hudi.common.metrics.Registry;
 import org.apache.hudi.common.model.HoodieRecord;
 import org.apache.hudi.common.table.HoodieTableMetaClient;
 import org.apache.hudi.common.model.HoodieRecordGlobalLocation;
@@ -39,6 +38,7 @@ import org.apache.hudi.common.util.collection.Pair;
 import org.apache.hudi.common.util.hash.ColumnIndexID;
 import org.apache.hudi.common.util.hash.FileIndexID;
 import org.apache.hudi.common.util.hash.PartitionIndexID;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.exception.HoodieIOException;
 import org.apache.hudi.exception.HoodieMetadataException;
 
@@ -97,7 +97,7 @@ public abstract class BaseTableMetadata extends 
AbstractHoodieTableMetadata {
     this.isMetadataTableInitialized = 
dataMetaClient.getTableConfig().isMetadataTableAvailable();
 
     if (metadataConfig.enableMetrics()) {
-      this.metrics = Option.of(new 
HoodieMetadataMetrics(Registry.getRegistry("HoodieMetadata")));
+      this.metrics = Option.of(new 
HoodieMetadataMetrics(HoodieMetricsConfig.newBuilder().fromProperties(metadataConfig.getProps()).build()));
     } else {
       this.metrics = Option.empty();
     }
diff --git 
a/hudi-common/src/main/java/org/apache/hudi/metadata/HoodieMetadataMetrics.java 
b/hudi-common/src/main/java/org/apache/hudi/metadata/HoodieMetadataMetrics.java
index ca9bf7b0834..7b73fc6d2d7 100644
--- 
a/hudi-common/src/main/java/org/apache/hudi/metadata/HoodieMetadataMetrics.java
+++ 
b/hudi-common/src/main/java/org/apache/hudi/metadata/HoodieMetadataMetrics.java
@@ -18,13 +18,17 @@
 
 package org.apache.hudi.metadata;
 
-import org.apache.hudi.common.metrics.Registry;
 import org.apache.hudi.common.model.FileSlice;
 import org.apache.hudi.common.model.HoodieLogFile;
 import org.apache.hudi.common.table.HoodieTableMetaClient;
 import org.apache.hudi.common.table.view.HoodieTableFileSystemView;
+import org.apache.hudi.common.util.Option;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.exception.HoodieIOException;
+import org.apache.hudi.metrics.HoodieGauge;
+import org.apache.hudi.metrics.Metrics;
 
+import com.codahale.metrics.MetricRegistry;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
@@ -73,10 +77,12 @@ public class HoodieMetadataMetrics implements Serializable {
 
   private static final Logger LOG = 
LoggerFactory.getLogger(HoodieMetadataMetrics.class);
 
-  private final Registry metricsRegistry;
+  private final transient MetricRegistry metricsRegistry;
+  private final transient Metrics metrics;
 
-  public HoodieMetadataMetrics(Registry metricsRegistry) {
-    this.metricsRegistry = metricsRegistry;
+  public HoodieMetadataMetrics(HoodieMetricsConfig metricsConfig) {
+    this.metrics = Metrics.getInstance(metricsConfig);
+    this.metricsRegistry = metrics.getRegistry();
   }
 
   public Map<String, String> getStats(boolean detailed, HoodieTableMetaClient 
metaClient, HoodieTableMetadata metadata, Set<String> metadataPartitions) {
@@ -148,14 +154,15 @@ public class HoodieMetadataMetrics implements 
Serializable {
 
   protected void incrementMetric(String action, long value) {
     LOG.info(String.format("Updating metadata metrics (%s=%d) in %s", action, 
value, metricsRegistry));
-    metricsRegistry.add(action, value);
+    Option<HoodieGauge<Long>> gaugeOpt = metrics.registerGauge(action);
+    gaugeOpt.ifPresent(gauge -> gauge.setValue(gauge.getValue() + value));
   }
 
   protected void setMetric(String action, long value) {
-    metricsRegistry.set(action, value);
+    metrics.registerGauge(action, value);
   }
 
-  public Registry registry() {
+  public MetricRegistry registry() {
     return metricsRegistry;
   }
 }
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/ConsoleMetricsReporter.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/ConsoleMetricsReporter.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/ConsoleMetricsReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/ConsoleMetricsReporter.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/HoodieGauge.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/HoodieGauge.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/HoodieGauge.java
rename to hudi-common/src/main/java/org/apache/hudi/metrics/HoodieGauge.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/InMemoryMetricsReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/InMemoryMetricsReporter.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/InMemoryMetricsReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/InMemoryMetricsReporter.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/JmxMetricsReporter.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/JmxMetricsReporter.java
similarity index 96%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/JmxMetricsReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/JmxMetricsReporter.java
index c64d5fd6b51..b341fc356f1 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/JmxMetricsReporter.java
+++ b/hudi-common/src/main/java/org/apache/hudi/metrics/JmxMetricsReporter.java
@@ -18,7 +18,7 @@
 
 package org.apache.hudi.metrics;
 
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.exception.HoodieException;
 
 import com.codahale.metrics.MetricRegistry;
@@ -41,7 +41,7 @@ public class JmxMetricsReporter extends MetricsReporter {
   private final MetricRegistry registry;
   private JmxReporterServer jmxReporterServer;
 
-  public JmxMetricsReporter(HoodieWriteConfig config, MetricRegistry registry) 
{
+  public JmxMetricsReporter(HoodieMetricsConfig config, MetricRegistry 
registry) {
     try {
       this.registry = registry;
       // Check the host and port here
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/JmxReporterServer.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/JmxReporterServer.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/JmxReporterServer.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/JmxReporterServer.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricUtils.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/MetricUtils.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricUtils.java
rename to hudi-common/src/main/java/org/apache/hudi/metrics/MetricUtils.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/Metrics.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/Metrics.java
similarity index 80%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/Metrics.java
rename to hudi-common/src/main/java/org/apache/hudi/metrics/Metrics.java
index ef088091732..17e21254593 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/Metrics.java
+++ b/hudi-common/src/main/java/org/apache/hudi/metrics/Metrics.java
@@ -19,9 +19,10 @@
 package org.apache.hudi.metrics;
 
 import org.apache.hudi.common.metrics.Registry;
+import org.apache.hudi.common.table.HoodieTableMetaClient;
 import org.apache.hudi.common.util.Option;
 import org.apache.hudi.common.util.StringUtils;
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.hadoop.fs.HadoopFSUtils;
 
 import com.codahale.metrics.MetricRegistry;
@@ -33,9 +34,9 @@ import org.slf4j.LoggerFactory;
 
 import java.io.IOException;
 import java.util.ArrayList;
-import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
+import java.util.concurrent.ConcurrentHashMap;
 
 /**
  * This is the main class of the metrics system.
@@ -44,15 +45,16 @@ public class Metrics {
 
   private static final Logger LOG = LoggerFactory.getLogger(Metrics.class);
 
-  private static final Map<String, Metrics> METRICS_INSTANCE_PER_BASEPATH = 
new HashMap<>();
+  private static final Map<String, Metrics> METRICS_INSTANCE_PER_BASEPATH = 
new ConcurrentHashMap<>();
 
   private final MetricRegistry registry;
   private final List<MetricsReporter> reporters;
   private final String commonMetricPrefix;
+  private final String basePath;
   private boolean initialized = false;
   private transient Thread shutdownThread = null;
 
-  public Metrics(HoodieWriteConfig metricConfig) {
+  public Metrics(HoodieMetricsConfig metricConfig) {
     registry = new MetricRegistry();
     commonMetricPrefix = metricConfig.getMetricReporterMetricsNamePrefix();
     reporters = new ArrayList<>();
@@ -65,6 +67,7 @@ public class Metrics {
       throw new RuntimeException("Cannot initialize Reporters.");
     }
     reporters.forEach(MetricsReporter::start);
+    basePath = getBasePath(metricConfig);
 
     shutdownThread = new Thread(() -> shutdown(true));
     Runtime.getRuntime().addShutdownHook(shutdownThread);
@@ -75,8 +78,8 @@ public class Metrics {
     registerGauges(Registry.getAllMetrics(true, true), 
Option.of(commonMetricPrefix));
   }
 
-  public static synchronized Metrics getInstance(HoodieWriteConfig 
metricConfig) {
-    String basePath = metricConfig.getBasePath();
+  public static synchronized Metrics getInstance(HoodieMetricsConfig 
metricConfig) {
+    String basePath = getBasePath(metricConfig);
     if (METRICS_INSTANCE_PER_BASEPATH.containsKey(basePath)) {
       return METRICS_INSTANCE_PER_BASEPATH.get(basePath);
     }
@@ -92,12 +95,12 @@ public class Metrics {
     METRICS_INSTANCE_PER_BASEPATH.clear();
   }
 
-  private List<MetricsReporter> 
addAdditionalMetricsExporters(HoodieWriteConfig metricConfig) {
+  private List<MetricsReporter> 
addAdditionalMetricsExporters(HoodieMetricsConfig metricConfig) {
     List<MetricsReporter> reporterList = new ArrayList<>();
     List<String> propPathList = 
StringUtils.split(metricConfig.getMetricReporterFileBasedConfigs(), ",");
     try (FileSystem fs = HadoopFSUtils.getFs(propPathList.get(0), new 
Configuration())) {
       for (String propPath : propPathList) {
-        HoodieWriteConfig secondarySourceConfig = 
HoodieWriteConfig.newBuilder().fromInputStream(
+        HoodieMetricsConfig secondarySourceConfig = 
HoodieMetricsConfig.newBuilder().fromInputStream(
             fs.open(new 
Path(propPath))).withPath(metricConfig.getBasePath()).build();
         Option<MetricsReporter> reporter = 
MetricsReporterFactory.createReporter(secondarySourceConfig, registry);
         if (reporter.isPresent()) {
@@ -155,15 +158,21 @@ public class Metrics {
     metricsMap.forEach((k, v) -> registerGauge(metricPrefix + k, v));
   }
 
-  public void registerGauge(String metricName, final long value) {
+  public Option<HoodieGauge<Long>> registerGauge(String metricName, final long 
value) {
+    HoodieGauge<Long> gauge = null;
     try {
-      HoodieGauge guage = (HoodieGauge) registry.gauge(metricName, () -> new 
HoodieGauge<>(value));
-      guage.setValue(value);
+      gauge = (HoodieGauge) registry.gauge(metricName, () -> new 
HoodieGauge<>(value));
+      gauge.setValue(value);
     } catch (Exception e) {
       // Here we catch all exception, so the major upsert pipeline will not be 
affected if the
       // metrics system has some issues.
       LOG.error("Failed to send metrics: ", e);
     }
+    return Option.ofNullable(gauge);
+  }
+
+  public Option<HoodieGauge<Long>> registerGauge(String metricName) {
+    return registerGauge(metricName, 0);
   }
 
   public MetricRegistry getRegistry() {
@@ -176,4 +185,16 @@ public class Metrics {
     }
     return false;
   }
+
+  /**
+   * Use the same base path as the hudi table so that Metrics instance is 
shared.
+   */
+  private static String getBasePath(HoodieMetricsConfig metricsConfig) {
+    String basePath = metricsConfig.getBasePath();
+    if (basePath.endsWith(HoodieTableMetaClient.METADATA_TABLE_FOLDER_PATH)) {
+      String toRemoveSuffix = Path.SEPARATOR + 
HoodieTableMetaClient.METADATA_TABLE_FOLDER_PATH;
+      basePath = basePath.substring(0, basePath.length() - 
toRemoveSuffix.length());
+    }
+    return basePath;
+  }
 }
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsGraphiteReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/MetricsGraphiteReporter.java
similarity index 84%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsGraphiteReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/MetricsGraphiteReporter.java
index c62edea8b1c..e3acab9a90b 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsGraphiteReporter.java
+++ 
b/hudi-common/src/main/java/org/apache/hudi/metrics/MetricsGraphiteReporter.java
@@ -18,7 +18,7 @@
 
 package org.apache.hudi.metrics;
 
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 
 import com.codahale.metrics.MetricFilter;
 import com.codahale.metrics.MetricRegistry;
@@ -38,25 +38,25 @@ public class MetricsGraphiteReporter extends 
MetricsReporter {
   private static final Logger LOG = 
LoggerFactory.getLogger(MetricsGraphiteReporter.class);
   private final MetricRegistry registry;
   private final GraphiteReporter graphiteReporter;
-  private final HoodieWriteConfig config;
+  private final HoodieMetricsConfig metricsConfig;
   private String serverHost;
   private int serverPort;
   private final int periodSeconds;
 
-  public MetricsGraphiteReporter(HoodieWriteConfig config, MetricRegistry 
registry) {
+  public MetricsGraphiteReporter(HoodieMetricsConfig metricsConfig, 
MetricRegistry registry) {
     this.registry = registry;
-    this.config = config;
+    this.metricsConfig = metricsConfig;
 
     // Check the serverHost and serverPort here
-    this.serverHost = config.getGraphiteServerHost();
-    this.serverPort = config.getGraphiteServerPort();
+    this.serverHost = metricsConfig.getGraphiteServerHost();
+    this.serverPort = metricsConfig.getGraphiteServerPort();
     if (serverHost == null || serverPort == 0) {
       throw new RuntimeException(String.format("Graphite cannot be initialized 
with serverHost[%s] and serverPort[%s].",
           serverHost, serverPort));
     }
 
     this.graphiteReporter = createGraphiteReport();
-    this.periodSeconds = config.getGraphiteReportPeriodSeconds();
+    this.periodSeconds = metricsConfig.getGraphiteReportPeriodSeconds();
   }
 
   @Override
@@ -79,7 +79,7 @@ public class MetricsGraphiteReporter extends MetricsReporter {
 
   private GraphiteReporter createGraphiteReport() {
     Graphite graphite = new Graphite(new InetSocketAddress(serverHost, 
serverPort));
-    String reporterPrefix = config.getGraphiteMetricPrefix();
+    String reporterPrefix = metricsConfig.getGraphiteMetricPrefix();
     return 
GraphiteReporter.forRegistry(registry).prefixedWith(reporterPrefix).convertRatesTo(TimeUnit.SECONDS)
         
.convertDurationsTo(TimeUnit.MILLISECONDS).filter(MetricFilter.ALL).build(graphite);
   }
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsReporter.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/MetricsReporter.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsReporter.java
rename to hudi-common/src/main/java/org/apache/hudi/metrics/MetricsReporter.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsReporterFactory.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/MetricsReporterFactory.java
similarity index 73%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsReporterFactory.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/MetricsReporterFactory.java
index 0d20337fa5c..455cf8de1c5 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsReporterFactory.java
+++ 
b/hudi-common/src/main/java/org/apache/hudi/metrics/MetricsReporterFactory.java
@@ -21,10 +21,8 @@ package org.apache.hudi.metrics;
 import org.apache.hudi.common.util.Option;
 import org.apache.hudi.common.util.ReflectionUtils;
 import org.apache.hudi.common.util.StringUtils;
-import org.apache.hudi.config.HoodieWriteConfig;
 import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.exception.HoodieException;
-import org.apache.hudi.metrics.cloudwatch.CloudWatchMetricsReporter;
 import org.apache.hudi.metrics.custom.CustomizableMetricsReporter;
 import org.apache.hudi.metrics.datadog.DatadogMetricsReporter;
 import org.apache.hudi.metrics.m3.M3MetricsReporter;
@@ -44,20 +42,20 @@ public class MetricsReporterFactory {
 
   private static final Logger LOG = 
LoggerFactory.getLogger(MetricsReporterFactory.class);
 
-  public static Option<MetricsReporter> createReporter(HoodieWriteConfig 
config, MetricRegistry registry) {
-    String reporterClassName = config.getMetricReporterClassName();
+  public static Option<MetricsReporter> createReporter(HoodieMetricsConfig 
metricsConfig, MetricRegistry registry) {
+    String reporterClassName = metricsConfig.getMetricReporterClassName();
 
     if (!StringUtils.isNullOrEmpty(reporterClassName)) {
       Object instance = ReflectionUtils.loadClass(
-          reporterClassName, new Class<?>[] {Properties.class, 
MetricRegistry.class}, config.getProps(), registry);
+          reporterClassName, new Class<?>[] {Properties.class, 
MetricRegistry.class}, metricsConfig.getProps(), registry);
       if (!(instance instanceof CustomizableMetricsReporter)) {
-        throw new HoodieException(config.getMetricReporterClassName()
+        throw new HoodieException(metricsConfig.getMetricReporterClassName()
             + " is not a subclass of CustomizableMetricsReporter");
       }
       return Option.of((MetricsReporter) instance);
     }
 
-    MetricsReporterType type = config.getMetricsReporterType();
+    MetricsReporterType type = metricsConfig.getMetricsReporterType();
     MetricsReporter reporter = null;
     if (type == null) {
       LOG.warn(String.format("Metric creation failed. %s is not configured",
@@ -67,31 +65,32 @@ public class MetricsReporterFactory {
 
     switch (type) {
       case GRAPHITE:
-        reporter = new MetricsGraphiteReporter(config, registry);
+        reporter = new MetricsGraphiteReporter(metricsConfig, registry);
         break;
       case INMEMORY:
         reporter = new InMemoryMetricsReporter();
         break;
       case JMX:
-        reporter = new JmxMetricsReporter(config, registry);
+        reporter = new JmxMetricsReporter(metricsConfig, registry);
         break;
       case DATADOG:
-        reporter = new DatadogMetricsReporter(config, registry);
+        reporter = new DatadogMetricsReporter(metricsConfig, registry);
         break;
       case PROMETHEUS_PUSHGATEWAY:
-        reporter = new PushGatewayMetricsReporter(config, registry);
+        reporter = new PushGatewayMetricsReporter(metricsConfig, registry);
         break;
       case PROMETHEUS:
-        reporter = new PrometheusReporter(config, registry);
+        reporter = new PrometheusReporter(metricsConfig, registry);
         break;
       case CONSOLE:
         reporter = new ConsoleMetricsReporter(registry);
         break;
       case CLOUDWATCH:
-        reporter = new CloudWatchMetricsReporter(config, registry);
+        reporter = (MetricsReporter) 
ReflectionUtils.loadClass("org.apache.hudi.metrics.cloudwatch.CloudWatchMetricsReporter",
+            new Class[]{HoodieMetricsConfig.class, MetricRegistry.class}, 
metricsConfig, registry);
         break;
       case M3:
-        reporter = new M3MetricsReporter(config, registry);
+        reporter = new M3MetricsReporter(metricsConfig, registry);
         break;
       default:
         LOG.error("Reporter type[" + type + "] is not supported.");
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsReporterType.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/MetricsReporterType.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/MetricsReporterType.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/MetricsReporterType.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/custom/CustomizableMetricsReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/custom/CustomizableMetricsReporter.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/custom/CustomizableMetricsReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/custom/CustomizableMetricsReporter.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogHttpClient.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogHttpClient.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogHttpClient.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogHttpClient.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogMetricsReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogMetricsReporter.java
similarity index 95%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogMetricsReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogMetricsReporter.java
index 3f598f34a2d..e13539d5924 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogMetricsReporter.java
+++ 
b/hudi-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogMetricsReporter.java
@@ -21,7 +21,7 @@ package org.apache.hudi.metrics.datadog;
 import org.apache.hudi.common.util.Option;
 import org.apache.hudi.common.util.StringUtils;
 import org.apache.hudi.common.util.ValidationUtils;
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.MetricsReporter;
 import org.apache.hudi.metrics.datadog.DatadogHttpClient.ApiSite;
 
@@ -43,7 +43,7 @@ public class DatadogMetricsReporter extends MetricsReporter {
   private final DatadogReporter reporter;
   private final int reportPeriodSeconds;
 
-  public DatadogMetricsReporter(HoodieWriteConfig config, MetricRegistry 
registry) {
+  public DatadogMetricsReporter(HoodieMetricsConfig config, MetricRegistry 
registry) {
     reportPeriodSeconds = config.getDatadogReportPeriodSeconds();
     ApiSite apiSite = config.getDatadogApiSite();
     String apiKey = config.getDatadogApiKey();
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogReporter.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/datadog/DatadogReporter.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/m3/M3MetricsReporter.java
 b/hudi-common/src/main/java/org/apache/hudi/metrics/m3/M3MetricsReporter.java
similarity index 88%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/m3/M3MetricsReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/m3/M3MetricsReporter.java
index a658476ef75..869b721f4d8 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/m3/M3MetricsReporter.java
+++ 
b/hudi-common/src/main/java/org/apache/hudi/metrics/m3/M3MetricsReporter.java
@@ -29,7 +29,7 @@ import java.util.Arrays;
 import java.util.HashMap;
 import java.util.Map;
 import java.util.concurrent.TimeUnit;
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.MetricsReporter;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
@@ -40,18 +40,18 @@ import org.slf4j.LoggerFactory;
 public class M3MetricsReporter extends MetricsReporter {
 
   private static final Logger LOG = 
LoggerFactory.getLogger(M3MetricsReporter.class);
-  private final HoodieWriteConfig config;
+  private final HoodieMetricsConfig metricsConfig;
   private final MetricRegistry registry;
   private final ImmutableMap<String, String> tags;
 
-  public M3MetricsReporter(HoodieWriteConfig config, MetricRegistry registry) {
-    this.config = config;
+  public M3MetricsReporter(HoodieMetricsConfig metricsConfig, MetricRegistry 
registry) {
+    this.metricsConfig = metricsConfig;
     this.registry = registry;
 
     ImmutableMap.Builder tagBuilder = new ImmutableMap.Builder<>();
-    tagBuilder.putAll(parseOptionalTags(config.getM3Tags()));
-    tagBuilder.put("service", config.getM3Service());
-    tagBuilder.put("env", config.getM3Env());
+    tagBuilder.putAll(parseOptionalTags(metricsConfig.getM3Tags()));
+    tagBuilder.put("service", metricsConfig.getM3Service());
+    tagBuilder.put("env", metricsConfig.getM3Env());
     this.tags = tagBuilder.build();
     LOG.info(String.format("Building M3 Reporter with M3 tags mapping: %s", 
tags));
   }
@@ -93,7 +93,7 @@ public class M3MetricsReporter extends MetricsReporter {
     synchronized (this) {
       try (Scope scope = new RootScopeBuilder()
           .reporter(new M3Reporter.Builder(
-              new InetSocketAddress(config.getM3ServerHost(), 
config.getM3ServerPort()))
+              new InetSocketAddress(metricsConfig.getM3ServerHost(), 
metricsConfig.getM3ServerPort()))
               .includeHost(true).commonTags(tags)
               .build())
           .reportEvery(Duration.ofSeconds(Integer.MAX_VALUE))
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/m3/M3ScopeReporterAdaptor.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/m3/M3ScopeReporterAdaptor.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/m3/M3ScopeReporterAdaptor.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/m3/M3ScopeReporterAdaptor.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/prometheus/PrometheusReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/prometheus/PrometheusReporter.java
similarity index 92%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/prometheus/PrometheusReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/prometheus/PrometheusReporter.java
index 34fd7a07f65..44fd9f9175d 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/prometheus/PrometheusReporter.java
+++ 
b/hudi-common/src/main/java/org/apache/hudi/metrics/prometheus/PrometheusReporter.java
@@ -19,7 +19,7 @@
 package org.apache.hudi.metrics.prometheus;
 
 import org.apache.hudi.common.util.StringUtils;
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.exception.HoodieException;
 import org.apache.hudi.metrics.MetricsReporter;
 
@@ -55,15 +55,15 @@ public class PrometheusReporter extends MetricsReporter {
   private final CollectorRegistry collectorRegistry;
   private final int serverPort;
 
-  public PrometheusReporter(HoodieWriteConfig config, MetricRegistry registry) 
{
-    this.serverPort = config.getPrometheusPort();
+  public PrometheusReporter(HoodieMetricsConfig metricsConfig, MetricRegistry 
registry) {
+    this.serverPort = metricsConfig.getPrometheusPort();
     if (!PORT_TO_SERVER.containsKey(serverPort) || 
!PORT_TO_COLLECTOR_REGISTRY.containsKey(serverPort)) {
       startHttpServer(serverPort);
     }
     List<String> labelNames = new ArrayList<>();
     List<String> labelValues = new ArrayList<>();
-    if (StringUtils.nonEmpty(config.getPushGatewayLabels())) {
-      LABEL_PATTERN.splitAsStream(config.getPushGatewayLabels().trim()).map(s 
-> s.split(":", 2))
+    if (StringUtils.nonEmpty(metricsConfig.getPushGatewayLabels())) {
+      
LABEL_PATTERN.splitAsStream(metricsConfig.getPushGatewayLabels().trim()).map(s 
-> s.split(":", 2))
           .forEach(parts -> {
             labelNames.add(parts[0]);
             labelValues.add(parts[1]);
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayMetricsReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayMetricsReporter.java
similarity index 79%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayMetricsReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayMetricsReporter.java
index 805e5d7c0d7..ddd4155bce9 100644
--- 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayMetricsReporter.java
+++ 
b/hudi-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayMetricsReporter.java
@@ -19,7 +19,7 @@
 package org.apache.hudi.metrics.prometheus;
 
 import org.apache.hudi.common.util.StringUtils;
-import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.MetricUtils;
 import org.apache.hudi.metrics.MetricsReporter;
 
@@ -40,15 +40,15 @@ public class PushGatewayMetricsReporter extends 
MetricsReporter {
   private final Map<String, String> configuredLabels;
   private final boolean randomSuffix;
 
-  public PushGatewayMetricsReporter(HoodieWriteConfig config, MetricRegistry 
registry) {
+  public PushGatewayMetricsReporter(HoodieMetricsConfig metricsConfig, 
MetricRegistry registry) {
 
-    String serverHost = config.getPushGatewayHost();
-    int serverPort = config.getPushGatewayPort();
-    periodSeconds = config.getPushGatewayReportPeriodSeconds();
-    deleteShutdown = config.getPushGatewayDeleteOnShutdown();
-    configuredJobName = config.getPushGatewayJobName();
-    configuredLabels = 
Collections.unmodifiableMap(parseLabels(config.getPushGatewayLabels()));
-    randomSuffix = config.getPushGatewayRandomJobNameSuffix();
+    String serverHost = metricsConfig.getPushGatewayHost();
+    int serverPort = metricsConfig.getPushGatewayPort();
+    periodSeconds = metricsConfig.getPushGatewayReportPeriodSeconds();
+    deleteShutdown = metricsConfig.getPushGatewayDeleteOnShutdown();
+    configuredJobName = metricsConfig.getPushGatewayJobName();
+    configuredLabels = 
Collections.unmodifiableMap(parseLabels(metricsConfig.getPushGatewayLabels()));
+    randomSuffix = metricsConfig.getPushGatewayRandomJobNameSuffix();
 
     pushGatewayReporter = new PushGatewayReporter(
         registry,
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayReporter.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/prometheus/PushGatewayReporter.java
diff --git 
a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/userdefined/AbstractUserDefinedMetricsReporter.java
 
b/hudi-common/src/main/java/org/apache/hudi/metrics/userdefined/AbstractUserDefinedMetricsReporter.java
similarity index 100%
rename from 
hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metrics/userdefined/AbstractUserDefinedMetricsReporter.java
rename to 
hudi-common/src/main/java/org/apache/hudi/metrics/userdefined/AbstractUserDefinedMetricsReporter.java
diff --git 
a/hudi-utilities/src/main/java/org/apache/hudi/utilities/deltastreamer/HoodieDeltaStreamerMetrics.java
 
b/hudi-utilities/src/main/java/org/apache/hudi/utilities/deltastreamer/HoodieDeltaStreamerMetrics.java
index 0a15745483d..cd7867edf3e 100644
--- 
a/hudi-utilities/src/main/java/org/apache/hudi/utilities/deltastreamer/HoodieDeltaStreamerMetrics.java
+++ 
b/hudi-utilities/src/main/java/org/apache/hudi/utilities/deltastreamer/HoodieDeltaStreamerMetrics.java
@@ -20,6 +20,7 @@
 package org.apache.hudi.utilities.deltastreamer;
 
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.utilities.streamer.HoodieStreamerMetrics;
 
 /**
@@ -28,7 +29,12 @@ import 
org.apache.hudi.utilities.streamer.HoodieStreamerMetrics;
  */
 @Deprecated
 public class HoodieDeltaStreamerMetrics extends HoodieStreamerMetrics {
+
   public HoodieDeltaStreamerMetrics(HoodieWriteConfig writeConfig) {
-    super(writeConfig);
+    super(writeConfig.getMetricsConfig());
+  }
+
+  public HoodieDeltaStreamerMetrics(HoodieMetricsConfig metricsConfig) {
+    super(metricsConfig);
   }
 }
diff --git 
a/hudi-utilities/src/main/java/org/apache/hudi/utilities/ingestion/HoodieIngestionMetrics.java
 
b/hudi-utilities/src/main/java/org/apache/hudi/utilities/ingestion/HoodieIngestionMetrics.java
index bd31b8f2b46..3d07610993d 100644
--- 
a/hudi-utilities/src/main/java/org/apache/hudi/utilities/ingestion/HoodieIngestionMetrics.java
+++ 
b/hudi-utilities/src/main/java/org/apache/hudi/utilities/ingestion/HoodieIngestionMetrics.java
@@ -19,6 +19,7 @@
 package org.apache.hudi.utilities.ingestion;
 
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 
 import com.codahale.metrics.Timer;
 
@@ -29,9 +30,13 @@ import java.io.Serializable;
  */
 public abstract class HoodieIngestionMetrics implements Serializable {
 
-  protected final HoodieWriteConfig writeConfig;
+  protected final HoodieMetricsConfig writeConfig;
 
   public HoodieIngestionMetrics(HoodieWriteConfig writeConfig) {
+    this(writeConfig.getMetricsConfig());
+  }
+
+  public HoodieIngestionMetrics(HoodieMetricsConfig writeConfig) {
     this.writeConfig = writeConfig;
   }
 
diff --git 
a/hudi-utilities/src/main/java/org/apache/hudi/utilities/streamer/HoodieStreamerMetrics.java
 
b/hudi-utilities/src/main/java/org/apache/hudi/utilities/streamer/HoodieStreamerMetrics.java
index 7f1e0a29792..fcbf431ed6f 100644
--- 
a/hudi-utilities/src/main/java/org/apache/hudi/utilities/streamer/HoodieStreamerMetrics.java
+++ 
b/hudi-utilities/src/main/java/org/apache/hudi/utilities/streamer/HoodieStreamerMetrics.java
@@ -20,6 +20,7 @@
 package org.apache.hudi.utilities.streamer;
 
 import org.apache.hudi.config.HoodieWriteConfig;
+import org.apache.hudi.config.metrics.HoodieMetricsConfig;
 import org.apache.hudi.metrics.Metrics;
 import org.apache.hudi.utilities.ingestion.HoodieIngestionMetrics;
 
@@ -37,6 +38,10 @@ public class HoodieStreamerMetrics extends 
HoodieIngestionMetrics {
   private transient Timer metaSyncTimer;
 
   public HoodieStreamerMetrics(HoodieWriteConfig writeConfig) {
+    this(writeConfig.getMetricsConfig());
+  }
+
+  public HoodieStreamerMetrics(HoodieMetricsConfig writeConfig) {
     super(writeConfig);
     if (writeConfig.isMetricsOn()) {
       metrics = Metrics.getInstance(writeConfig);
diff --git 
a/hudi-utilities/src/main/java/org/apache/hudi/utilities/streamer/StreamSync.java
 
b/hudi-utilities/src/main/java/org/apache/hudi/utilities/streamer/StreamSync.java
index 42d218a5b4a..2b0d94da74a 100644
--- 
a/hudi-utilities/src/main/java/org/apache/hudi/utilities/streamer/StreamSync.java
+++ 
b/hudi-utilities/src/main/java/org/apache/hudi/utilities/streamer/StreamSync.java
@@ -304,7 +304,7 @@ public class StreamSync implements Serializable, Closeable {
     this.conf = conf;
 
     HoodieWriteConfig hoodieWriteConfig = getHoodieClientConfig();
-    this.metrics = (HoodieIngestionMetrics) 
ReflectionUtils.loadClass(cfg.ingestionMetricsClass, hoodieWriteConfig);
+    this.metrics = (HoodieIngestionMetrics) 
ReflectionUtils.loadClass(cfg.ingestionMetricsClass, 
hoodieWriteConfig.getMetricsConfig());
     this.hoodieMetrics = new HoodieMetrics(hoodieWriteConfig);
     if (props.getBoolean(ERROR_TABLE_ENABLED.key(), 
ERROR_TABLE_ENABLED.defaultValue())) {
       this.errorTableWriter = ErrorTableUtils.getErrorTableWriter(cfg, 
sparkSession, props, hoodieSparkContext, fs);

Reply via email to