HBASE-22731 ReplicationSource and HBaseInterClusterReplicationEndpoint log messages should include a target Peer identifier
Signed-off-by: Peter Somogyi <psomogyi@apache.org>
This commit is contained in:
parent
eef8e140c5
commit
3b0013ded9
|
@ -198,11 +198,14 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
protected boolean sleepForRetries(String msg, int sleepMultiplier) {
|
protected boolean sleepForRetries(String msg, int sleepMultiplier) {
|
||||||
try {
|
try {
|
||||||
if (LOG.isTraceEnabled()) {
|
if (LOG.isTraceEnabled()) {
|
||||||
LOG.trace(msg + ", sleeping " + sleepForRetries + " times " + sleepMultiplier);
|
LOG.trace("{} {}, sleeping {} times {}",
|
||||||
|
logPeerId(), msg, sleepForRetries, sleepMultiplier);
|
||||||
}
|
}
|
||||||
Thread.sleep(this.sleepForRetries * sleepMultiplier);
|
Thread.sleep(this.sleepForRetries * sleepMultiplier);
|
||||||
} catch (InterruptedException e) {
|
} catch (InterruptedException e) {
|
||||||
LOG.debug("Interrupted while sleeping between retries");
|
if (LOG.isDebugEnabled()) {
|
||||||
|
LOG.debug("{} Interrupted while sleeping between retries", logPeerId());
|
||||||
|
}
|
||||||
}
|
}
|
||||||
return sleepMultiplier < maxRetriesMultiplier;
|
return sleepMultiplier < maxRetriesMultiplier;
|
||||||
}
|
}
|
||||||
|
@ -288,7 +291,7 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
connection =
|
connection =
|
||||||
ClusterConnectionFactory.createAsyncClusterConnection(conf, null, User.getCurrent());
|
ClusterConnectionFactory.createAsyncClusterConnection(conf, null, User.getCurrent());
|
||||||
} catch (IOException ioe) {
|
} catch (IOException ioe) {
|
||||||
LOG.warn("Failed to create connection for peer cluster", ioe);
|
LOG.warn("{} Failed to create connection for peer cluster", logPeerId(), ioe);
|
||||||
}
|
}
|
||||||
if (connection != null) {
|
if (connection != null) {
|
||||||
this.conn = connection;
|
this.conn = connection;
|
||||||
|
@ -301,8 +304,10 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
for (int i = 0; i < batches.size(); i++) {
|
for (int i = 0; i < batches.size(); i++) {
|
||||||
List<Entry> entries = batches.get(i);
|
List<Entry> entries = batches.get(i);
|
||||||
if (!entries.isEmpty()) {
|
if (!entries.isEmpty()) {
|
||||||
LOG.trace("Submitting {} entries of total size {}", entries.size(),
|
if (LOG.isTraceEnabled()) {
|
||||||
|
LOG.trace("{} Submitting {} entries of total size {}", logPeerId(), entries.size(),
|
||||||
replicateContext.getSize());
|
replicateContext.getSize());
|
||||||
|
}
|
||||||
// RuntimeExceptions encountered here bubble up and are handled in ReplicationSource
|
// RuntimeExceptions encountered here bubble up and are handled in ReplicationSource
|
||||||
pool.submit(createReplicator(entries, i));
|
pool.submit(createReplicator(entries, i));
|
||||||
futures++;
|
futures++;
|
||||||
|
@ -353,8 +358,8 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
|
|
||||||
int numSinks = replicationSinkMgr.getNumSinks();
|
int numSinks = replicationSinkMgr.getNumSinks();
|
||||||
if (numSinks == 0) {
|
if (numSinks == 0) {
|
||||||
LOG.warn("No replication sinks found, returning without replicating. The source should " +
|
LOG.warn("{} No replication sinks found, returning without replicating. "
|
||||||
"retry with the same set of edits.");
|
+ "The source should retry with the same set of edits.", logPeerId());
|
||||||
return false;
|
return false;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -376,7 +381,8 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
} catch (IOException ioe) {
|
} catch (IOException ioe) {
|
||||||
if (ioe instanceof RemoteException) {
|
if (ioe instanceof RemoteException) {
|
||||||
ioe = ((RemoteException) ioe).unwrapRemoteException();
|
ioe = ((RemoteException) ioe).unwrapRemoteException();
|
||||||
LOG.warn("Can't replicate because of an error on the remote cluster: ", ioe);
|
LOG.warn("{} Can't replicate because of an error on the remote cluster: ", logPeerId(),
|
||||||
|
ioe);
|
||||||
if (ioe instanceof TableNotFoundException) {
|
if (ioe instanceof TableNotFoundException) {
|
||||||
if (dropOnDeletedTables) {
|
if (dropOnDeletedTables) {
|
||||||
// this is a bit fragile, but cannot change how TNFE is serialized
|
// this is a bit fragile, but cannot change how TNFE is serialized
|
||||||
|
@ -389,19 +395,20 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
// Would potentially be better to retry in one of the outer loops
|
// Would potentially be better to retry in one of the outer loops
|
||||||
// and add a table filter there; but that would break the encapsulation,
|
// and add a table filter there; but that would break the encapsulation,
|
||||||
// so we're doing the filtering here.
|
// so we're doing the filtering here.
|
||||||
LOG.info("Missing table detected at sink, local table also does not exist, " +
|
LOG.info("{} Missing table detected at sink, local table also does not "
|
||||||
"filtering edits for '" + table + "'");
|
+ "exist, filtering edits for '{}'", logPeerId(), table);
|
||||||
batches = filterBatches(batches, table);
|
batches = filterBatches(batches, table);
|
||||||
continue;
|
continue;
|
||||||
}
|
}
|
||||||
} catch (IOException iox) {
|
} catch (IOException iox) {
|
||||||
LOG.warn("Exception checking for local table: ", iox);
|
LOG.warn("{} Exception checking for local table: ", logPeerId(), iox);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
// fall through and sleep below
|
// fall through and sleep below
|
||||||
} else {
|
} else {
|
||||||
LOG.warn("Peer encountered RemoteException, rechecking all sinks: ", ioe);
|
LOG.warn("{} Peer encountered RemoteException, rechecking all sinks: ", logPeerId(),
|
||||||
|
ioe);
|
||||||
replicationSinkMgr.chooseSinks();
|
replicationSinkMgr.chooseSinks();
|
||||||
}
|
}
|
||||||
} else {
|
} else {
|
||||||
|
@ -414,10 +421,10 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
"caused by a machine failure or a massive slowdown",
|
"caused by a machine failure or a massive slowdown",
|
||||||
this.socketTimeoutMultiplier);
|
this.socketTimeoutMultiplier);
|
||||||
} else if (ioe instanceof ConnectException || ioe instanceof UnknownHostException) {
|
} else if (ioe instanceof ConnectException || ioe instanceof UnknownHostException) {
|
||||||
LOG.warn("Peer is unavailable, rechecking all sinks: ", ioe);
|
LOG.warn("{} Peer is unavailable, rechecking all sinks: ", logPeerId(), ioe);
|
||||||
replicationSinkMgr.chooseSinks();
|
replicationSinkMgr.chooseSinks();
|
||||||
} else {
|
} else {
|
||||||
LOG.warn("Can't replicate because of a local or network error: ", ioe);
|
LOG.warn("{} Can't replicate because of a local or network error: ", logPeerId(), ioe);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
if (sleepForRetries("Since we are unable to replicate", sleepMultiplier)) {
|
if (sleepForRetries("Since we are unable to replicate", sleepMultiplier)) {
|
||||||
|
@ -440,7 +447,7 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
this.conn.close();
|
this.conn.close();
|
||||||
this.conn = null;
|
this.conn = null;
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
LOG.warn("Failed to close the connection");
|
LOG.warn("{} Failed to close the connection", logPeerId());
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
// Allow currently running replication tasks to finish
|
// Allow currently running replication tasks to finish
|
||||||
|
@ -466,8 +473,8 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
int entriesHashCode = System.identityHashCode(entries);
|
int entriesHashCode = System.identityHashCode(entries);
|
||||||
if (LOG.isTraceEnabled()) {
|
if (LOG.isTraceEnabled()) {
|
||||||
long size = entries.stream().mapToLong(this::getEstimatedEntrySize).sum();
|
long size = entries.stream().mapToLong(this::getEstimatedEntrySize).sum();
|
||||||
LOG.trace("Replicating batch {} of {} entries with total size {} bytes to {}",
|
LOG.trace("{} Replicating batch {} of {} entries with total size {} bytes to {}",
|
||||||
entriesHashCode, entries.size(), size, replicationClusterId);
|
logPeerId(), entriesHashCode, entries.size(), size, replicationClusterId);
|
||||||
}
|
}
|
||||||
sinkPeer = replicationSinkMgr.getReplicationSink();
|
sinkPeer = replicationSinkMgr.getReplicationSink();
|
||||||
AsyncRegionServerAdmin rsAdmin = sinkPeer.getRegionServer();
|
AsyncRegionServerAdmin rsAdmin = sinkPeer.getRegionServer();
|
||||||
|
@ -475,9 +482,13 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
ReplicationProtbufUtil.replicateWALEntry(rsAdmin,
|
ReplicationProtbufUtil.replicateWALEntry(rsAdmin,
|
||||||
entries.toArray(new Entry[entries.size()]), replicationClusterId, baseNamespaceDir,
|
entries.toArray(new Entry[entries.size()]), replicationClusterId, baseNamespaceDir,
|
||||||
hfileArchiveDir);
|
hfileArchiveDir);
|
||||||
LOG.trace("Completed replicating batch {}", entriesHashCode);
|
if (LOG.isTraceEnabled()) {
|
||||||
|
LOG.trace("{} Completed replicating batch {}", logPeerId(), entriesHashCode);
|
||||||
|
}
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
LOG.trace("Failed replicating batch {}", entriesHashCode, e);
|
if (LOG.isTraceEnabled()) {
|
||||||
|
LOG.trace("{} Failed replicating batch {}", logPeerId(), entriesHashCode, e);
|
||||||
|
}
|
||||||
throw e;
|
throw e;
|
||||||
}
|
}
|
||||||
replicationSinkMgr.reportSinkSuccess(sinkPeer);
|
replicationSinkMgr.reportSinkSuccess(sinkPeer);
|
||||||
|
@ -515,4 +526,9 @@ public class HBaseInterClusterReplicationEndpoint extends HBaseReplicationEndpoi
|
||||||
return isSerial ? () -> serialReplicateRegionEntries(entries, batchIndex)
|
return isSerial ? () -> serialReplicateRegionEntries(entries, batchIndex)
|
||||||
: () -> replicateEntries(entries, batchIndex);
|
: () -> replicateEntries(entries, batchIndex);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
private String logPeerId(){
|
||||||
|
return "[Source for peer " + this.ctx.getPeerId() + "]:";
|
||||||
|
}
|
||||||
|
|
||||||
}
|
}
|
||||||
|
|
|
@ -209,14 +209,17 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
} else {
|
} else {
|
||||||
queue.put(log);
|
queue.put(log);
|
||||||
}
|
}
|
||||||
LOG.trace("Added log file {} to queue of source {}.", logPrefix,
|
if (LOG.isTraceEnabled()) {
|
||||||
|
LOG.trace("{} Added log file {} to queue of source {}.", logPeerId(), logPrefix,
|
||||||
this.replicationQueueInfo.getQueueId());
|
this.replicationQueueInfo.getQueueId());
|
||||||
|
}
|
||||||
this.metrics.incrSizeOfLogQueue();
|
this.metrics.incrSizeOfLogQueue();
|
||||||
// This will log a warning for each new log that gets created above the warn threshold
|
// This will log a warning for each new log that gets created above the warn threshold
|
||||||
int queueSize = queue.size();
|
int queueSize = queue.size();
|
||||||
if (queueSize > this.logQueueWarnThreshold) {
|
if (queueSize > this.logQueueWarnThreshold) {
|
||||||
LOG.warn("WAL group " + logPrefix + " queue size: " + queueSize
|
LOG.warn("{} WAL group {} queue size: {} exceeds value of "
|
||||||
+ " exceeds value of replication.source.log.queue.warn: " + logQueueWarnThreshold);
|
+ "replication.source.log.queue.warn: {}", logPeerId(),
|
||||||
|
logPrefix, queueSize, logQueueWarnThreshold);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -232,8 +235,8 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
this.queueStorage.addHFileRefs(peerId, pairs);
|
this.queueStorage.addHFileRefs(peerId, pairs);
|
||||||
metrics.incrSizeOfHFileRefsQueue(pairs.size());
|
metrics.incrSizeOfHFileRefsQueue(pairs.size());
|
||||||
} else {
|
} else {
|
||||||
LOG.debug("HFiles will not be replicated belonging to the table " + tableName + " family "
|
LOG.debug("HFiles will not be replicated belonging to the table {} family {} to peer id {}",
|
||||||
+ Bytes.toString(family) + " to peer id " + peerId);
|
tableName, Bytes.toString(family), peerId);
|
||||||
}
|
}
|
||||||
} else {
|
} else {
|
||||||
// user has explicitly not defined any table cfs for replication, means replicate all the
|
// user has explicitly not defined any table cfs for replication, means replicate all the
|
||||||
|
@ -305,9 +308,14 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
ReplicationSourceShipper worker = createNewShipper(walGroupId, queue);
|
ReplicationSourceShipper worker = createNewShipper(walGroupId, queue);
|
||||||
ReplicationSourceShipper extant = workerThreads.putIfAbsent(walGroupId, worker);
|
ReplicationSourceShipper extant = workerThreads.putIfAbsent(walGroupId, worker);
|
||||||
if (extant != null) {
|
if (extant != null) {
|
||||||
LOG.debug("Someone has beat us to start a worker thread for wal group {}", walGroupId);
|
if(LOG.isDebugEnabled()) {
|
||||||
|
LOG.debug("{} Someone has beat us to start a worker thread for wal group {}", logPeerId(),
|
||||||
|
walGroupId);
|
||||||
|
}
|
||||||
} else {
|
} else {
|
||||||
LOG.debug("Starting up worker for wal group {}", walGroupId);
|
if(LOG.isDebugEnabled()) {
|
||||||
|
LOG.debug("{} Starting up worker for wal group {}", logPeerId(), walGroupId);
|
||||||
|
}
|
||||||
ReplicationSourceWALReader walReader =
|
ReplicationSourceWALReader walReader =
|
||||||
createNewWALReader(walGroupId, queue, worker.getStartPosition());
|
createNewWALReader(walGroupId, queue, worker.getStartPosition());
|
||||||
Threads.setDaemonThreadRunning(walReader, Thread.currentThread().getName() +
|
Threads.setDaemonThreadRunning(walReader, Thread.currentThread().getName() +
|
||||||
|
@ -337,7 +345,7 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
}
|
}
|
||||||
} else {
|
} else {
|
||||||
currentPath = new Path("NO_LOGS_IN_QUEUE");
|
currentPath = new Path("NO_LOGS_IN_QUEUE");
|
||||||
LOG.warn("No replication ongoing, waiting for new log");
|
LOG.warn("{} No replication ongoing, waiting for new log", logPeerId());
|
||||||
}
|
}
|
||||||
ReplicationStatus.ReplicationStatusBuilder statusBuilder = ReplicationStatus.newBuilder();
|
ReplicationStatus.ReplicationStatusBuilder statusBuilder = ReplicationStatus.newBuilder();
|
||||||
statusBuilder.withPeerId(this.getPeerId())
|
statusBuilder.withPeerId(this.getPeerId())
|
||||||
|
@ -378,7 +386,8 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
|
|
||||||
protected final void uncaughtException(Thread t, Throwable e) {
|
protected final void uncaughtException(Thread t, Throwable e) {
|
||||||
RSRpcServices.exitIfOOME(e);
|
RSRpcServices.exitIfOOME(e);
|
||||||
LOG.error("Unexpected exception in " + t.getName() + " currentPath=" + getCurrentPath(), e);
|
LOG.error("Unexpected exception in {} currentPath={}",
|
||||||
|
t.getName(), getCurrentPath(), e);
|
||||||
server.abort("Unexpected exception in " + t.getName(), e);
|
server.abort("Unexpected exception in " + t.getName(), e);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -399,7 +408,7 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
long sleepTicks = throttler.getNextSleepInterval(batchSize);
|
long sleepTicks = throttler.getNextSleepInterval(batchSize);
|
||||||
if (sleepTicks > 0) {
|
if (sleepTicks > 0) {
|
||||||
if (LOG.isTraceEnabled()) {
|
if (LOG.isTraceEnabled()) {
|
||||||
LOG.trace("To sleep " + sleepTicks + "ms for throttling control");
|
LOG.trace("{} To sleep {}ms for throttling control", logPeerId(), sleepTicks);
|
||||||
}
|
}
|
||||||
Thread.sleep(sleepTicks);
|
Thread.sleep(sleepTicks);
|
||||||
// reset throttler's cycle start tick when sleep for throttling occurs
|
// reset throttler's cycle start tick when sleep for throttling occurs
|
||||||
|
@ -433,11 +442,14 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
protected boolean sleepForRetries(String msg, int sleepMultiplier) {
|
protected boolean sleepForRetries(String msg, int sleepMultiplier) {
|
||||||
try {
|
try {
|
||||||
if (LOG.isTraceEnabled()) {
|
if (LOG.isTraceEnabled()) {
|
||||||
LOG.trace(msg + ", sleeping " + sleepForRetries + " times " + sleepMultiplier);
|
LOG.trace("{} {}, sleeping {} times {}",
|
||||||
|
logPeerId(), msg, sleepForRetries, sleepMultiplier);
|
||||||
}
|
}
|
||||||
Thread.sleep(this.sleepForRetries * sleepMultiplier);
|
Thread.sleep(this.sleepForRetries * sleepMultiplier);
|
||||||
} catch (InterruptedException e) {
|
} catch (InterruptedException e) {
|
||||||
LOG.debug("Interrupted while sleeping between retries");
|
if(LOG.isDebugEnabled()) {
|
||||||
|
LOG.debug("{} Interrupted while sleeping between retries", logPeerId());
|
||||||
|
}
|
||||||
Thread.currentThread().interrupt();
|
Thread.currentThread().interrupt();
|
||||||
}
|
}
|
||||||
return sleepMultiplier < maxRetriesMultiplier;
|
return sleepMultiplier < maxRetriesMultiplier;
|
||||||
|
@ -450,7 +462,7 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
try {
|
try {
|
||||||
replicationEndpoint = createReplicationEndpoint();
|
replicationEndpoint = createReplicationEndpoint();
|
||||||
} catch (Exception e) {
|
} catch (Exception e) {
|
||||||
LOG.warn("error creating ReplicationEndpoint, retry", e);
|
LOG.warn("{} error creating ReplicationEndpoint, retry", logPeerId(), e);
|
||||||
if (sleepForRetries("Error creating ReplicationEndpoint", sleepMultiplier)) {
|
if (sleepForRetries("Error creating ReplicationEndpoint", sleepMultiplier)) {
|
||||||
sleepMultiplier++;
|
sleepMultiplier++;
|
||||||
}
|
}
|
||||||
|
@ -462,7 +474,7 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
this.replicationEndpoint = replicationEndpoint;
|
this.replicationEndpoint = replicationEndpoint;
|
||||||
break;
|
break;
|
||||||
} catch (Exception e) {
|
} catch (Exception e) {
|
||||||
LOG.warn("Error starting ReplicationEndpoint, retry", e);
|
LOG.warn("{} Error starting ReplicationEndpoint, retry", logPeerId(), e);
|
||||||
replicationEndpoint.stop();
|
replicationEndpoint.stop();
|
||||||
if (sleepForRetries("Error starting ReplicationEndpoint", sleepMultiplier)) {
|
if (sleepForRetries("Error starting ReplicationEndpoint", sleepMultiplier)) {
|
||||||
sleepMultiplier++;
|
sleepMultiplier++;
|
||||||
|
@ -480,8 +492,10 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
for (;;) {
|
for (;;) {
|
||||||
peerClusterId = replicationEndpoint.getPeerUUID();
|
peerClusterId = replicationEndpoint.getPeerUUID();
|
||||||
if (this.isSourceActive() && peerClusterId == null) {
|
if (this.isSourceActive() && peerClusterId == null) {
|
||||||
LOG.debug("Could not connect to Peer ZK. Sleeping for "
|
if(LOG.isDebugEnabled()) {
|
||||||
+ (this.sleepForRetries * sleepMultiplier) + " millis.");
|
LOG.debug("{} Could not connect to Peer ZK. Sleeping for {} millis", logPeerId(),
|
||||||
|
(this.sleepForRetries * sleepMultiplier));
|
||||||
|
}
|
||||||
if (sleepForRetries("Cannot contact the peer's zk ensemble", sleepMultiplier)) {
|
if (sleepForRetries("Cannot contact the peer's zk ensemble", sleepMultiplier)) {
|
||||||
sleepMultiplier++;
|
sleepMultiplier++;
|
||||||
}
|
}
|
||||||
|
@ -499,8 +513,8 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
this.manager.removeSource(this);
|
this.manager.removeSource(this);
|
||||||
return;
|
return;
|
||||||
}
|
}
|
||||||
LOG.info("Source: {}, is now replicating from cluster: {}; to peer cluster: {};",
|
LOG.info("{} Source: {}, is now replicating from cluster: {}; to peer cluster: {};",
|
||||||
this.replicationQueueInfo.getQueueId(), clusterId, peerClusterId);
|
logPeerId(), this.replicationQueueInfo.getQueueId(), clusterId, peerClusterId);
|
||||||
|
|
||||||
initializeWALEntryFilter(peerClusterId);
|
initializeWALEntryFilter(peerClusterId);
|
||||||
// start workers
|
// start workers
|
||||||
|
@ -533,10 +547,10 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
|
|
||||||
public void terminate(String reason, Exception cause, boolean join) {
|
public void terminate(String reason, Exception cause, boolean join) {
|
||||||
if (cause == null) {
|
if (cause == null) {
|
||||||
LOG.info("Closing source " + this.queueId + " because: " + reason);
|
LOG.info("{} Closing source {} because: {}", logPeerId(), this.queueId, reason);
|
||||||
} else {
|
} else {
|
||||||
LOG.error("Closing source " + this.queueId + " because an error occurred: " + reason,
|
LOG.error("{} Closing source {} because an error occurred: {}",
|
||||||
cause);
|
logPeerId(), this.queueId, reason, cause);
|
||||||
}
|
}
|
||||||
this.sourceRunning = false;
|
this.sourceRunning = false;
|
||||||
if (initThread != null && Thread.currentThread() != initThread) {
|
if (initThread != null && Thread.currentThread() != initThread) {
|
||||||
|
@ -560,7 +574,7 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
// Wait worker to stop
|
// Wait worker to stop
|
||||||
Thread.sleep(this.sleepForRetries);
|
Thread.sleep(this.sleepForRetries);
|
||||||
} catch (InterruptedException e) {
|
} catch (InterruptedException e) {
|
||||||
LOG.info("Interrupted while waiting " + worker.getName() + " to stop");
|
LOG.info("{} Interrupted while waiting {} to stop", logPeerId(), worker.getName());
|
||||||
Thread.currentThread().interrupt();
|
Thread.currentThread().interrupt();
|
||||||
}
|
}
|
||||||
// If worker still is alive after waiting, interrupt it
|
// If worker still is alive after waiting, interrupt it
|
||||||
|
@ -581,15 +595,15 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
if (join) {
|
if (join) {
|
||||||
for (ReplicationSourceShipper worker : workers) {
|
for (ReplicationSourceShipper worker : workers) {
|
||||||
Threads.shutdown(worker, this.sleepForRetries);
|
Threads.shutdown(worker, this.sleepForRetries);
|
||||||
LOG.info("ReplicationSourceWorker " + worker.getName() + " terminated");
|
LOG.info("{} ReplicationSourceWorker {} terminated", logPeerId(), worker.getName());
|
||||||
}
|
}
|
||||||
if (this.replicationEndpoint != null) {
|
if (this.replicationEndpoint != null) {
|
||||||
try {
|
try {
|
||||||
this.replicationEndpoint.awaitTerminated(sleepForRetries * maxRetriesMultiplier,
|
this.replicationEndpoint.awaitTerminated(sleepForRetries * maxRetriesMultiplier,
|
||||||
TimeUnit.MILLISECONDS);
|
TimeUnit.MILLISECONDS);
|
||||||
} catch (TimeoutException te) {
|
} catch (TimeoutException te) {
|
||||||
LOG.warn("Got exception while waiting for endpoint to shutdown for replication source :" +
|
LOG.warn("{} Got exception while waiting for endpoint to shutdown "
|
||||||
this.queueId, te);
|
+ "for replication source : {}", logPeerId(), this.queueId, te);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
@ -721,4 +735,8 @@ public class ReplicationSource implements ReplicationSourceInterface {
|
||||||
void removeWorker(ReplicationSourceShipper worker) {
|
void removeWorker(ReplicationSourceShipper worker) {
|
||||||
workerThreads.remove(worker.walGroupId, worker);
|
workerThreads.remove(worker.walGroupId, worker);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
private String logPeerId(){
|
||||||
|
return "[Source for peer " + this.getPeer().getId() + "]:";
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
Loading…
Reference in New Issue