001/** 002 * Licensed to the Apache Software Foundation (ASF) under one 003 * or more contributor license agreements. See the NOTICE file 004 * distributed with this work for additional information 005 * regarding copyright ownership. The ASF licenses this file 006 * to you under the Apache License, Version 2.0 (the 007 * "License"); you may not use this file except in compliance 008 * with the License. You may obtain a copy of the License at 009 * 010 * http://www.apache.org/licenses/LICENSE-2.0 011 * 012 * Unless required by applicable law or agreed to in writing, software 013 * distributed under the License is distributed on an "AS IS" BASIS, 014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 015 * See the License for the specific language governing permissions and 016 * limitations under the License. 017 */ 018 019package org.apache.hadoop.hbase.replication.regionserver; 020 021import java.io.IOException; 022import java.io.InterruptedIOException; 023import java.util.ArrayList; 024import java.util.List; 025import java.util.Map; 026import java.util.concurrent.Callable; 027import java.util.concurrent.ExecutionException; 028import java.util.concurrent.ExecutorService; 029import java.util.concurrent.Future; 030import java.util.concurrent.LinkedBlockingQueue; 031import java.util.concurrent.ThreadPoolExecutor; 032import java.util.concurrent.TimeUnit; 033import java.util.concurrent.atomic.AtomicLong; 034import org.apache.hadoop.conf.Configuration; 035import org.apache.hadoop.fs.Path; 036import org.apache.hadoop.hbase.CellScanner; 037import org.apache.hadoop.hbase.CellUtil; 038import org.apache.hadoop.hbase.HBaseConfiguration; 039import org.apache.hadoop.hbase.HBaseIOException; 040import org.apache.hadoop.hbase.HConstants; 041import org.apache.hadoop.hbase.HRegionLocation; 042import org.apache.hadoop.hbase.RegionLocations; 043import org.apache.hadoop.hbase.TableDescriptors; 044import org.apache.hadoop.hbase.TableName; 045import org.apache.hadoop.hbase.TableNotFoundException; 046import org.apache.hadoop.hbase.client.ClusterConnection; 047import org.apache.hadoop.hbase.client.ConnectionFactory; 048import org.apache.hadoop.hbase.client.RegionAdminServiceCallable; 049import org.apache.hadoop.hbase.client.RegionInfo; 050import org.apache.hadoop.hbase.client.RegionReplicaUtil; 051import org.apache.hadoop.hbase.client.RetryingCallable; 052import org.apache.hadoop.hbase.client.RpcRetryingCallerFactory; 053import org.apache.hadoop.hbase.client.TableDescriptor; 054import org.apache.hadoop.hbase.ipc.HBaseRpcController; 055import org.apache.hadoop.hbase.ipc.RpcControllerFactory; 056import org.apache.hadoop.hbase.protobuf.ReplicationProtbufUtil; 057import org.apache.hadoop.hbase.replication.HBaseReplicationEndpoint; 058import org.apache.hadoop.hbase.replication.WALEntryFilter; 059import org.apache.hadoop.hbase.util.Bytes; 060import org.apache.hadoop.hbase.util.Pair; 061import org.apache.hadoop.hbase.util.Threads; 062import org.apache.hadoop.hbase.wal.EntryBuffers; 063import org.apache.hadoop.hbase.wal.EntryBuffers.RegionEntryBuffer; 064import org.apache.hadoop.hbase.wal.OutputSink; 065import org.apache.hadoop.hbase.wal.WAL.Entry; 066import org.apache.hadoop.hbase.wal.WALSplitter.PipelineController; 067import org.apache.hadoop.util.StringUtils; 068import org.apache.yetus.audience.InterfaceAudience; 069import org.slf4j.Logger; 070import org.slf4j.LoggerFactory; 071 072import org.apache.hbase.thirdparty.com.google.common.cache.Cache; 073import org.apache.hbase.thirdparty.com.google.common.cache.CacheBuilder; 074 075import org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos; 076import org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos.ReplicateWALEntryResponse; 077 078/** 079 * A {@link org.apache.hadoop.hbase.replication.ReplicationEndpoint} endpoint 080 * which receives the WAL edits from the WAL, and sends the edits to replicas 081 * of regions. 082 */ 083@InterfaceAudience.Private 084public class RegionReplicaReplicationEndpoint extends HBaseReplicationEndpoint { 085 086 private static final Logger LOG = LoggerFactory.getLogger(RegionReplicaReplicationEndpoint.class); 087 088 // Can be configured differently than hbase.client.retries.number 089 private static String CLIENT_RETRIES_NUMBER 090 = "hbase.region.replica.replication.client.retries.number"; 091 092 private Configuration conf; 093 private ClusterConnection connection; 094 private TableDescriptors tableDescriptors; 095 096 // Reuse WALSplitter constructs as a WAL pipe 097 private PipelineController controller; 098 private RegionReplicaOutputSink outputSink; 099 private EntryBuffers entryBuffers; 100 101 // Number of writer threads 102 private int numWriterThreads; 103 104 private int operationTimeout; 105 106 private ExecutorService pool; 107 108 @Override 109 public void init(Context context) throws IOException { 110 super.init(context); 111 112 this.conf = HBaseConfiguration.create(context.getConfiguration()); 113 this.tableDescriptors = context.getTableDescriptors(); 114 115 // HRS multiplies client retries by 10 globally for meta operations, but we do not want this. 116 // We are resetting it here because we want default number of retries (35) rather than 10 times 117 // that which makes very long retries for disabled tables etc. 118 int defaultNumRetries = conf.getInt(HConstants.HBASE_CLIENT_RETRIES_NUMBER, 119 HConstants.DEFAULT_HBASE_CLIENT_RETRIES_NUMBER); 120 if (defaultNumRetries > 10) { 121 int mult = conf.getInt(HConstants.HBASE_CLIENT_SERVERSIDE_RETRIES_MULTIPLIER, 122 HConstants.DEFAULT_HBASE_CLIENT_SERVERSIDE_RETRIES_MULTIPLIER); 123 defaultNumRetries = defaultNumRetries / mult; // reset if HRS has multiplied this already 124 } 125 126 conf.setInt(HConstants.HBASE_CLIENT_SERVERSIDE_RETRIES_MULTIPLIER, 1); 127 int numRetries = conf.getInt(CLIENT_RETRIES_NUMBER, defaultNumRetries); 128 conf.setInt(HConstants.HBASE_CLIENT_RETRIES_NUMBER, numRetries); 129 130 this.numWriterThreads = this.conf.getInt( 131 "hbase.region.replica.replication.writer.threads", 3); 132 controller = new PipelineController(); 133 entryBuffers = new EntryBuffers(controller, 134 this.conf.getLong("hbase.region.replica.replication.buffersize", 128 * 1024 * 1024)); 135 136 // use the regular RPC timeout for replica replication RPC's 137 this.operationTimeout = conf.getInt(HConstants.HBASE_CLIENT_OPERATION_TIMEOUT, 138 HConstants.DEFAULT_HBASE_CLIENT_OPERATION_TIMEOUT); 139 } 140 141 @Override 142 protected void doStart() { 143 try { 144 connection = (ClusterConnection) ConnectionFactory.createConnection(this.conf); 145 this.pool = getDefaultThreadPool(conf); 146 outputSink = new RegionReplicaOutputSink(controller, tableDescriptors, entryBuffers, 147 connection, pool, numWriterThreads, operationTimeout); 148 outputSink.startWriterThreads(); 149 super.doStart(); 150 } catch (IOException ex) { 151 LOG.warn("Received exception while creating connection :" + ex); 152 notifyFailed(ex); 153 } 154 } 155 156 @Override 157 protected void doStop() { 158 if (outputSink != null) { 159 try { 160 outputSink.close(); 161 } catch (IOException ex) { 162 LOG.warn("Got exception while trying to close OutputSink", ex); 163 } 164 } 165 if (this.pool != null) { 166 this.pool.shutdownNow(); 167 try { 168 // wait for 10 sec 169 boolean shutdown = this.pool.awaitTermination(10000, TimeUnit.MILLISECONDS); 170 if (!shutdown) { 171 LOG.warn("Failed to shutdown the thread pool after 10 seconds"); 172 } 173 } catch (InterruptedException e) { 174 LOG.warn("Got interrupted while waiting for the thread pool to shut down" + e); 175 } 176 } 177 if (connection != null) { 178 try { 179 connection.close(); 180 } catch (IOException ex) { 181 LOG.warn("Got exception closing connection :" + ex); 182 } 183 } 184 super.doStop(); 185 } 186 187 /** 188 * Returns a Thread pool for the RPC's to region replicas. Similar to 189 * Connection's thread pool. 190 */ 191 private ExecutorService getDefaultThreadPool(Configuration conf) { 192 int maxThreads = conf.getInt("hbase.region.replica.replication.threads.max", 256); 193 if (maxThreads == 0) { 194 maxThreads = Runtime.getRuntime().availableProcessors() * 8; 195 } 196 long keepAliveTime = conf.getLong("hbase.region.replica.replication.threads.keepalivetime", 60); 197 LinkedBlockingQueue<Runnable> workQueue = 198 new LinkedBlockingQueue<>(maxThreads * 199 conf.getInt(HConstants.HBASE_CLIENT_MAX_TOTAL_TASKS, 200 HConstants.DEFAULT_HBASE_CLIENT_MAX_TOTAL_TASKS)); 201 ThreadPoolExecutor tpe = new ThreadPoolExecutor( 202 maxThreads, 203 maxThreads, 204 keepAliveTime, 205 TimeUnit.SECONDS, 206 workQueue, 207 Threads.newDaemonThreadFactory(this.getClass().getSimpleName() + "-rpc-shared-")); 208 tpe.allowCoreThreadTimeOut(true); 209 return tpe; 210 } 211 212 @Override 213 public boolean replicate(ReplicateContext replicateContext) { 214 /* A note on batching in RegionReplicaReplicationEndpoint (RRRE): 215 * 216 * RRRE relies on batching from two different mechanisms. The first is the batching from 217 * ReplicationSource since RRRE is a ReplicationEndpoint driven by RS. RS reads from a single 218 * WAL file filling up a buffer of heap size "replication.source.size.capacity"(64MB) or at most 219 * "replication.source.nb.capacity" entries or until it sees the end of file (in live tailing). 220 * Then RS passes all the buffered edits in this replicate() call context. RRRE puts the edits 221 * to the WALSplitter.EntryBuffers which is a blocking buffer space of up to 222 * "hbase.region.replica.replication.buffersize" (128MB) in size. This buffer splits the edits 223 * based on regions. 224 * 225 * There are "hbase.region.replica.replication.writer.threads"(default 3) writer threads which 226 * pick largest per-region buffer and send it to the SinkWriter (see RegionReplicaOutputSink). 227 * The SinkWriter in this case will send the wal edits to all secondary region replicas in 228 * parallel via a retrying rpc call. EntryBuffers guarantees that while a buffer is 229 * being written to the sink, another buffer for the same region will not be made available to 230 * writers ensuring regions edits are not replayed out of order. 231 * 232 * The replicate() call won't return until all the buffers are sent and ack'd by the sinks so 233 * that the replication can assume all edits are persisted. We may be able to do a better 234 * pipelining between the replication thread and output sinks later if it becomes a bottleneck. 235 */ 236 237 while (this.isRunning()) { 238 try { 239 for (Entry entry: replicateContext.getEntries()) { 240 entryBuffers.appendEntry(entry); 241 } 242 outputSink.flush(); // make sure everything is flushed 243 ctx.getMetrics().incrLogEditsFiltered( 244 outputSink.getSkippedEditsCounter().getAndSet(0)); 245 return true; 246 } catch (InterruptedException e) { 247 Thread.currentThread().interrupt(); 248 return false; 249 } catch (IOException e) { 250 LOG.warn("Received IOException while trying to replicate" 251 + StringUtils.stringifyException(e)); 252 outputSink.restartWriterThreadsIfNeeded(); 253 } 254 } 255 256 return false; 257 } 258 259 @Override 260 public boolean canReplicateToSameCluster() { 261 return true; 262 } 263 264 @Override 265 protected WALEntryFilter getScopeWALEntryFilter() { 266 // we do not care about scope. We replicate everything. 267 return null; 268 } 269 270 static class RegionReplicaOutputSink extends OutputSink { 271 private final RegionReplicaSinkWriter sinkWriter; 272 private final TableDescriptors tableDescriptors; 273 private final Cache<TableName, Boolean> memstoreReplicationEnabled; 274 275 public RegionReplicaOutputSink(PipelineController controller, TableDescriptors tableDescriptors, 276 EntryBuffers entryBuffers, ClusterConnection connection, ExecutorService pool, 277 int numWriters, int operationTimeout) { 278 super(controller, entryBuffers, numWriters); 279 this.sinkWriter = 280 new RegionReplicaSinkWriter(this, connection, pool, operationTimeout, tableDescriptors); 281 this.tableDescriptors = tableDescriptors; 282 283 // A cache for the table "memstore replication enabled" flag. 284 // It has a default expiry of 5 sec. This means that if the table is altered 285 // with a different flag value, we might miss to replicate for that amount of 286 // time. But this cache avoid the slow lookup and parsing of the TableDescriptor. 287 int memstoreReplicationEnabledCacheExpiryMs = connection.getConfiguration() 288 .getInt("hbase.region.replica.replication.cache.memstoreReplicationEnabled.expiryMs", 5000); 289 this.memstoreReplicationEnabled = CacheBuilder.newBuilder() 290 .expireAfterWrite(memstoreReplicationEnabledCacheExpiryMs, TimeUnit.MILLISECONDS) 291 .initialCapacity(10) 292 .maximumSize(1000) 293 .build(); 294 } 295 296 @Override 297 public void append(RegionEntryBuffer buffer) throws IOException { 298 List<Entry> entries = buffer.getEntries(); 299 300 if (entries.isEmpty() || entries.get(0).getEdit().getCells().isEmpty()) { 301 return; 302 } 303 304 // meta edits (e.g. flush) are always replicated. 305 // data edits (e.g. put) are replicated if the table requires them. 306 if (!requiresReplication(buffer.getTableName(), entries)) { 307 return; 308 } 309 310 sinkWriter.append(buffer.getTableName(), buffer.getEncodedRegionName(), 311 CellUtil.cloneRow(entries.get(0).getEdit().getCells().get(0)), entries); 312 } 313 314 void flush() throws IOException { 315 // nothing much to do for now. Wait for the Writer threads to finish up 316 // append()'ing the data. 317 entryBuffers.waitUntilDrained(); 318 } 319 320 @Override 321 public boolean keepRegionEvent(Entry entry) { 322 return true; 323 } 324 325 @Override 326 public List<Path> close() throws IOException { 327 finishWriterThreads(true); 328 return null; 329 } 330 331 @Override 332 public Map<String, Long> getOutputCounts() { 333 return null; // only used in tests 334 } 335 336 @Override 337 public int getNumberOfRecoveredRegions() { 338 return 0; 339 } 340 341 AtomicLong getSkippedEditsCounter() { 342 return totalSkippedEdits; 343 } 344 345 /** 346 * returns true if the specified entry must be replicated. 347 * We should always replicate meta operations (e.g. flush) 348 * and use the user HTD flag to decide whether or not replicate the memstore. 349 */ 350 private boolean requiresReplication(final TableName tableName, final List<Entry> entries) 351 throws IOException { 352 // unit-tests may not the TableDescriptors, bypass the check and always replicate 353 if (tableDescriptors == null) return true; 354 355 Boolean requiresReplication = memstoreReplicationEnabled.getIfPresent(tableName); 356 if (requiresReplication == null) { 357 // check if the table requires memstore replication 358 // some unit-test drop the table, so we should do a bypass check and always replicate. 359 TableDescriptor htd = tableDescriptors.get(tableName); 360 requiresReplication = htd == null || htd.hasRegionMemStoreReplication(); 361 memstoreReplicationEnabled.put(tableName, requiresReplication); 362 } 363 364 // if memstore replication is not required, check the entries. 365 // meta edits (e.g. flush) must be always replicated. 366 if (!requiresReplication) { 367 int skipEdits = 0; 368 java.util.Iterator<Entry> it = entries.iterator(); 369 while (it.hasNext()) { 370 Entry entry = it.next(); 371 if (entry.getEdit().isMetaEdit()) { 372 requiresReplication = true; 373 } else { 374 it.remove(); 375 skipEdits++; 376 } 377 } 378 totalSkippedEdits.addAndGet(skipEdits); 379 } 380 return requiresReplication; 381 } 382 383 @Override 384 protected int getNumOpenWriters() { 385 // TODO Auto-generated method stub 386 return 0; 387 } 388 } 389 390 static class RegionReplicaSinkWriter { 391 RegionReplicaOutputSink sink; 392 ClusterConnection connection; 393 RpcControllerFactory rpcControllerFactory; 394 RpcRetryingCallerFactory rpcRetryingCallerFactory; 395 int operationTimeout; 396 ExecutorService pool; 397 Cache<TableName, Boolean> disabledAndDroppedTables; 398 TableDescriptors tableDescriptors; 399 400 public RegionReplicaSinkWriter(RegionReplicaOutputSink sink, ClusterConnection connection, 401 ExecutorService pool, int operationTimeout, TableDescriptors tableDescriptors) { 402 this.sink = sink; 403 this.connection = connection; 404 this.operationTimeout = operationTimeout; 405 this.rpcRetryingCallerFactory 406 = RpcRetryingCallerFactory.instantiate(connection.getConfiguration()); 407 this.rpcControllerFactory = RpcControllerFactory.instantiate(connection.getConfiguration()); 408 this.pool = pool; 409 this.tableDescriptors = tableDescriptors; 410 411 int nonExistentTableCacheExpiryMs = connection.getConfiguration() 412 .getInt("hbase.region.replica.replication.cache.disabledAndDroppedTables.expiryMs", 5000); 413 // A cache for non existing tables that have a default expiry of 5 sec. This means that if the 414 // table is created again with the same name, we might miss to replicate for that amount of 415 // time. But this cache prevents overloading meta requests for every edit from a deleted file. 416 disabledAndDroppedTables = CacheBuilder.newBuilder() 417 .expireAfterWrite(nonExistentTableCacheExpiryMs, TimeUnit.MILLISECONDS) 418 .initialCapacity(10) 419 .maximumSize(1000) 420 .build(); 421 } 422 423 public void append(TableName tableName, byte[] encodedRegionName, byte[] row, 424 List<Entry> entries) throws IOException { 425 426 if (disabledAndDroppedTables.getIfPresent(tableName) != null) { 427 if (LOG.isTraceEnabled()) { 428 LOG.trace("Skipping " + entries.size() + " entries because table " + tableName 429 + " is cached as a disabled or dropped table"); 430 for (Entry entry : entries) { 431 LOG.trace("Skipping : " + entry); 432 } 433 } 434 sink.getSkippedEditsCounter().addAndGet(entries.size()); 435 return; 436 } 437 438 // If the table is disabled or dropped, we should not replay the entries, and we can skip 439 // replaying them. However, we might not know whether the table is disabled until we 440 // invalidate the cache and check from meta 441 RegionLocations locations = null; 442 boolean useCache = true; 443 while (true) { 444 // get the replicas of the primary region 445 try { 446 locations = RegionReplicaReplayCallable 447 .getRegionLocations(connection, tableName, row, useCache, 0); 448 449 if (locations == null) { 450 throw new HBaseIOException("Cannot locate locations for " 451 + tableName + ", row:" + Bytes.toStringBinary(row)); 452 } 453 } catch (TableNotFoundException e) { 454 if (LOG.isTraceEnabled()) { 455 LOG.trace("Skipping " + entries.size() + " entries because table " + tableName 456 + " is dropped. Adding table to cache."); 457 for (Entry entry : entries) { 458 LOG.trace("Skipping : " + entry); 459 } 460 } 461 disabledAndDroppedTables.put(tableName, Boolean.TRUE); // put to cache. Value ignored 462 // skip this entry 463 sink.getSkippedEditsCounter().addAndGet(entries.size()); 464 return; 465 } 466 467 // check whether we should still replay this entry. If the regions are changed, or the 468 // entry is not coming from the primary region, filter it out. 469 HRegionLocation primaryLocation = locations.getDefaultRegionLocation(); 470 if (!Bytes.equals(primaryLocation.getRegionInfo().getEncodedNameAsBytes(), 471 encodedRegionName)) { 472 if (useCache) { 473 useCache = false; 474 continue; // this will retry location lookup 475 } 476 if (LOG.isTraceEnabled()) { 477 LOG.trace("Skipping " + entries.size() + " entries in table " + tableName 478 + " because located region " + primaryLocation.getRegionInfo().getEncodedName() 479 + " is different than the original region " + Bytes.toStringBinary(encodedRegionName) 480 + " from WALEdit"); 481 for (Entry entry : entries) { 482 LOG.trace("Skipping : " + entry); 483 } 484 } 485 sink.getSkippedEditsCounter().addAndGet(entries.size()); 486 return; 487 } 488 break; 489 } 490 491 if (locations.size() == 1) { 492 return; 493 } 494 495 ArrayList<Future<ReplicateWALEntryResponse>> tasks = new ArrayList<>(locations.size() - 1); 496 497 // All passed entries should belong to one region because it is coming from the EntryBuffers 498 // split per region. But the regions might split and merge (unlike log recovery case). 499 for (int replicaId = 0; replicaId < locations.size(); replicaId++) { 500 HRegionLocation location = locations.getRegionLocation(replicaId); 501 if (!RegionReplicaUtil.isDefaultReplica(replicaId)) { 502 RegionInfo regionInfo = location == null 503 ? RegionReplicaUtil.getRegionInfoForReplica( 504 locations.getDefaultRegionLocation().getRegionInfo(), replicaId) 505 : location.getRegionInfo(); 506 RegionReplicaReplayCallable callable = new RegionReplicaReplayCallable(connection, 507 rpcControllerFactory, tableName, location, regionInfo, row, entries, 508 sink.getSkippedEditsCounter()); 509 Future<ReplicateWALEntryResponse> task = pool.submit( 510 new RetryingRpcCallable<>(rpcRetryingCallerFactory, callable, operationTimeout)); 511 tasks.add(task); 512 } 513 } 514 515 boolean tasksCancelled = false; 516 for (int replicaId = 0; replicaId < tasks.size(); replicaId++) { 517 try { 518 tasks.get(replicaId).get(); 519 } catch (InterruptedException e) { 520 throw new InterruptedIOException(e.getMessage()); 521 } catch (ExecutionException e) { 522 Throwable cause = e.getCause(); 523 boolean canBeSkipped = false; 524 if (cause instanceof IOException) { 525 // The table can be disabled or dropped at this time. For disabled tables, we have no 526 // cheap mechanism to detect this case because meta does not contain this information. 527 // ClusterConnection.isTableDisabled() is a zk call which we cannot do for every replay 528 // RPC. So instead we start the replay RPC with retries and check whether the table is 529 // dropped or disabled which might cause SocketTimeoutException, or 530 // RetriesExhaustedException or similar if we get IOE. 531 if (cause instanceof TableNotFoundException 532 || connection.isTableDisabled(tableName)) { 533 disabledAndDroppedTables.put(tableName, Boolean.TRUE); // put to cache for later. 534 canBeSkipped = true; 535 } else if (tableDescriptors != null) { 536 TableDescriptor tableDescriptor = tableDescriptors.get(tableName); 537 if (tableDescriptor != null 538 //(replicaId + 1) as no task is added for primary replica for replication 539 && tableDescriptor.getRegionReplication() <= (replicaId + 1)) { 540 canBeSkipped = true; 541 } 542 } 543 if (canBeSkipped) { 544 if (LOG.isTraceEnabled()) { 545 LOG.trace("Skipping " + entries.size() + " entries in table " + tableName 546 + " because received exception for dropped or disabled table", 547 cause); 548 for (Entry entry : entries) { 549 LOG.trace("Skipping : " + entry); 550 } 551 } 552 if (!tasksCancelled) { 553 sink.getSkippedEditsCounter().addAndGet(entries.size()); 554 tasksCancelled = true; // so that we do not add to skipped counter again 555 } 556 continue; 557 } 558 559 // otherwise rethrow 560 throw (IOException)cause; 561 } 562 // unexpected exception 563 throw new IOException(cause); 564 } 565 } 566 } 567 } 568 569 static class RetryingRpcCallable<V> implements Callable<V> { 570 RpcRetryingCallerFactory factory; 571 RetryingCallable<V> callable; 572 int timeout; 573 public RetryingRpcCallable(RpcRetryingCallerFactory factory, RetryingCallable<V> callable, 574 int timeout) { 575 this.factory = factory; 576 this.callable = callable; 577 this.timeout = timeout; 578 } 579 @Override 580 public V call() throws Exception { 581 return factory.<V>newCaller().callWithRetries(callable, timeout); 582 } 583 } 584 585 /** 586 * Calls replay on the passed edits for the given set of entries belonging to the region. It skips 587 * the entry if the region boundaries have changed or the region is gone. 588 */ 589 static class RegionReplicaReplayCallable extends 590 RegionAdminServiceCallable<ReplicateWALEntryResponse> { 591 private final List<Entry> entries; 592 private final byte[] initialEncodedRegionName; 593 private final AtomicLong skippedEntries; 594 595 public RegionReplicaReplayCallable(ClusterConnection connection, 596 RpcControllerFactory rpcControllerFactory, TableName tableName, 597 HRegionLocation location, RegionInfo regionInfo, byte[] row,List<Entry> entries, 598 AtomicLong skippedEntries) { 599 super(connection, rpcControllerFactory, location, tableName, row, regionInfo.getReplicaId()); 600 this.entries = entries; 601 this.skippedEntries = skippedEntries; 602 this.initialEncodedRegionName = regionInfo.getEncodedNameAsBytes(); 603 } 604 605 @Override 606 public ReplicateWALEntryResponse call(HBaseRpcController controller) throws Exception { 607 // Check whether we should still replay this entry. If the regions are changed, or the 608 // entry is not coming form the primary region, filter it out because we do not need it. 609 // Regions can change because of (1) region split (2) region merge (3) table recreated 610 boolean skip = false; 611 if (!Bytes.equals(location.getRegionInfo().getEncodedNameAsBytes(), 612 initialEncodedRegionName)) { 613 skip = true; 614 } 615 if (!this.entries.isEmpty() && !skip) { 616 Entry[] entriesArray = new Entry[this.entries.size()]; 617 entriesArray = this.entries.toArray(entriesArray); 618 619 // set the region name for the target region replica 620 Pair<AdminProtos.ReplicateWALEntryRequest, CellScanner> p = 621 ReplicationProtbufUtil.buildReplicateWALEntryRequest(entriesArray, location 622 .getRegionInfo().getEncodedNameAsBytes(), null, null, null); 623 controller.setCellScanner(p.getSecond()); 624 return stub.replay(controller, p.getFirst()); 625 } 626 627 if (skip) { 628 if (LOG.isTraceEnabled()) { 629 LOG.trace("Skipping " + entries.size() + " entries in table " + tableName 630 + " because located region " + location.getRegionInfo().getEncodedName() 631 + " is different than the original region " 632 + Bytes.toStringBinary(initialEncodedRegionName) + " from WALEdit"); 633 for (Entry entry : entries) { 634 LOG.trace("Skipping : " + entry); 635 } 636 } 637 skippedEntries.addAndGet(entries.size()); 638 } 639 return ReplicateWALEntryResponse.newBuilder().build(); 640 } 641 } 642}