Skip to content

Commit b8d7583

Browse files
author
Wangx
committed
Show the create time of the replication peer
1 parent d85574a commit b8d7583

File tree

9 files changed

+82
-7
lines changed

9 files changed

+82
-7
lines changed

hbase-client/src/main/java/org/apache/hadoop/hbase/replication/ReplicationPeerDescription.java

Lines changed: 15 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -32,6 +32,7 @@ public class ReplicationPeerDescription {
3232
private final boolean enabled;
3333
private final ReplicationPeerConfig config;
3434
private final SyncReplicationState syncReplicationState;
35+
private String createTime;
3536

3637
public ReplicationPeerDescription(String id, boolean enabled, ReplicationPeerConfig config,
3738
SyncReplicationState syncReplicationState) {
@@ -41,6 +42,19 @@ public ReplicationPeerDescription(String id, boolean enabled, ReplicationPeerCon
4142
this.syncReplicationState = syncReplicationState;
4243
}
4344

45+
public ReplicationPeerDescription(String id, boolean enabled, ReplicationPeerConfig config,
46+
SyncReplicationState syncReplicationState, String createTime) {
47+
this.id = id;
48+
this.enabled = enabled;
49+
this.config = config;
50+
this.syncReplicationState = syncReplicationState;
51+
this.createTime = createTime;
52+
}
53+
54+
public String getCreateTime() {
55+
return createTime;
56+
}
57+
4458
public String getPeerId() {
4559
return this.id;
4660
}
@@ -63,6 +77,7 @@ public String toString() {
6377
builder.append(", enabled : " + enabled);
6478
builder.append(", config : " + config);
6579
builder.append(", syncReplicationState : " + syncReplicationState);
80+
builder.append(", createTime : " + createTime);
6681
return builder.toString();
6782
}
6883
}

hbase-replication/src/main/java/org/apache/hadoop/hbase/replication/FSReplicationPeerStorage.java

Lines changed: 10 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -227,6 +227,16 @@ public ReplicationPeerConfig getPeerConfig(String peerId) throws ReplicationExce
227227
}
228228
}
229229

230+
@Override
231+
public long getPeerCreateTime(String peerId) throws ReplicationException {
232+
Path peerDir = getPeerDir(peerId);
233+
try {
234+
return fs.getFileStatus(peerDir).getModificationTime();
235+
} catch (IOException e) {
236+
throw new ReplicationException("Unable to get create time of the peer with id=" + peerId, e);
237+
}
238+
}
239+
230240
private Pair<SyncReplicationState, SyncReplicationState> getStateAndNewState(String peerId)
231241
throws IOException {
232242
Path peerDir = getPeerDir(peerId);

hbase-replication/src/main/java/org/apache/hadoop/hbase/replication/ReplicationPeerStorage.java

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -70,6 +70,12 @@ void updatePeerConfig(String peerId, ReplicationPeerConfig peerConfig)
7070
*/
7171
ReplicationPeerConfig getPeerConfig(String peerId) throws ReplicationException;
7272

73+
/**
74+
* Get the peer create time of a replication peer.
75+
* @throws ReplicationException if there are errors accessing the storage service.
76+
*/
77+
long getPeerCreateTime(String peerId) throws ReplicationException;
78+
7379
/**
7480
* Set the new sync replication state that we are going to transit to.
7581
* @throws ReplicationException if there are errors accessing the storage service.

hbase-replication/src/main/java/org/apache/hadoop/hbase/replication/ZKReplicationPeerStorage.java

Lines changed: 11 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -185,6 +185,17 @@ public ReplicationPeerConfig getPeerConfig(String peerId) throws ReplicationExce
185185
}
186186
}
187187

188+
@Override
189+
public long getPeerCreateTime(String peerId) throws ReplicationException {
190+
long createTimeIfNodeExists;
191+
try {
192+
createTimeIfNodeExists = ZKUtil.getCreateTimeIfNodeExists(zookeeper, getPeerNode(peerId));
193+
return createTimeIfNodeExists;
194+
} catch (KeeperException e) {
195+
throw new ReplicationException("Error getting create time for peer with id=" + peerId, e);
196+
}
197+
}
198+
188199
@Override
189200
public void setPeerNewSyncReplicationState(String peerId, SyncReplicationState state)
190201
throws ReplicationException {

hbase-server/src/main/jamon/org/apache/hadoop/hbase/tmpl/master/MasterStatusTmpl.jamon

Lines changed: 2 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -750,6 +750,7 @@ AssignmentManager assignmentManager = master.getAssignmentManager();
750750
</%java>
751751
<table class="table table-striped">
752752
<tr>
753+
<th>Create Time</th>
753754
<th>Peer Id</th>
754755
<th>Cluster Key</th>
755756
<th>Endpoint</th>
@@ -771,6 +772,7 @@ AssignmentManager assignmentManager = master.getAssignmentManager();
771772
ReplicationPeerConfig peerConfig = peer.getPeerConfig();
772773
</%java>
773774
<tr>
775+
<td><% peer.getCreateTime() %></td>
774776
<td><% peerId %></td>
775777
<td><% peerConfig.getClusterKey() %></td>
776778
<td><% peerConfig.getReplicationEndpointImpl() %></td>

hbase-server/src/main/java/org/apache/hadoop/hbase/master/replication/ReplicationPeerManager.java

Lines changed: 14 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -20,6 +20,8 @@
2020
import com.google.errorprone.annotations.RestrictedApi;
2121
import java.io.IOException;
2222
import java.net.URI;
23+
import java.text.DateFormat;
24+
import java.text.SimpleDateFormat;
2325
import java.util.ArrayList;
2426
import java.util.Collection;
2527
import java.util.EnumSet;
@@ -96,6 +98,8 @@ public class ReplicationPeerManager implements ConfigurationObserver {
9698

9799
private static final Logger LOG = LoggerFactory.getLogger(ReplicationPeerManager.class);
98100

101+
private static final DateFormat DATE_FORMAT = new SimpleDateFormat("yyyy-MM-dd HH:mm:ss");
102+
99103
private volatile ReplicationPeerStorage peerStorage;
100104

101105
private final ReplicationQueueStorage queueStorage;
@@ -288,8 +292,10 @@ public void addPeer(String peerId, ReplicationPeerConfig peerConfig, boolean ena
288292
? SyncReplicationState.DOWNGRADE_ACTIVE
289293
: SyncReplicationState.NONE;
290294
peerStorage.addPeer(peerId, copiedPeerConfig, enabled, syncReplicationState);
295+
long peerCreateTime = peerStorage.getPeerCreateTime(peerId);
296+
String peerCreateTimeStr = peerCreateTime == -1 ? "" : DATE_FORMAT.format(peerCreateTime);
291297
peers.put(peerId,
292-
new ReplicationPeerDescription(peerId, enabled, copiedPeerConfig, syncReplicationState));
298+
new ReplicationPeerDescription(peerId, enabled, copiedPeerConfig, syncReplicationState, peerCreateTimeStr));
293299
}
294300

295301
public void removePeer(String peerId) throws ReplicationException {
@@ -309,7 +315,7 @@ private void setPeerState(String peerId, boolean enabled) throws ReplicationExce
309315
}
310316
peerStorage.setPeerState(peerId, enabled);
311317
peers.put(peerId, new ReplicationPeerDescription(peerId, enabled, desc.getPeerConfig(),
312-
desc.getSyncReplicationState()));
318+
desc.getSyncReplicationState(), desc.getCreateTime()));
313319
}
314320

315321
public boolean getPeerState(String peerId) throws ReplicationException {
@@ -342,7 +348,7 @@ public void updatePeerConfig(String peerId, ReplicationPeerConfig peerConfig)
342348
ReplicationPeerConfig newPeerConfig = newPeerConfigBuilder.build();
343349
peerStorage.updatePeerConfig(peerId, newPeerConfig);
344350
peers.put(peerId, new ReplicationPeerDescription(peerId, desc.isEnabled(), newPeerConfig,
345-
desc.getSyncReplicationState()));
351+
desc.getSyncReplicationState(), desc.getCreateTime()));
346352
}
347353

348354
public List<ReplicationPeerDescription> listPeers(Pattern pattern) {
@@ -377,7 +383,7 @@ public void transitPeerSyncReplicationState(String peerId, SyncReplicationState
377383
if (desc.getSyncReplicationState() != newState) {
378384
// Only recreate the desc if this is not a retry
379385
peers.put(peerId,
380-
new ReplicationPeerDescription(peerId, desc.isEnabled(), desc.getPeerConfig(), newState));
386+
new ReplicationPeerDescription(peerId, desc.isEnabled(), desc.getPeerConfig(), newState, desc.getCreateTime()));
381387
}
382388
}
383389

@@ -692,7 +698,10 @@ public void run() {
692698
peerStorage.updatePeerConfig(peerId, peerConfig);
693699
boolean enabled = peerStorage.isPeerEnabled(peerId);
694700
SyncReplicationState state = peerStorage.getPeerSyncReplicationState(peerId);
695-
peers.put(peerId, new ReplicationPeerDescription(peerId, enabled, peerConfig, state));
701+
long peerCreateTime = peerStorage.getPeerCreateTime(peerId);
702+
String peerCreateTimeStr = peerCreateTime == -1 ? "" : DATE_FORMAT.format(peerCreateTime);
703+
peers.put(peerId,
704+
new ReplicationPeerDescription(peerId, enabled, peerConfig, state, peerCreateTimeStr));
696705
}
697706
return new ReplicationPeerManager(fs, zk, peerStorage, queueStorage, peers, conf, clusterId,
698707
pair.getSecond());

hbase-server/src/test/java/org/apache/hadoop/hbase/master/cleaner/TestLogsCleaner.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -207,7 +207,7 @@ public void testLogCleaning() throws Exception {
207207
}
208208
// Case 4: the newest 3 WALs will be kept because they are beyond the replication offset
209209
masterServices.getReplicationPeerManager().listPeers(null)
210-
.add(new ReplicationPeerDescription(peerId, true, null, null));
210+
.add(new ReplicationPeerDescription(peerId, true, null, null, null));
211211
queueStorage.setOffset(new ReplicationQueueId(server.getServerName(), peerId), fakeMachineName,
212212
new ReplicationGroupOffset(fakeMachineName + "." + (now - 3), 0), Collections.emptyMap());
213213
// Case 5: 5 Procedure WALs that are new, will stay

hbase-server/src/test/java/org/apache/hadoop/hbase/replication/master/TestReplicationLogCleaner.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -123,7 +123,7 @@ private static FileStatus createFileStatus(ServerName sn, int number) {
123123
}
124124

125125
private static ReplicationPeerDescription createPeer(String peerId) {
126-
return new ReplicationPeerDescription(peerId, true, null, null);
126+
return new ReplicationPeerDescription(peerId, true, null, null, null);
127127
}
128128

129129
private void addServer(ServerName serverName) {

hbase-zookeeper/src/main/java/org/apache/hadoop/hbase/zookeeper/ZKUtil.java

Lines changed: 22 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -168,6 +168,28 @@ public static int checkExists(ZKWatcher zkw, String znode) throws KeeperExceptio
168168
}
169169
}
170170

171+
/**
172+
* Get the create time if the specified node exists.
173+
* @param zkw zk reference
174+
* @param znode path of node to watch
175+
* @return create time of the node if it exists, -1 if does not exist
176+
* @throws KeeperException if unexpected zookeeper exception
177+
*/
178+
public static long getCreateTimeIfNodeExists(ZKWatcher zkw, String znode) throws KeeperException {
179+
try {
180+
Stat s = zkw.getRecoverableZooKeeper().exists(znode, false);
181+
return s != null ? s.getCtime() : -1;
182+
} catch (KeeperException e) {
183+
LOG.warn(zkw.prefix("Unable to get create time on znode (" + znode + ")"), e);
184+
zkw.keeperException(e);
185+
return -1;
186+
} catch (InterruptedException e) {
187+
LOG.warn(zkw.prefix("Unable to get create time on znode (" + znode + ")"), e);
188+
zkw.interruptedException(e);
189+
return -1;
190+
}
191+
}
192+
171193
//
172194
// Znode listings
173195
//

0 commit comments

Comments
 (0)