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.hdfs.server.datanode; 020 021import java.io.DataOutputStream; 022import java.io.FileNotFoundException; 023import java.io.IOException; 024import java.util.Iterator; 025import java.util.LinkedHashSet; 026import java.util.LinkedList; 027import java.util.List; 028import java.util.concurrent.TimeUnit; 029 030import com.google.common.annotations.VisibleForTesting; 031import com.google.common.base.Preconditions; 032import com.google.common.cache.Cache; 033import com.google.common.cache.CacheBuilder; 034import org.apache.hadoop.hdfs.protocol.Block; 035import org.apache.hadoop.hdfs.protocol.ExtendedBlock; 036import org.apache.hadoop.hdfs.server.datanode.BlockScanner.Conf; 037import org.apache.hadoop.hdfs.server.datanode.fsdataset.FsVolumeReference; 038import org.apache.hadoop.hdfs.server.datanode.fsdataset.FsVolumeSpi.BlockIterator; 039import org.apache.hadoop.hdfs.server.datanode.fsdataset.FsVolumeSpi; 040import org.apache.hadoop.hdfs.server.datanode.metrics.DataNodeMetrics; 041import org.apache.hadoop.hdfs.util.DataTransferThrottler; 042import org.apache.hadoop.io.IOUtils; 043import org.apache.hadoop.util.Time; 044import org.slf4j.Logger; 045import org.slf4j.LoggerFactory; 046 047/** 048 * VolumeScanner scans a single volume. Each VolumeScanner has its own thread.<p/> 049 * They are all managed by the DataNode's BlockScanner. 050 */ 051public class VolumeScanner extends Thread { 052 public static final Logger LOG = 053 LoggerFactory.getLogger(VolumeScanner.class); 054 055 /** 056 * Number of seconds in a minute. 057 */ 058 private final static int SECONDS_PER_MINUTE = 60; 059 060 /** 061 * Number of minutes in an hour. 062 */ 063 private final static int MINUTES_PER_HOUR = 60; 064 065 /** 066 * Name of the block iterator used by this scanner. 067 */ 068 private final static String BLOCK_ITERATOR_NAME = "scanner"; 069 070 /** 071 * The configuration. 072 */ 073 private Conf conf; 074 075 @VisibleForTesting 076 void setConf(Conf conf) { 077 this.conf = conf; 078 } 079 080 /** 081 * The DataNode this VolumEscanner is associated with. 082 */ 083 private final DataNode datanode; 084 085 private final DataNodeMetrics metrics; 086 087 /** 088 * A reference to the volume that we're scanning. 089 */ 090 private final FsVolumeReference ref; 091 092 /** 093 * The volume that we're scanning. 094 */ 095 final FsVolumeSpi volume; 096 097 /** 098 * The number of scanned bytes in each minute of the last hour.<p/> 099 * 100 * This array is managed as a circular buffer. We take the monotonic time and 101 * divide it up into one-minute periods. Each entry in the array represents 102 * how many bytes were scanned during that period. 103 */ 104 private final long scannedBytes[] = new long[MINUTES_PER_HOUR]; 105 106 /** 107 * The sum of all the values of scannedBytes. 108 */ 109 private long scannedBytesSum = 0; 110 111 /** 112 * The throttler to use with BlockSender objects. 113 */ 114 private final DataTransferThrottler throttler = new DataTransferThrottler(1); 115 116 /** 117 * The null output stream to use with BlockSender objects. 118 */ 119 private final DataOutputStream nullStream = 120 new DataOutputStream(new IOUtils.NullOutputStream()); 121 122 /** 123 * The block iterators associated with this VolumeScanner.<p/> 124 * 125 * Each block pool has its own BlockIterator. 126 */ 127 private final List<BlockIterator> blockIters = 128 new LinkedList<BlockIterator>(); 129 130 /** 131 * Blocks which are suspect. 132 * The scanner prioritizes scanning these blocks. 133 */ 134 private final LinkedHashSet<ExtendedBlock> suspectBlocks = 135 new LinkedHashSet<ExtendedBlock>(); 136 137 /** 138 * Blocks which were suspect which we have scanned. 139 * This is used to avoid scanning the same suspect block over and over. 140 */ 141 private final Cache<ExtendedBlock, Boolean> recentSuspectBlocks = 142 CacheBuilder.newBuilder().maximumSize(1000) 143 .expireAfterAccess(10, TimeUnit.MINUTES).build(); 144 145 /** 146 * The current block iterator, or null if there is none. 147 */ 148 private BlockIterator curBlockIter = null; 149 150 /** 151 * True if the thread is stopping.<p/> 152 * Protected by this object's lock. 153 */ 154 private boolean stopping = false; 155 156 /** 157 * The monotonic minute that the volume scanner was started on. 158 */ 159 private long startMinute = 0; 160 161 /** 162 * The current minute, in monotonic terms. 163 */ 164 private long curMinute = 0; 165 166 /** 167 * Handles scan results. 168 */ 169 private final ScanResultHandler resultHandler; 170 171 private final Statistics stats = new Statistics(); 172 173 static class Statistics { 174 long bytesScannedInPastHour = 0; 175 long blocksScannedInCurrentPeriod = 0; 176 long blocksScannedSinceRestart = 0; 177 long scansSinceRestart = 0; 178 long scanErrorsSinceRestart = 0; 179 long nextBlockPoolScanStartMs = -1; 180 long blockPoolPeriodEndsMs = -1; 181 ExtendedBlock lastBlockScanned = null; 182 boolean eof = false; 183 184 Statistics() { 185 } 186 187 Statistics(Statistics other) { 188 this.bytesScannedInPastHour = other.bytesScannedInPastHour; 189 this.blocksScannedInCurrentPeriod = other.blocksScannedInCurrentPeriod; 190 this.blocksScannedSinceRestart = other.blocksScannedSinceRestart; 191 this.scansSinceRestart = other.scansSinceRestart; 192 this.scanErrorsSinceRestart = other.scanErrorsSinceRestart; 193 this.nextBlockPoolScanStartMs = other.nextBlockPoolScanStartMs; 194 this.blockPoolPeriodEndsMs = other.blockPoolPeriodEndsMs; 195 this.lastBlockScanned = other.lastBlockScanned; 196 this.eof = other.eof; 197 } 198 199 @Override 200 public String toString() { 201 return new StringBuilder(). 202 append("Statistics{"). 203 append("bytesScannedInPastHour=").append(bytesScannedInPastHour). 204 append(", blocksScannedInCurrentPeriod="). 205 append(blocksScannedInCurrentPeriod). 206 append(", blocksScannedSinceRestart="). 207 append(blocksScannedSinceRestart). 208 append(", scansSinceRestart=").append(scansSinceRestart). 209 append(", scanErrorsSinceRestart=").append(scanErrorsSinceRestart). 210 append(", nextBlockPoolScanStartMs=").append(nextBlockPoolScanStartMs). 211 append(", blockPoolPeriodEndsMs=").append(blockPoolPeriodEndsMs). 212 append(", lastBlockScanned=").append(lastBlockScanned). 213 append(", eof=").append(eof). 214 append("}").toString(); 215 } 216 } 217 218 private static double positiveMsToHours(long ms) { 219 if (ms <= 0) { 220 return 0; 221 } else { 222 return TimeUnit.HOURS.convert(ms, TimeUnit.MILLISECONDS); 223 } 224 } 225 226 public void printStats(StringBuilder p) { 227 p.append("Block scanner information for volume " + 228 volume.getStorageID() + " with base path " + volume.getBasePath() + 229 "%n"); 230 synchronized (stats) { 231 p.append(String.format("Bytes verified in last hour : %57d%n", 232 stats.bytesScannedInPastHour)); 233 p.append(String.format("Blocks scanned in current period : %57d%n", 234 stats.blocksScannedInCurrentPeriod)); 235 p.append(String.format("Blocks scanned since restart : %57d%n", 236 stats.blocksScannedSinceRestart)); 237 p.append(String.format("Block pool scans since restart : %57d%n", 238 stats.scansSinceRestart)); 239 p.append(String.format("Block scan errors since restart : %57d%n", 240 stats.scanErrorsSinceRestart)); 241 if (stats.nextBlockPoolScanStartMs > 0) { 242 p.append(String.format("Hours until next block pool scan : %57.3f%n", 243 positiveMsToHours(stats.nextBlockPoolScanStartMs - 244 Time.monotonicNow()))); 245 } 246 if (stats.blockPoolPeriodEndsMs > 0) { 247 p.append(String.format("Hours until possible pool rescan : %57.3f%n", 248 positiveMsToHours(stats.blockPoolPeriodEndsMs - 249 Time.now()))); 250 } 251 p.append(String.format("Last block scanned : %57s%n", 252 ((stats.lastBlockScanned == null) ? "none" : 253 stats.lastBlockScanned.toString()))); 254 p.append(String.format("More blocks to scan in period : %57s%n", 255 !stats.eof)); 256 p.append("%n"); 257 } 258 } 259 260 static class ScanResultHandler { 261 private VolumeScanner scanner; 262 263 public void setup(VolumeScanner scanner) { 264 LOG.trace("Starting VolumeScanner {}", 265 scanner.volume.getBasePath()); 266 this.scanner = scanner; 267 } 268 269 public void handle(ExtendedBlock block, IOException e) { 270 FsVolumeSpi volume = scanner.volume; 271 if (e == null) { 272 LOG.trace("Successfully scanned {} on {}", block, volume.getBasePath()); 273 return; 274 } 275 // If the block does not exist anymore, then it's not an error. 276 if (!volume.getDataset().contains(block)) { 277 LOG.debug("Volume {}: block {} is no longer in the dataset.", 278 volume.getBasePath(), block); 279 return; 280 } 281 // If the block exists, the exception may due to a race with write: 282 // The BlockSender got an old block path in rbw. BlockReceiver removed 283 // the rbw block from rbw to finalized but BlockSender tried to open the 284 // file before BlockReceiver updated the VolumeMap. The state of the 285 // block can be changed again now, so ignore this error here. If there 286 // is a block really deleted by mistake, DirectoryScan should catch it. 287 if (e instanceof FileNotFoundException ) { 288 LOG.info("Volume {}: verification failed for {} because of " + 289 "FileNotFoundException. This may be due to a race with write.", 290 volume.getBasePath(), block); 291 return; 292 } 293 LOG.warn("Reporting bad " + block + " with volume " 294 + volume.getBasePath(), e); 295 try { 296 scanner.datanode.reportBadBlocks(block, volume); 297 } catch (IOException ie) { 298 // This is bad, but not bad enough to shut down the scanner. 299 LOG.warn("Cannot report bad block " + block, ie); 300 } 301 } 302 } 303 304 VolumeScanner(Conf conf, DataNode datanode, FsVolumeReference ref) { 305 this.conf = conf; 306 this.datanode = datanode; 307 this.metrics = datanode.getMetrics(); 308 this.ref = ref; 309 this.volume = ref.getVolume(); 310 ScanResultHandler handler; 311 try { 312 handler = conf.resultHandler.newInstance(); 313 } catch (Throwable e) { 314 LOG.error("unable to instantiate {}", conf.resultHandler, e); 315 handler = new ScanResultHandler(); 316 } 317 this.resultHandler = handler; 318 setName("VolumeScannerThread(" + volume.getBasePath() + ")"); 319 setDaemon(true); 320 } 321 322 private void saveBlockIterator(BlockIterator iter) { 323 try { 324 iter.save(); 325 } catch (IOException e) { 326 LOG.warn("{}: error saving {}.", this, iter, e); 327 } 328 } 329 330 private void expireOldScannedBytesRecords(long monotonicMs) { 331 long newMinute = 332 TimeUnit.MINUTES.convert(monotonicMs, TimeUnit.MILLISECONDS); 333 if (curMinute == newMinute) { 334 return; 335 } 336 // If a minute or more has gone past since we last updated the scannedBytes 337 // array, zero out the slots corresponding to those minutes. 338 for (long m = curMinute + 1; m <= newMinute; m++) { 339 int slotIdx = (int)(m % MINUTES_PER_HOUR); 340 LOG.trace("{}: updateScannedBytes is zeroing out slotIdx {}. " + 341 "curMinute = {}; newMinute = {}", this, slotIdx, 342 curMinute, newMinute); 343 scannedBytesSum -= scannedBytes[slotIdx]; 344 scannedBytes[slotIdx] = 0; 345 } 346 curMinute = newMinute; 347 } 348 349 /** 350 * Find a usable block iterator.<p/> 351 * 352 * We will consider available block iterators in order. This property is 353 * important so that we don't keep rescanning the same block pool id over 354 * and over, while other block pools stay unscanned.<p/> 355 * 356 * A block pool is always ready to scan if the iterator is not at EOF. If 357 * the iterator is at EOF, the block pool will be ready to scan when 358 * conf.scanPeriodMs milliseconds have elapsed since the iterator was last 359 * rewound.<p/> 360 * 361 * @return 0 if we found a usable block iterator; the 362 * length of time we should delay before 363 * checking again otherwise. 364 */ 365 private synchronized long findNextUsableBlockIter() { 366 int numBlockIters = blockIters.size(); 367 if (numBlockIters == 0) { 368 LOG.debug("{}: no block pools are registered.", this); 369 return Long.MAX_VALUE; 370 } 371 int curIdx; 372 if (curBlockIter == null) { 373 curIdx = 0; 374 } else { 375 curIdx = blockIters.indexOf(curBlockIter); 376 Preconditions.checkState(curIdx >= 0); 377 } 378 // Note that this has to be wall-clock time, not monotonic time. This is 379 // because the time saved in the cursor file is a wall-clock time. We do 380 // not want to save a monotonic time in the cursor file, because it resets 381 // every time the machine reboots (on most platforms). 382 long nowMs = Time.now(); 383 long minTimeoutMs = Long.MAX_VALUE; 384 for (int i = 0; i < numBlockIters; i++) { 385 int idx = (curIdx + i + 1) % numBlockIters; 386 BlockIterator iter = blockIters.get(idx); 387 if (!iter.atEnd()) { 388 LOG.info("Now scanning bpid {} on volume {}", 389 iter.getBlockPoolId(), volume.getBasePath()); 390 curBlockIter = iter; 391 return 0L; 392 } 393 long iterStartMs = iter.getIterStartMs(); 394 long waitMs = (iterStartMs + conf.scanPeriodMs) - nowMs; 395 if (waitMs <= 0) { 396 iter.rewind(); 397 LOG.info("Now rescanning bpid {} on volume {}, after more than " + 398 "{} hour(s)", iter.getBlockPoolId(), volume.getBasePath(), 399 TimeUnit.HOURS.convert(conf.scanPeriodMs, TimeUnit.MILLISECONDS)); 400 curBlockIter = iter; 401 return 0L; 402 } 403 minTimeoutMs = Math.min(minTimeoutMs, waitMs); 404 } 405 LOG.info("{}: no suitable block pools found to scan. Waiting {} ms.", 406 this, minTimeoutMs); 407 return minTimeoutMs; 408 } 409 410 /** 411 * Scan a block. 412 * 413 * @param cblock The block to scan. 414 * @param bytesPerSec The bytes per second to scan at. 415 * 416 * @return The length of the block that was scanned, or 417 * -1 if the block could not be scanned. 418 */ 419 private long scanBlock(ExtendedBlock cblock, long bytesPerSec) { 420 // 'cblock' has a valid blockId and block pool id, but we don't yet know the 421 // genstamp the block is supposed to have. Ask the FsDatasetImpl for this 422 // information. 423 ExtendedBlock block = null; 424 try { 425 Block b = volume.getDataset().getStoredBlock( 426 cblock.getBlockPoolId(), cblock.getBlockId()); 427 if (b == null) { 428 LOG.info("FileNotFound while finding block {} on volume {}", 429 cblock, volume.getBasePath()); 430 } else { 431 block = new ExtendedBlock(cblock.getBlockPoolId(), b); 432 } 433 } catch (FileNotFoundException e) { 434 LOG.info("FileNotFoundException while finding block {} on volume {}", 435 cblock, volume.getBasePath()); 436 } catch (IOException e) { 437 LOG.warn("I/O error while finding block {} on volume {}", 438 cblock, volume.getBasePath()); 439 } 440 if (block == null) { 441 return -1; // block not found. 442 } 443 LOG.debug("start scanning block {}", block); 444 BlockSender blockSender = null; 445 try { 446 blockSender = new BlockSender(block, 0, -1, 447 false, true, true, datanode, null, 448 CachingStrategy.newDropBehind()); 449 throttler.setBandwidth(bytesPerSec); 450 long bytesRead = blockSender.sendBlock(nullStream, null, throttler); 451 resultHandler.handle(block, null); 452 metrics.incrBlocksVerified(); 453 return bytesRead; 454 } catch (IOException e) { 455 resultHandler.handle(block, e); 456 } finally { 457 IOUtils.cleanup(null, blockSender); 458 } 459 metrics.incrBlockVerificationFailures(); 460 return -1; 461 } 462 463 @VisibleForTesting 464 static boolean calculateShouldScan(String storageId, long targetBytesPerSec, 465 long scannedBytesSum, long startMinute, long curMinute) { 466 long runMinutes = curMinute - startMinute; 467 long effectiveBytesPerSec; 468 if (runMinutes <= 0) { 469 // avoid division by zero 470 effectiveBytesPerSec = scannedBytesSum; 471 } else { 472 if (runMinutes > MINUTES_PER_HOUR) { 473 // we only keep an hour's worth of rate information 474 runMinutes = MINUTES_PER_HOUR; 475 } 476 effectiveBytesPerSec = scannedBytesSum / 477 (SECONDS_PER_MINUTE * runMinutes); 478 } 479 480 boolean shouldScan = effectiveBytesPerSec <= targetBytesPerSec; 481 LOG.trace("{}: calculateShouldScan: effectiveBytesPerSec = {}, and " + 482 "targetBytesPerSec = {}. startMinute = {}, curMinute = {}, " + 483 "shouldScan = {}", 484 storageId, effectiveBytesPerSec, targetBytesPerSec, 485 startMinute, curMinute, shouldScan); 486 return shouldScan; 487 } 488 489 /** 490 * Run an iteration of the VolumeScanner loop. 491 * 492 * @param suspectBlock A suspect block which we should scan, or null to 493 * scan the next regularly scheduled block. 494 * 495 * @return The number of milliseconds to delay before running the loop 496 * again, or 0 to re-run the loop immediately. 497 */ 498 private long runLoop(ExtendedBlock suspectBlock) { 499 long bytesScanned = -1; 500 boolean scanError = false; 501 ExtendedBlock block = null; 502 try { 503 long monotonicMs = Time.monotonicNow(); 504 expireOldScannedBytesRecords(monotonicMs); 505 506 if (!calculateShouldScan(volume.getStorageID(), conf.targetBytesPerSec, 507 scannedBytesSum, startMinute, curMinute)) { 508 // If neededBytesPerSec is too low, then wait few seconds for some old 509 // scannedBytes records to expire. 510 return 30000L; 511 } 512 513 // Find a usable block pool to scan. 514 if (suspectBlock != null) { 515 block = suspectBlock; 516 } else { 517 if ((curBlockIter == null) || curBlockIter.atEnd()) { 518 long timeout = findNextUsableBlockIter(); 519 if (timeout > 0) { 520 LOG.trace("{}: no block pools are ready to scan yet. Waiting " + 521 "{} ms.", this, timeout); 522 synchronized (stats) { 523 stats.nextBlockPoolScanStartMs = Time.monotonicNow() + timeout; 524 } 525 return timeout; 526 } 527 synchronized (stats) { 528 stats.scansSinceRestart++; 529 stats.blocksScannedInCurrentPeriod = 0; 530 stats.nextBlockPoolScanStartMs = -1; 531 } 532 return 0L; 533 } 534 try { 535 block = curBlockIter.nextBlock(); 536 } catch (IOException e) { 537 // There was an error listing the next block in the volume. This is a 538 // serious issue. 539 LOG.warn("{}: nextBlock error on {}", this, curBlockIter); 540 // On the next loop iteration, curBlockIter#eof will be set to true, and 541 // we will pick a different block iterator. 542 return 0L; 543 } 544 if (block == null) { 545 // The BlockIterator is at EOF. 546 LOG.info("{}: finished scanning block pool {}", 547 this, curBlockIter.getBlockPoolId()); 548 saveBlockIterator(curBlockIter); 549 return 0; 550 } 551 } 552 if (curBlockIter != null) { 553 long saveDelta = monotonicMs - curBlockIter.getLastSavedMs(); 554 if (saveDelta >= conf.cursorSaveMs) { 555 LOG.debug("{}: saving block iterator {} after {} ms.", 556 this, curBlockIter, saveDelta); 557 saveBlockIterator(curBlockIter); 558 } 559 } 560 bytesScanned = scanBlock(block, conf.targetBytesPerSec); 561 if (bytesScanned >= 0) { 562 scannedBytesSum += bytesScanned; 563 scannedBytes[(int)(curMinute % MINUTES_PER_HOUR)] += bytesScanned; 564 } else { 565 scanError = true; 566 } 567 return 0L; 568 } finally { 569 synchronized (stats) { 570 stats.bytesScannedInPastHour = scannedBytesSum; 571 if (bytesScanned > 0) { 572 stats.blocksScannedInCurrentPeriod++; 573 stats.blocksScannedSinceRestart++; 574 } 575 if (scanError) { 576 stats.scanErrorsSinceRestart++; 577 } 578 if (block != null) { 579 stats.lastBlockScanned = block; 580 } 581 if (curBlockIter == null) { 582 stats.eof = true; 583 stats.blockPoolPeriodEndsMs = -1; 584 } else { 585 stats.eof = curBlockIter.atEnd(); 586 stats.blockPoolPeriodEndsMs = 587 curBlockIter.getIterStartMs() + conf.scanPeriodMs; 588 } 589 } 590 } 591 } 592 593 /** 594 * If there are elements in the suspectBlocks list, removes 595 * and returns the first one. Otherwise, returns null. 596 */ 597 private synchronized ExtendedBlock popNextSuspectBlock() { 598 Iterator<ExtendedBlock> iter = suspectBlocks.iterator(); 599 if (!iter.hasNext()) { 600 return null; 601 } 602 ExtendedBlock block = iter.next(); 603 iter.remove(); 604 return block; 605 } 606 607 @Override 608 public void run() { 609 // Record the minute on which the scanner started. 610 this.startMinute = 611 TimeUnit.MINUTES.convert(Time.monotonicNow(), TimeUnit.MILLISECONDS); 612 this.curMinute = startMinute; 613 try { 614 LOG.trace("{}: thread starting.", this); 615 resultHandler.setup(this); 616 try { 617 long timeout = 0; 618 while (true) { 619 ExtendedBlock suspectBlock = null; 620 // Take the lock to check if we should stop, and access the 621 // suspect block list. 622 synchronized (this) { 623 if (stopping) { 624 break; 625 } 626 if (timeout > 0) { 627 LOG.debug("{}: wait for {} milliseconds", this, timeout); 628 wait(timeout); 629 if (stopping) { 630 break; 631 } 632 } 633 suspectBlock = popNextSuspectBlock(); 634 } 635 timeout = runLoop(suspectBlock); 636 } 637 } catch (InterruptedException e) { 638 // We are exiting because of an InterruptedException, 639 // probably sent by VolumeScanner#shutdown. 640 LOG.trace("{} exiting because of InterruptedException.", this); 641 } catch (Throwable e) { 642 LOG.error("{} exiting because of exception ", this, e); 643 } 644 LOG.info("{} exiting.", this); 645 // Save the current position of all block iterators and close them. 646 for (BlockIterator iter : blockIters) { 647 saveBlockIterator(iter); 648 IOUtils.cleanup(null, iter); 649 } 650 } finally { 651 // When the VolumeScanner exits, release the reference we were holding 652 // on the volume. This will allow the volume to be removed later. 653 IOUtils.cleanup(null, ref); 654 } 655 } 656 657 @Override 658 public String toString() { 659 return "VolumeScanner(" + volume.getBasePath() + 660 ", " + volume.getStorageID() + ")"; 661 } 662 663 /** 664 * Shut down this scanner. 665 */ 666 public synchronized void shutdown() { 667 stopping = true; 668 notify(); 669 this.interrupt(); 670 } 671 672 673 public synchronized void markSuspectBlock(ExtendedBlock block) { 674 if (stopping) { 675 LOG.debug("{}: Not scheduling suspect block {} for " + 676 "rescanning, because this volume scanner is stopping.", this, block); 677 return; 678 } 679 Boolean recent = recentSuspectBlocks.getIfPresent(block); 680 if (recent != null) { 681 LOG.debug("{}: Not scheduling suspect block {} for " + 682 "rescanning, because we rescanned it recently.", this, block); 683 return; 684 } 685 if (suspectBlocks.contains(block)) { 686 LOG.debug("{}: suspect block {} is already queued for " + 687 "rescanning.", this, block); 688 return; 689 } 690 suspectBlocks.add(block); 691 recentSuspectBlocks.put(block, true); 692 LOG.debug("{}: Scheduling suspect block {} for rescanning.", this, block); 693 notify(); // wake scanner thread. 694 } 695 696 /** 697 * Allow the scanner to scan the given block pool. 698 * 699 * @param bpid The block pool id. 700 */ 701 public synchronized void enableBlockPoolId(String bpid) { 702 for (BlockIterator iter : blockIters) { 703 if (iter.getBlockPoolId().equals(bpid)) { 704 LOG.warn("{}: already enabled scanning on block pool {}", this, bpid); 705 return; 706 } 707 } 708 BlockIterator iter = null; 709 try { 710 // Load a block iterator for the next block pool on the volume. 711 iter = volume.loadBlockIterator(bpid, BLOCK_ITERATOR_NAME); 712 LOG.trace("{}: loaded block iterator for {}.", this, bpid); 713 } catch (FileNotFoundException e) { 714 LOG.debug("{}: failed to load block iterator: " + e.getMessage(), this); 715 } catch (IOException e) { 716 LOG.warn("{}: failed to load block iterator.", this, e); 717 } 718 if (iter == null) { 719 iter = volume.newBlockIterator(bpid, BLOCK_ITERATOR_NAME); 720 LOG.trace("{}: created new block iterator for {}.", this, bpid); 721 } 722 iter.setMaxStalenessMs(conf.maxStalenessMs); 723 blockIters.add(iter); 724 notify(); 725 } 726 727 /** 728 * Disallow the scanner from scanning the given block pool. 729 * 730 * @param bpid The block pool id. 731 */ 732 public synchronized void disableBlockPoolId(String bpid) { 733 Iterator<BlockIterator> i = blockIters.iterator(); 734 while (i.hasNext()) { 735 BlockIterator iter = i.next(); 736 if (iter.getBlockPoolId().equals(bpid)) { 737 LOG.trace("{}: disabling scanning on block pool {}", this, bpid); 738 i.remove(); 739 IOUtils.cleanup(null, iter); 740 if (curBlockIter == iter) { 741 curBlockIter = null; 742 } 743 notify(); 744 return; 745 } 746 } 747 LOG.warn("{}: can't remove block pool {}, because it was never " + 748 "added.", this, bpid); 749 } 750 751 @VisibleForTesting 752 Statistics getStatistics() { 753 synchronized (stats) { 754 return new Statistics(stats); 755 } 756 } 757}