From 5aeaa63b6de2c1d3e2b2bfcea6df9caa61392c6f Mon Sep 17 00:00:00 2001 From: Xu Cang Date: Wed, 22 Aug 2018 23:21:04 -0700 Subject: [PATCH] HBASE-18549 Add metrics for failed replication queue recovery to global source --- .../regionserver/MetricsReplicationSourceSource.java | 2 ++ .../regionserver/MetricsReplicationGlobalSourceSource.java | 8 +++++++- .../regionserver/MetricsReplicationSourceSourceImpl.java | 3 +++ .../hbase/replication/regionserver/MetricsSource.java | 4 ++++ .../replication/regionserver/ReplicationSourceManager.java | 8 +++++++- .../hadoop/hbase/replication/TestReplicationEndpoint.java | 14 +++++++++++++- 6 files changed, 36 insertions(+), 3 deletions(-) diff --git a/hbase-hadoop-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationSourceSource.java b/hbase-hadoop-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationSourceSource.java index 10451138b4..4ea5aae8ad 100644 --- a/hbase-hadoop-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationSourceSource.java +++ b/hbase-hadoop-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationSourceSource.java @@ -53,6 +53,7 @@ public interface MetricsReplicationSourceSource extends BaseSource { public static final String SOURCE_REPEATED_LOG_FILE_BYTES = "source.repeatedLogFileBytes"; public static final String SOURCE_COMPLETED_LOGS = "source.completedLogs"; public static final String SOURCE_COMPLETED_RECOVERY_QUEUES = "source.completedRecoverQueues"; + public static final String SOURCE_FAILD_RECOVERY_QUEUES = "source.failedRecoverQueues"; void setLastShippedAge(long age); void incrSizeOfLogQueue(int size); @@ -76,4 +77,5 @@ public interface MetricsReplicationSourceSource extends BaseSource { void incrRepeatedFileBytes(final long bytes); void incrCompletedWAL(); void incrCompletedRecoveryQueue(); + void incrFailedRecoveryQueue(); } diff --git a/hbase-hadoop2-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationGlobalSourceSource.java b/hbase-hadoop2-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationGlobalSourceSource.java index 9a86cf2f9e..18ef0abdfc 100644 --- a/hbase-hadoop2-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationGlobalSourceSource.java +++ b/hbase-hadoop2-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationGlobalSourceSource.java @@ -52,6 +52,7 @@ public class MetricsReplicationGlobalSourceSource implements MetricsReplicationS private final MutableFastCounter repeatedFileBytes; private final MutableFastCounter completedWAL; private final MutableFastCounter completedRecoveryQueue; + private final MutableFastCounter failedRecoveryQueue; public MetricsReplicationGlobalSourceSource(MetricsReplicationSourceImpl rms) { this.rms = rms; @@ -89,6 +90,8 @@ public class MetricsReplicationGlobalSourceSource implements MetricsReplicationS completedWAL = rms.getMetricsRegistry().getCounter(SOURCE_COMPLETED_LOGS, 0L); completedRecoveryQueue = rms.getMetricsRegistry() .getCounter(SOURCE_COMPLETED_RECOVERY_QUEUES, 0L); + failedRecoveryQueue = rms.getMetricsRegistry() + .getCounter(SOURCE_FAILD_RECOVERY_QUEUES, 0L); } @Override public void setLastShippedAge(long age) { @@ -199,7 +202,10 @@ public class MetricsReplicationGlobalSourceSource implements MetricsReplicationS public void incrCompletedRecoveryQueue() { completedRecoveryQueue.incr(1L); } - + @Override + public void incrFailedRecoveryQueue() { + failedRecoveryQueue.incr(1L); + } @Override public void init() { rms.init(); diff --git a/hbase-hadoop2-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationSourceSourceImpl.java b/hbase-hadoop2-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationSourceSourceImpl.java index 719c91631d..0ad50524f1 100644 --- a/hbase-hadoop2-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationSourceSourceImpl.java +++ b/hbase-hadoop2-compat/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsReplicationSourceSourceImpl.java @@ -257,6 +257,9 @@ public class MetricsReplicationSourceSourceImpl implements MetricsReplicationSou completedRecoveryQueue.incr(1L); } + @Override + public void incrFailedRecoveryQueue() {/*no op*/} + @Override public void init() { rms.init(); diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsSource.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsSource.java index 906f0c6d33..830ebe182d 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsSource.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/MetricsSource.java @@ -326,6 +326,10 @@ public class MetricsSource implements BaseSource { globalSourceSource.incrCompletedRecoveryQueue(); } + public void incrFailedRecoveryQueue() { + globalSourceSource.incrFailedRecoveryQueue(); + } + @Override public void init() { singleSourceSource.init(); diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSourceManager.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSourceManager.java index 428ec98df9..8d62e5e3e4 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSourceManager.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSourceManager.java @@ -45,6 +45,7 @@ import java.util.stream.Collectors; import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.fs.FileSystem; import org.apache.hadoop.fs.Path; +import org.apache.hadoop.hbase.CompatibilitySingletonFactory; import org.apache.hadoop.hbase.HConstants; import org.apache.hadoop.hbase.Server; import org.apache.hadoop.hbase.ServerName; @@ -820,6 +821,8 @@ public class ReplicationSourceManager implements ReplicationListener { try { this.executor.execute(transfer); } catch (RejectedExecutionException ex) { + CompatibilitySingletonFactory.getInstance(MetricsReplicationSourceFactory.class) + .getGlobalSource().incrFailedRecoveryQueue(); LOG.info("Cancelling the transfer of " + deadRS + " because of " + ex.getMessage()); } } @@ -891,7 +894,10 @@ public class ReplicationSourceManager implements ReplicationListener { queueStorage.removeReplicatorIfQueueIsEmpty(deadRS); } } catch (ReplicationException e) { - server.abort("Failed to claim queue from dead regionserver", e); + LOG.error("ReplicationException: cannot claim dead reagion's replication queue," + + " possible solution: check if znode size exceeds jute.maxBuffer value. " + + " If so, increase it for both client and server side."); + server.abort("Failed to claim queue from dead regionserver.", e); return; } // Copying over the failed queue is completed. diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/replication/TestReplicationEndpoint.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/replication/TestReplicationEndpoint.java index 5d833cc5a9..03fbb59f26 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/replication/TestReplicationEndpoint.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/replication/TestReplicationEndpoint.java @@ -17,7 +17,9 @@ */ package org.apache.hadoop.hbase.replication; +import static org.mockito.Mockito.doNothing; import static org.mockito.Mockito.mock; +import static org.mockito.Mockito.spy; import static org.mockito.Mockito.verify; import static org.mockito.Mockito.when; @@ -316,11 +318,17 @@ public class TestReplicationEndpoint extends TestReplicationBase { MetricsReplicationSourceImpl globalRms = mock(MetricsReplicationSourceImpl.class); when(globalRms.getMetricsRegistry()).thenReturn(mockRegistry); + MetricsReplicationSourceSource singleSourceSource = new MetricsReplicationSourceSourceImpl(singleRms, id); MetricsReplicationSourceSource globalSourceSource = new MetricsReplicationGlobalSourceSource(globalRms); + MetricsReplicationSourceSource spyglobalSourceSource = spy(globalSourceSource); + doNothing().when(spyglobalSourceSource).incrFailedRecoveryQueue(); + Map singleSourceSourceByTable = new HashMap<>(); - MetricsSource source = new MetricsSource(id, singleSourceSource, globalSourceSource, + MetricsSource source = new MetricsSource(id, singleSourceSource, spyglobalSourceSource, singleSourceSourceByTable); + + String gaugeName = "gauge"; String singleGaugeName = "source.id." + gaugeName; String globalGaugeName = "source." + gaugeName; @@ -340,6 +348,8 @@ public class TestReplicationEndpoint extends TestReplicationBase { source.removeMetric(gaugeName); source.setGauge(gaugeName, delta); source.updateHistogram(counterName, count); + source.incrFailedRecoveryQueue(); + verify(singleRms).decGauge(singleGaugeName, delta); verify(globalRms).decGauge(globalGaugeName, delta); @@ -357,6 +367,7 @@ public class TestReplicationEndpoint extends TestReplicationBase { verify(globalRms).setGauge(globalGaugeName, delta); verify(singleRms).updateHistogram(singleCounterName, count); verify(globalRms).updateHistogram(globalCounterName, count); + verify(spyglobalSourceSource).incrFailedRecoveryQueue(); //check singleSourceSourceByTable metrics. // singleSourceSourceByTable map entry will be created only @@ -373,6 +384,7 @@ public class TestReplicationEndpoint extends TestReplicationBase { // cannot put more concreate value here to verify because the age is arbitrary. // as long as it's greater than 0, we see it as correct answer. Assert.assertTrue(msr.getLastShippedAge() > 0); + } private void doPut(byte[] row) throws IOException { -- 2.14.3 (Apple Git-98)