001/*
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019package org.apache.hadoop.hbase.regionserver;
020
021import java.io.IOException;
022import java.io.InterruptedIOException;
023import java.net.InetSocketAddress;
024import java.util.ArrayList;
025import java.util.Collection;
026import java.util.Collections;
027import java.util.HashMap;
028import java.util.HashSet;
029import java.util.Iterator;
030import java.util.List;
031import java.util.Map;
032import java.util.Map.Entry;
033import java.util.NavigableSet;
034import java.util.Optional;
035import java.util.OptionalDouble;
036import java.util.OptionalInt;
037import java.util.OptionalLong;
038import java.util.Set;
039import java.util.concurrent.Callable;
040import java.util.concurrent.CompletionService;
041import java.util.concurrent.ConcurrentHashMap;
042import java.util.concurrent.ExecutionException;
043import java.util.concurrent.ExecutorCompletionService;
044import java.util.concurrent.Future;
045import java.util.concurrent.ThreadPoolExecutor;
046import java.util.concurrent.atomic.AtomicBoolean;
047import java.util.concurrent.atomic.AtomicInteger;
048import java.util.concurrent.atomic.AtomicLong;
049import java.util.concurrent.atomic.LongAdder;
050import java.util.concurrent.locks.ReentrantLock;
051import java.util.concurrent.locks.ReentrantReadWriteLock;
052import java.util.function.Predicate;
053import java.util.function.ToLongFunction;
054import java.util.stream.Collectors;
055import java.util.stream.LongStream;
056
057import org.apache.hadoop.conf.Configuration;
058import org.apache.hadoop.fs.FileSystem;
059import org.apache.hadoop.fs.Path;
060import org.apache.hadoop.fs.permission.FsAction;
061import org.apache.hadoop.hbase.Cell;
062import org.apache.hadoop.hbase.CellComparator;
063import org.apache.hadoop.hbase.CellUtil;
064import org.apache.hadoop.hbase.CompoundConfiguration;
065import org.apache.hadoop.hbase.HConstants;
066import org.apache.hadoop.hbase.MemoryCompactionPolicy;
067import org.apache.hadoop.hbase.TableName;
068import org.apache.hadoop.hbase.backup.FailedArchiveException;
069import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
070import org.apache.hadoop.hbase.client.RegionInfo;
071import org.apache.hadoop.hbase.client.Scan;
072import org.apache.hadoop.hbase.conf.ConfigurationManager;
073import org.apache.hadoop.hbase.conf.PropagatingConfigurationObserver;
074import org.apache.hadoop.hbase.io.HeapSize;
075import org.apache.hadoop.hbase.io.compress.Compression;
076import org.apache.hadoop.hbase.io.crypto.Encryption;
077import org.apache.hadoop.hbase.io.hfile.CacheConfig;
078import org.apache.hadoop.hbase.io.hfile.HFile;
079import org.apache.hadoop.hbase.io.hfile.HFileContext;
080import org.apache.hadoop.hbase.io.hfile.HFileContextBuilder;
081import org.apache.hadoop.hbase.io.hfile.HFileDataBlockEncoder;
082import org.apache.hadoop.hbase.io.hfile.HFileDataBlockEncoderImpl;
083import org.apache.hadoop.hbase.io.hfile.HFileScanner;
084import org.apache.hadoop.hbase.io.hfile.InvalidHFileException;
085import org.apache.hadoop.hbase.log.HBaseMarkers;
086import org.apache.hadoop.hbase.monitoring.MonitoredTask;
087import org.apache.hadoop.hbase.quotas.RegionSizeStore;
088import org.apache.hadoop.hbase.regionserver.compactions.CompactionContext;
089import org.apache.hadoop.hbase.regionserver.compactions.CompactionLifeCycleTracker;
090import org.apache.hadoop.hbase.regionserver.compactions.CompactionProgress;
091import org.apache.hadoop.hbase.regionserver.compactions.CompactionRequestImpl;
092import org.apache.hadoop.hbase.regionserver.compactions.DefaultCompactor;
093import org.apache.hadoop.hbase.regionserver.compactions.OffPeakHours;
094import org.apache.hadoop.hbase.regionserver.querymatcher.ScanQueryMatcher;
095import org.apache.hadoop.hbase.regionserver.throttle.ThroughputController;
096import org.apache.hadoop.hbase.regionserver.wal.WALUtil;
097import org.apache.hadoop.hbase.security.EncryptionUtil;
098import org.apache.hadoop.hbase.security.User;
099import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
100import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.CompactionDescriptor;
101import org.apache.hadoop.hbase.util.Bytes;
102import org.apache.hadoop.hbase.util.ChecksumType;
103import org.apache.hadoop.hbase.util.ClassSize;
104import org.apache.hadoop.hbase.util.CommonFSUtils;
105import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
106import org.apache.hadoop.hbase.util.Pair;
107import org.apache.hadoop.hbase.util.ReflectionUtils;
108import org.apache.hadoop.util.StringUtils;
109import org.apache.hadoop.util.StringUtils.TraditionalBinaryPrefix;
110import org.apache.yetus.audience.InterfaceAudience;
111import org.slf4j.Logger;
112import org.slf4j.LoggerFactory;
113import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
114import org.apache.hbase.thirdparty.com.google.common.base.Preconditions;
115import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableCollection;
116import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableList;
117import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
118import org.apache.hbase.thirdparty.com.google.common.collect.Maps;
119import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
120import org.apache.hbase.thirdparty.org.apache.commons.collections4.CollectionUtils;
121import org.apache.hbase.thirdparty.org.apache.commons.collections4.IterableUtils;
122/**
123 * A Store holds a column family in a Region.  Its a memstore and a set of zero
124 * or more StoreFiles, which stretch backwards over time.
125 *
126 * <p>There's no reason to consider append-logging at this level; all logging
127 * and locking is handled at the HRegion level.  Store just provides
128 * services to manage sets of StoreFiles.  One of the most important of those
129 * services is compaction services where files are aggregated once they pass
130 * a configurable threshold.
131 *
132 * <p>Locking and transactions are handled at a higher level.  This API should
133 * not be called directly but by an HRegion manager.
134 */
135@InterfaceAudience.Private
136public class HStore implements Store, HeapSize, StoreConfigInformation,
137    PropagatingConfigurationObserver {
138  public static final String MEMSTORE_CLASS_NAME = "hbase.regionserver.memstore.class";
139  public static final String COMPACTCHECKER_INTERVAL_MULTIPLIER_KEY =
140      "hbase.server.compactchecker.interval.multiplier";
141  public static final String BLOCKING_STOREFILES_KEY = "hbase.hstore.blockingStoreFiles";
142  public static final String BLOCK_STORAGE_POLICY_KEY = "hbase.hstore.block.storage.policy";
143  // keep in accordance with HDFS default storage policy
144  public static final String DEFAULT_BLOCK_STORAGE_POLICY = "HOT";
145  public static final int DEFAULT_COMPACTCHECKER_INTERVAL_MULTIPLIER = 1000;
146  public static final int DEFAULT_BLOCKING_STOREFILE_COUNT = 16;
147
148  // HBASE-24428 : Update compaction priority for recently split daughter regions
149  // so as to prioritize their compaction.
150  // Any compaction candidate with higher priority than compaction of newly split daugher regions
151  // should have priority value < (Integer.MIN_VALUE + 1000)
152  private static final int SPLIT_REGION_COMPACTION_PRIORITY = Integer.MIN_VALUE + 1000;
153
154  private static final Logger LOG = LoggerFactory.getLogger(HStore.class);
155
156  protected final MemStore memstore;
157  // This stores directory in the filesystem.
158  protected final HRegion region;
159  private final ColumnFamilyDescriptor family;
160  private final HRegionFileSystem fs;
161  protected Configuration conf;
162  protected CacheConfig cacheConf;
163  private long lastCompactSize = 0;
164  volatile boolean forceMajor = false;
165  private AtomicLong storeSize = new AtomicLong();
166  private AtomicLong totalUncompressedBytes = new AtomicLong();
167  private LongAdder memstoreOnlyRowReadsCount = new LongAdder();
168  // rows that has cells from both memstore and files (or only files)
169  private LongAdder mixedRowReadsCount = new LongAdder();
170
171  private boolean cacheOnWriteLogged;
172
173  /**
174   * RWLock for store operations.
175   * Locked in shared mode when the list of component stores is looked at:
176   *   - all reads/writes to table data
177   *   - checking for split
178   * Locked in exclusive mode when the list of component stores is modified:
179   *   - closing
180   *   - completing a compaction
181   */
182  final ReentrantReadWriteLock lock = new ReentrantReadWriteLock();
183  /**
184   * Lock specific to archiving compacted store files.  This avoids races around
185   * the combination of retrieving the list of compacted files and moving them to
186   * the archive directory.  Since this is usually a background process (other than
187   * on close), we don't want to handle this with the store write lock, which would
188   * block readers and degrade performance.
189   *
190   * Locked by:
191   *   - CompactedHFilesDispatchHandler via closeAndArchiveCompactedFiles()
192   *   - close()
193   */
194  final ReentrantLock archiveLock = new ReentrantLock();
195
196  private final boolean verifyBulkLoads;
197
198  /**
199   * Use this counter to track concurrent puts. If TRACE-log is enabled, if we are over the
200   * threshold set by hbase.region.store.parallel.put.print.threshold (Default is 50) we will
201   * log a message that identifies the Store experience this high-level of concurrency.
202   */
203  private final AtomicInteger currentParallelPutCount = new AtomicInteger(0);
204  private final int parallelPutCountPrintThreshold;
205
206  private ScanInfo scanInfo;
207
208  // All access must be synchronized.
209  // TODO: ideally, this should be part of storeFileManager, as we keep passing this to it.
210  private final List<HStoreFile> filesCompacting = Lists.newArrayList();
211
212  // All access must be synchronized.
213  private final Set<ChangedReadersObserver> changedReaderObservers =
214    Collections.newSetFromMap(new ConcurrentHashMap<ChangedReadersObserver, Boolean>());
215
216  protected final int blocksize;
217  private HFileDataBlockEncoder dataBlockEncoder;
218
219  /** Checksum configuration */
220  protected ChecksumType checksumType;
221  protected int bytesPerChecksum;
222
223  // Comparing KeyValues
224  protected final CellComparator comparator;
225
226  final StoreEngine<?, ?, ?, ?> storeEngine;
227
228  private static final AtomicBoolean offPeakCompactionTracker = new AtomicBoolean();
229  private volatile OffPeakHours offPeakHours;
230
231  private static final int DEFAULT_FLUSH_RETRIES_NUMBER = 10;
232  private int flushRetriesNumber;
233  private int pauseTime;
234
235  private long blockingFileCount;
236  private int compactionCheckMultiplier;
237  protected Encryption.Context cryptoContext = Encryption.Context.NONE;
238
239  private AtomicLong flushedCellsCount = new AtomicLong();
240  private AtomicLong compactedCellsCount = new AtomicLong();
241  private AtomicLong majorCompactedCellsCount = new AtomicLong();
242  private AtomicLong flushedCellsSize = new AtomicLong();
243  private AtomicLong flushedOutputFileSize = new AtomicLong();
244  private AtomicLong compactedCellsSize = new AtomicLong();
245  private AtomicLong majorCompactedCellsSize = new AtomicLong();
246
247  /**
248   * Constructor
249   * @param family HColumnDescriptor for this column
250   * @param confParam configuration object failed.  Can be null.
251   */
252  protected HStore(final HRegion region, final ColumnFamilyDescriptor family,
253      final Configuration confParam, boolean warmup) throws IOException {
254
255    this.fs = region.getRegionFileSystem();
256
257    // Assemble the store's home directory and Ensure it exists.
258    fs.createStoreDir(family.getNameAsString());
259    this.region = region;
260    this.family = family;
261    // 'conf' renamed to 'confParam' b/c we use this.conf in the constructor
262    // CompoundConfiguration will look for keys in reverse order of addition, so we'd
263    // add global config first, then table and cf overrides, then cf metadata.
264    this.conf = new CompoundConfiguration()
265      .add(confParam)
266      .addBytesMap(region.getTableDescriptor().getValues())
267      .addStringMap(family.getConfiguration())
268      .addBytesMap(family.getValues());
269    this.blocksize = family.getBlocksize();
270
271    // set block storage policy for store directory
272    String policyName = family.getStoragePolicy();
273    if (null == policyName) {
274      policyName = this.conf.get(BLOCK_STORAGE_POLICY_KEY, DEFAULT_BLOCK_STORAGE_POLICY);
275    }
276    this.fs.setStoragePolicy(family.getNameAsString(), policyName.trim());
277
278    this.dataBlockEncoder = new HFileDataBlockEncoderImpl(family.getDataBlockEncoding());
279
280    this.comparator = region.getCellComparator();
281    // used by ScanQueryMatcher
282    long timeToPurgeDeletes =
283        Math.max(conf.getLong("hbase.hstore.time.to.purge.deletes", 0), 0);
284    LOG.trace("Time to purge deletes set to {}ms in {}", timeToPurgeDeletes, this);
285    // Get TTL
286    long ttl = determineTTLFromFamily(family);
287    // Why not just pass a HColumnDescriptor in here altogether?  Even if have
288    // to clone it?
289    scanInfo = new ScanInfo(conf, family, ttl, timeToPurgeDeletes, this.comparator);
290    this.memstore = getMemstore();
291
292    this.offPeakHours = OffPeakHours.getInstance(conf);
293
294    // Setting up cache configuration for this family
295    createCacheConf(family);
296
297    this.verifyBulkLoads = conf.getBoolean("hbase.hstore.bulkload.verify", false);
298
299    this.blockingFileCount =
300        conf.getInt(BLOCKING_STOREFILES_KEY, DEFAULT_BLOCKING_STOREFILE_COUNT);
301    this.compactionCheckMultiplier = conf.getInt(
302        COMPACTCHECKER_INTERVAL_MULTIPLIER_KEY, DEFAULT_COMPACTCHECKER_INTERVAL_MULTIPLIER);
303    if (this.compactionCheckMultiplier <= 0) {
304      LOG.error("Compaction check period multiplier must be positive, setting default: {}",
305          DEFAULT_COMPACTCHECKER_INTERVAL_MULTIPLIER);
306      this.compactionCheckMultiplier = DEFAULT_COMPACTCHECKER_INTERVAL_MULTIPLIER;
307    }
308
309    this.storeEngine = createStoreEngine(this, this.conf, this.comparator);
310    List<HStoreFile> hStoreFiles = loadStoreFiles(warmup);
311    // Move the storeSize calculation out of loadStoreFiles() method, because the secondary read
312    // replica's refreshStoreFiles() will also use loadStoreFiles() to refresh its store files and
313    // update the storeSize in the completeCompaction(..) finally (just like compaction) , so
314    // no need calculate the storeSize twice.
315    this.storeSize.addAndGet(getStorefilesSize(hStoreFiles, sf -> true));
316    this.totalUncompressedBytes.addAndGet(getTotalUncompressedBytes(hStoreFiles));
317    this.storeEngine.getStoreFileManager().loadFiles(hStoreFiles);
318
319    // Initialize checksum type from name. The names are CRC32, CRC32C, etc.
320    this.checksumType = getChecksumType(conf);
321    // Initialize bytes per checksum
322    this.bytesPerChecksum = getBytesPerChecksum(conf);
323    flushRetriesNumber = conf.getInt(
324        "hbase.hstore.flush.retries.number", DEFAULT_FLUSH_RETRIES_NUMBER);
325    pauseTime = conf.getInt(HConstants.HBASE_SERVER_PAUSE, HConstants.DEFAULT_HBASE_SERVER_PAUSE);
326    if (flushRetriesNumber <= 0) {
327      throw new IllegalArgumentException(
328          "hbase.hstore.flush.retries.number must be > 0, not "
329              + flushRetriesNumber);
330    }
331    cryptoContext = EncryptionUtil.createEncryptionContext(conf, family);
332
333    int confPrintThreshold =
334        this.conf.getInt("hbase.region.store.parallel.put.print.threshold", 50);
335    if (confPrintThreshold < 10) {
336      confPrintThreshold = 10;
337    }
338    this.parallelPutCountPrintThreshold = confPrintThreshold;
339
340    LOG.info("Store={},  memstore type={}, storagePolicy={}, verifyBulkLoads={}, "
341            + "parallelPutCountPrintThreshold={}, encoding={}, compression={}",
342        this, memstore.getClass().getSimpleName(), policyName, verifyBulkLoads,
343        parallelPutCountPrintThreshold, family.getDataBlockEncoding(),
344        family.getCompressionType());
345    cacheOnWriteLogged = false;
346  }
347
348  /**
349   * @return MemStore Instance to use in this store.
350   */
351  private MemStore getMemstore() {
352    MemStore ms = null;
353    // Check if in-memory-compaction configured. Note MemoryCompactionPolicy is an enum!
354    MemoryCompactionPolicy inMemoryCompaction = null;
355    if (this.getTableName().isSystemTable()) {
356      inMemoryCompaction = MemoryCompactionPolicy.valueOf(
357          conf.get("hbase.systemtables.compacting.memstore.type", "NONE"));
358    } else {
359      inMemoryCompaction = family.getInMemoryCompaction();
360    }
361    if (inMemoryCompaction == null) {
362      inMemoryCompaction =
363          MemoryCompactionPolicy.valueOf(conf.get(CompactingMemStore.COMPACTING_MEMSTORE_TYPE_KEY,
364              CompactingMemStore.COMPACTING_MEMSTORE_TYPE_DEFAULT).toUpperCase());
365    }
366    switch (inMemoryCompaction) {
367      case NONE:
368        ms = ReflectionUtils.newInstance(DefaultMemStore.class,
369            new Object[] { conf, this.comparator,
370                this.getHRegion().getRegionServicesForStores()});
371        break;
372      default:
373        Class<? extends CompactingMemStore> clz = conf.getClass(MEMSTORE_CLASS_NAME,
374            CompactingMemStore.class, CompactingMemStore.class);
375        ms = ReflectionUtils.newInstance(clz, new Object[]{conf, this.comparator, this,
376            this.getHRegion().getRegionServicesForStores(), inMemoryCompaction});
377    }
378    return ms;
379  }
380
381  /**
382   * Creates the cache config.
383   * @param family The current column family.
384   */
385  protected void createCacheConf(final ColumnFamilyDescriptor family) {
386    this.cacheConf = new CacheConfig(conf, family, region.getBlockCache(),
387        region.getRegionServicesForStores().getByteBuffAllocator());
388    LOG.info("Created cacheConfig: " + this.getCacheConfig() + " for " + this);
389  }
390
391  /**
392   * Creates the store engine configured for the given Store.
393   * @param store The store. An unfortunate dependency needed due to it
394   *              being passed to coprocessors via the compactor.
395   * @param conf Store configuration.
396   * @param kvComparator KVComparator for storeFileManager.
397   * @return StoreEngine to use.
398   */
399  protected StoreEngine<?, ?, ?, ?> createStoreEngine(HStore store, Configuration conf,
400      CellComparator kvComparator) throws IOException {
401    return StoreEngine.create(store, conf, comparator);
402  }
403
404  /**
405   * @return TTL in seconds of the specified family
406   */
407  public static long determineTTLFromFamily(final ColumnFamilyDescriptor family) {
408    // HCD.getTimeToLive returns ttl in seconds.  Convert to milliseconds.
409    long ttl = family.getTimeToLive();
410    if (ttl == HConstants.FOREVER) {
411      // Default is unlimited ttl.
412      ttl = Long.MAX_VALUE;
413    } else if (ttl == -1) {
414      ttl = Long.MAX_VALUE;
415    } else {
416      // Second -> ms adjust for user data
417      ttl *= 1000;
418    }
419    return ttl;
420  }
421
422  @Override
423  public String getColumnFamilyName() {
424    return this.family.getNameAsString();
425  }
426
427  @Override
428  public TableName getTableName() {
429    return this.getRegionInfo().getTable();
430  }
431
432  @Override
433  public FileSystem getFileSystem() {
434    return this.fs.getFileSystem();
435  }
436
437  public HRegionFileSystem getRegionFileSystem() {
438    return this.fs;
439  }
440
441  /* Implementation of StoreConfigInformation */
442  @Override
443  public long getStoreFileTtl() {
444    // TTL only applies if there's no MIN_VERSIONs setting on the column.
445    return (this.scanInfo.getMinVersions() == 0) ? this.scanInfo.getTtl() : Long.MAX_VALUE;
446  }
447
448  @Override
449  public long getMemStoreFlushSize() {
450    // TODO: Why is this in here?  The flushsize of the region rather than the store?  St.Ack
451    return this.region.memstoreFlushSize;
452  }
453
454  @Override
455  public MemStoreSize getFlushableSize() {
456    return this.memstore.getFlushableSize();
457  }
458
459  @Override
460  public MemStoreSize getSnapshotSize() {
461    return this.memstore.getSnapshotSize();
462  }
463
464  @Override
465  public long getCompactionCheckMultiplier() {
466    return this.compactionCheckMultiplier;
467  }
468
469  @Override
470  public long getBlockingFileCount() {
471    return blockingFileCount;
472  }
473  /* End implementation of StoreConfigInformation */
474
475  /**
476   * Returns the configured bytesPerChecksum value.
477   * @param conf The configuration
478   * @return The bytesPerChecksum that is set in the configuration
479   */
480  public static int getBytesPerChecksum(Configuration conf) {
481    return conf.getInt(HConstants.BYTES_PER_CHECKSUM,
482                       HFile.DEFAULT_BYTES_PER_CHECKSUM);
483  }
484
485  /**
486   * Returns the configured checksum algorithm.
487   * @param conf The configuration
488   * @return The checksum algorithm that is set in the configuration
489   */
490  public static ChecksumType getChecksumType(Configuration conf) {
491    String checksumName = conf.get(HConstants.CHECKSUM_TYPE_NAME);
492    if (checksumName == null) {
493      return ChecksumType.getDefaultChecksumType();
494    } else {
495      return ChecksumType.nameToType(checksumName);
496    }
497  }
498
499  @Override
500  public ColumnFamilyDescriptor getColumnFamilyDescriptor() {
501    return this.family;
502  }
503
504  @Override
505  public OptionalLong getMaxSequenceId() {
506    return StoreUtils.getMaxSequenceIdInList(this.getStorefiles());
507  }
508
509  @Override
510  public OptionalLong getMaxMemStoreTS() {
511    return StoreUtils.getMaxMemStoreTSInList(this.getStorefiles());
512  }
513
514  /**
515   * @param tabledir {@link Path} to where the table is being stored
516   * @param hri {@link RegionInfo} for the region.
517   * @param family {@link ColumnFamilyDescriptor} describing the column family
518   * @return Path to family/Store home directory.
519   * @deprecated Since 05/05/2013, HBase-7808, hbase-1.0.0
520   */
521  @Deprecated
522  public static Path getStoreHomedir(final Path tabledir,
523      final RegionInfo hri, final byte[] family) {
524    return getStoreHomedir(tabledir, hri.getEncodedName(), family);
525  }
526
527  /**
528   * @param tabledir {@link Path} to where the table is being stored
529   * @param encodedName Encoded region name.
530   * @param family {@link ColumnFamilyDescriptor} describing the column family
531   * @return Path to family/Store home directory.
532   * @deprecated Since 05/05/2013, HBase-7808, hbase-1.0.0
533   */
534  @Deprecated
535  public static Path getStoreHomedir(final Path tabledir,
536      final String encodedName, final byte[] family) {
537    return new Path(tabledir, new Path(encodedName, Bytes.toString(family)));
538  }
539
540  /**
541   * @return the data block encoder
542   */
543  public HFileDataBlockEncoder getDataBlockEncoder() {
544    return dataBlockEncoder;
545  }
546
547  /**
548   * Should be used only in tests.
549   * @param blockEncoder the block delta encoder to use
550   */
551  void setDataBlockEncoderInTest(HFileDataBlockEncoder blockEncoder) {
552    this.dataBlockEncoder = blockEncoder;
553  }
554
555  /**
556   * Creates an unsorted list of StoreFile loaded in parallel
557   * from the given directory.
558   */
559  private List<HStoreFile> loadStoreFiles(boolean warmup) throws IOException {
560    Collection<StoreFileInfo> files = fs.getStoreFiles(getColumnFamilyName());
561    return openStoreFiles(files, warmup);
562  }
563
564  private List<HStoreFile> openStoreFiles(Collection<StoreFileInfo> files, boolean warmup)
565      throws IOException {
566    if (CollectionUtils.isEmpty(files)) {
567      return Collections.emptyList();
568    }
569    // initialize the thread pool for opening store files in parallel..
570    ThreadPoolExecutor storeFileOpenerThreadPool =
571      this.region.getStoreFileOpenAndCloseThreadPool("StoreFileOpener-"
572        + this.region.getRegionInfo().getEncodedName() + "-" + this.getColumnFamilyName());
573    CompletionService<HStoreFile> completionService =
574      new ExecutorCompletionService<>(storeFileOpenerThreadPool);
575
576    int totalValidStoreFile = 0;
577    for (StoreFileInfo storeFileInfo : files) {
578      // open each store file in parallel
579      completionService.submit(() -> this.createStoreFileAndReader(storeFileInfo));
580      totalValidStoreFile++;
581    }
582
583    Set<String> compactedStoreFiles = new HashSet<>();
584    ArrayList<HStoreFile> results = new ArrayList<>(files.size());
585    IOException ioe = null;
586    try {
587      for (int i = 0; i < totalValidStoreFile; i++) {
588        try {
589          HStoreFile storeFile = completionService.take().get();
590          if (storeFile != null) {
591            LOG.debug("loaded {}", storeFile);
592            results.add(storeFile);
593            compactedStoreFiles.addAll(storeFile.getCompactedStoreFiles());
594          }
595        } catch (InterruptedException e) {
596          if (ioe == null) {
597            ioe = new InterruptedIOException(e.getMessage());
598          }
599        } catch (ExecutionException e) {
600          if (ioe == null) {
601            ioe = new IOException(e.getCause());
602          }
603        }
604      }
605    } finally {
606      storeFileOpenerThreadPool.shutdownNow();
607    }
608    if (ioe != null) {
609      // close StoreFile readers
610      boolean evictOnClose =
611          cacheConf != null? cacheConf.shouldEvictOnClose(): true;
612      for (HStoreFile file : results) {
613        try {
614          if (file != null) {
615            file.closeStoreFile(evictOnClose);
616          }
617        } catch (IOException e) {
618          LOG.warn("Could not close store file {}", file, e);
619        }
620      }
621      throw ioe;
622    }
623
624    // Should not archive the compacted store files when region warmup. See HBASE-22163.
625    if (!warmup) {
626      // Remove the compacted files from result
627      List<HStoreFile> filesToRemove = new ArrayList<>(compactedStoreFiles.size());
628      for (HStoreFile storeFile : results) {
629        if (compactedStoreFiles.contains(storeFile.getPath().getName())) {
630          LOG.warn("Clearing the compacted storefile {} from {}", storeFile, this);
631          storeFile.getReader().close(storeFile.getCacheConf() != null ?
632            storeFile.getCacheConf().shouldEvictOnClose() : true);
633          filesToRemove.add(storeFile);
634        }
635      }
636      results.removeAll(filesToRemove);
637      if (!filesToRemove.isEmpty() && this.isPrimaryReplicaStore()) {
638        LOG.debug("Moving the files {} to archive", filesToRemove);
639        this.fs.removeStoreFiles(this.getColumnFamilyDescriptor().getNameAsString(), filesToRemove);
640      }
641    }
642
643    return results;
644  }
645
646  @Override
647  public void refreshStoreFiles() throws IOException {
648    Collection<StoreFileInfo> newFiles = fs.getStoreFiles(getColumnFamilyName());
649    refreshStoreFilesInternal(newFiles);
650  }
651
652  /**
653   * Replaces the store files that the store has with the given files. Mainly used by secondary
654   * region replicas to keep up to date with the primary region files.
655   */
656  public void refreshStoreFiles(Collection<String> newFiles) throws IOException {
657    List<StoreFileInfo> storeFiles = new ArrayList<>(newFiles.size());
658    for (String file : newFiles) {
659      storeFiles.add(fs.getStoreFileInfo(getColumnFamilyName(), file));
660    }
661    refreshStoreFilesInternal(storeFiles);
662  }
663
664  /**
665   * Checks the underlying store files, and opens the files that  have not
666   * been opened, and removes the store file readers for store files no longer
667   * available. Mainly used by secondary region replicas to keep up to date with
668   * the primary region files.
669   */
670  private void refreshStoreFilesInternal(Collection<StoreFileInfo> newFiles) throws IOException {
671    StoreFileManager sfm = storeEngine.getStoreFileManager();
672    Collection<HStoreFile> currentFiles = sfm.getStorefiles();
673    Collection<HStoreFile> compactedFiles = sfm.getCompactedfiles();
674    if (currentFiles == null) {
675      currentFiles = Collections.emptySet();
676    }
677    if (newFiles == null) {
678      newFiles = Collections.emptySet();
679    }
680    if (compactedFiles == null) {
681      compactedFiles = Collections.emptySet();
682    }
683
684    HashMap<StoreFileInfo, HStoreFile> currentFilesSet = new HashMap<>(currentFiles.size());
685    for (HStoreFile sf : currentFiles) {
686      currentFilesSet.put(sf.getFileInfo(), sf);
687    }
688    HashMap<StoreFileInfo, HStoreFile> compactedFilesSet = new HashMap<>(compactedFiles.size());
689    for (HStoreFile sf : compactedFiles) {
690      compactedFilesSet.put(sf.getFileInfo(), sf);
691    }
692
693    Set<StoreFileInfo> newFilesSet = new HashSet<StoreFileInfo>(newFiles);
694    // Exclude the files that have already been compacted
695    newFilesSet = Sets.difference(newFilesSet, compactedFilesSet.keySet());
696    Set<StoreFileInfo> toBeAddedFiles = Sets.difference(newFilesSet, currentFilesSet.keySet());
697    Set<StoreFileInfo> toBeRemovedFiles = Sets.difference(currentFilesSet.keySet(), newFilesSet);
698
699    if (toBeAddedFiles.isEmpty() && toBeRemovedFiles.isEmpty()) {
700      return;
701    }
702
703    LOG.info("Refreshing store files for " + this + " files to add: "
704      + toBeAddedFiles + " files to remove: " + toBeRemovedFiles);
705
706    Set<HStoreFile> toBeRemovedStoreFiles = new HashSet<>(toBeRemovedFiles.size());
707    for (StoreFileInfo sfi : toBeRemovedFiles) {
708      toBeRemovedStoreFiles.add(currentFilesSet.get(sfi));
709    }
710
711    // try to open the files
712    List<HStoreFile> openedFiles = openStoreFiles(toBeAddedFiles, false);
713
714    // propogate the file changes to the underlying store file manager
715    replaceStoreFiles(toBeRemovedStoreFiles, openedFiles); //won't throw an exception
716
717    // Advance the memstore read point to be at least the new store files seqIds so that
718    // readers might pick it up. This assumes that the store is not getting any writes (otherwise
719    // in-flight transactions might be made visible)
720    if (!toBeAddedFiles.isEmpty()) {
721      // we must have the max sequence id here as we do have several store files
722      region.getMVCC().advanceTo(this.getMaxSequenceId().getAsLong());
723    }
724
725    completeCompaction(toBeRemovedStoreFiles);
726  }
727
728  @VisibleForTesting
729  protected HStoreFile createStoreFileAndReader(final Path p) throws IOException {
730    StoreFileInfo info = new StoreFileInfo(conf, this.getFileSystem(),
731        p, isPrimaryReplicaStore());
732    return createStoreFileAndReader(info);
733  }
734
735  private HStoreFile createStoreFileAndReader(StoreFileInfo info) throws IOException {
736    info.setRegionCoprocessorHost(this.region.getCoprocessorHost());
737    HStoreFile storeFile = new HStoreFile(info, this.family.getBloomFilterType(), this.cacheConf);
738    storeFile.initReader();
739    return storeFile;
740  }
741
742  /**
743   * This message intends to inform the MemStore that next coming updates
744   * are going to be part of the replaying edits from WAL
745   */
746  public void startReplayingFromWAL(){
747    this.memstore.startReplayingFromWAL();
748  }
749
750  /**
751   * This message intends to inform the MemStore that the replaying edits from WAL
752   * are done
753   */
754  public void stopReplayingFromWAL(){
755    this.memstore.stopReplayingFromWAL();
756  }
757
758  /**
759   * Adds a value to the memstore
760   */
761  public void add(final Cell cell, MemStoreSizing memstoreSizing) {
762    lock.readLock().lock();
763    try {
764      if (this.currentParallelPutCount.getAndIncrement() > this.parallelPutCountPrintThreshold) {
765        LOG.trace("tableName={}, encodedName={}, columnFamilyName={} is too busy!",
766            this.getTableName(), this.getRegionInfo().getEncodedName(), this.getColumnFamilyName());
767      }
768      this.memstore.add(cell, memstoreSizing);
769    } finally {
770      lock.readLock().unlock();
771      currentParallelPutCount.decrementAndGet();
772    }
773  }
774
775  /**
776   * Adds the specified value to the memstore
777   */
778  public void add(final Iterable<Cell> cells, MemStoreSizing memstoreSizing) {
779    lock.readLock().lock();
780    try {
781      if (this.currentParallelPutCount.getAndIncrement() > this.parallelPutCountPrintThreshold) {
782        LOG.trace("tableName={}, encodedName={}, columnFamilyName={} is too busy!",
783            this.getTableName(), this.getRegionInfo().getEncodedName(), this.getColumnFamilyName());
784      }
785      memstore.add(cells, memstoreSizing);
786    } finally {
787      lock.readLock().unlock();
788      currentParallelPutCount.decrementAndGet();
789    }
790  }
791
792  @Override
793  public long timeOfOldestEdit() {
794    return memstore.timeOfOldestEdit();
795  }
796
797  /**
798   * @return All store files.
799   */
800  @Override
801  public Collection<HStoreFile> getStorefiles() {
802    return this.storeEngine.getStoreFileManager().getStorefiles();
803  }
804
805  @Override
806  public Collection<HStoreFile> getCompactedFiles() {
807    return this.storeEngine.getStoreFileManager().getCompactedfiles();
808  }
809
810  /**
811   * This throws a WrongRegionException if the HFile does not fit in this region, or an
812   * InvalidHFileException if the HFile is not valid.
813   */
814  public void assertBulkLoadHFileOk(Path srcPath) throws IOException {
815    HFile.Reader reader  = null;
816    try {
817      LOG.info("Validating hfile at " + srcPath + " for inclusion in " + this);
818      FileSystem srcFs = srcPath.getFileSystem(conf);
819      srcFs.access(srcPath, FsAction.READ_WRITE);
820      reader = HFile.createReader(srcFs, srcPath, cacheConf, isPrimaryReplicaStore(), conf);
821
822      Optional<byte[]> firstKey = reader.getFirstRowKey();
823      Preconditions.checkState(firstKey.isPresent(), "First key can not be null");
824      Optional<Cell> lk = reader.getLastKey();
825      Preconditions.checkState(lk.isPresent(), "Last key can not be null");
826      byte[] lastKey =  CellUtil.cloneRow(lk.get());
827
828      if (LOG.isDebugEnabled()) {
829        LOG.debug("HFile bounds: first=" + Bytes.toStringBinary(firstKey.get()) +
830            " last=" + Bytes.toStringBinary(lastKey));
831        LOG.debug("Region bounds: first=" +
832            Bytes.toStringBinary(getRegionInfo().getStartKey()) +
833            " last=" + Bytes.toStringBinary(getRegionInfo().getEndKey()));
834      }
835
836      if (!this.getRegionInfo().containsRange(firstKey.get(), lastKey)) {
837        throw new WrongRegionException(
838            "Bulk load file " + srcPath.toString() + " does not fit inside region "
839            + this.getRegionInfo().getRegionNameAsString());
840      }
841
842      if(reader.length() > conf.getLong(HConstants.HREGION_MAX_FILESIZE,
843          HConstants.DEFAULT_MAX_FILE_SIZE)) {
844        LOG.warn("Trying to bulk load hfile " + srcPath + " with size: " +
845            reader.length() + " bytes can be problematic as it may lead to oversplitting.");
846      }
847
848      if (verifyBulkLoads) {
849        long verificationStartTime = EnvironmentEdgeManager.currentTime();
850        LOG.info("Full verification started for bulk load hfile: {}", srcPath);
851        Cell prevCell = null;
852        HFileScanner scanner = reader.getScanner(false, false, false);
853        scanner.seekTo();
854        do {
855          Cell cell = scanner.getCell();
856          if (prevCell != null) {
857            if (comparator.compareRows(prevCell, cell) > 0) {
858              throw new InvalidHFileException("Previous row is greater than"
859                  + " current row: path=" + srcPath + " previous="
860                  + CellUtil.getCellKeyAsString(prevCell) + " current="
861                  + CellUtil.getCellKeyAsString(cell));
862            }
863            if (CellComparator.getInstance().compareFamilies(prevCell, cell) != 0) {
864              throw new InvalidHFileException("Previous key had different"
865                  + " family compared to current key: path=" + srcPath
866                  + " previous="
867                  + Bytes.toStringBinary(prevCell.getFamilyArray(), prevCell.getFamilyOffset(),
868                      prevCell.getFamilyLength())
869                  + " current="
870                  + Bytes.toStringBinary(cell.getFamilyArray(), cell.getFamilyOffset(),
871                      cell.getFamilyLength()));
872            }
873          }
874          prevCell = cell;
875        } while (scanner.next());
876        LOG.info("Full verification complete for bulk load hfile: " + srcPath.toString() +
877          " took " + (EnvironmentEdgeManager.currentTime() - verificationStartTime) + " ms");
878      }
879    } finally {
880      if (reader != null) {
881        reader.close();
882      }
883    }
884  }
885
886  /**
887   * This method should only be called from Region. It is assumed that the ranges of values in the
888   * HFile fit within the stores assigned region. (assertBulkLoadHFileOk checks this)
889   *
890   * @param seqNum sequence Id associated with the HFile
891   */
892  public Pair<Path, Path> preBulkLoadHFile(String srcPathStr, long seqNum) throws IOException {
893    Path srcPath = new Path(srcPathStr);
894    return fs.bulkLoadStoreFile(getColumnFamilyName(), srcPath, seqNum);
895  }
896
897  public Path bulkLoadHFile(byte[] family, String srcPathStr, Path dstPath) throws IOException {
898    Path srcPath = new Path(srcPathStr);
899    try {
900      fs.commitStoreFile(srcPath, dstPath);
901    } finally {
902      if (this.getCoprocessorHost() != null) {
903        this.getCoprocessorHost().postCommitStoreFile(family, srcPath, dstPath);
904      }
905    }
906
907    LOG.info("Loaded HFile " + srcPath + " into " + this + " as "
908        + dstPath + " - updating store file list.");
909
910    HStoreFile sf = createStoreFileAndReader(dstPath);
911    bulkLoadHFile(sf);
912
913    LOG.info("Successfully loaded {} into {} (new location: {})",
914        srcPath, this, dstPath);
915
916    return dstPath;
917  }
918
919  public void bulkLoadHFile(StoreFileInfo fileInfo) throws IOException {
920    HStoreFile sf = createStoreFileAndReader(fileInfo);
921    bulkLoadHFile(sf);
922  }
923
924  private void bulkLoadHFile(HStoreFile sf) throws IOException {
925    StoreFileReader r = sf.getReader();
926    this.storeSize.addAndGet(r.length());
927    this.totalUncompressedBytes.addAndGet(r.getTotalUncompressedBytes());
928
929    // Append the new storefile into the list
930    this.lock.writeLock().lock();
931    try {
932      this.storeEngine.getStoreFileManager().insertNewFiles(Lists.newArrayList(sf));
933    } finally {
934      // We need the lock, as long as we are updating the storeFiles
935      // or changing the memstore. Let us release it before calling
936      // notifyChangeReadersObservers. See HBASE-4485 for a possible
937      // deadlock scenario that could have happened if continue to hold
938      // the lock.
939      this.lock.writeLock().unlock();
940    }
941    LOG.info("Loaded HFile " + sf.getFileInfo() + " into " + this);
942    if (LOG.isTraceEnabled()) {
943      String traceMessage = "BULK LOAD time,size,store size,store files ["
944          + EnvironmentEdgeManager.currentTime() + "," + r.length() + "," + storeSize
945          + "," + storeEngine.getStoreFileManager().getStorefileCount() + "]";
946      LOG.trace(traceMessage);
947    }
948  }
949
950  /**
951   * Close all the readers We don't need to worry about subsequent requests because the Region holds
952   * a write lock that will prevent any more reads or writes.
953   * @return the {@link StoreFile StoreFiles} that were previously being used.
954   * @throws IOException on failure
955   */
956  public ImmutableCollection<HStoreFile> close() throws IOException {
957    this.archiveLock.lock();
958    this.lock.writeLock().lock();
959    try {
960      // Clear so metrics doesn't find them.
961      ImmutableCollection<HStoreFile> result = storeEngine.getStoreFileManager().clearFiles();
962      Collection<HStoreFile> compactedfiles =
963          storeEngine.getStoreFileManager().clearCompactedFiles();
964      // clear the compacted files
965      if (CollectionUtils.isNotEmpty(compactedfiles)) {
966        removeCompactedfiles(compactedfiles, cacheConf != null ?
967          cacheConf.shouldEvictOnClose() : true);
968      }
969      if (!result.isEmpty()) {
970        // initialize the thread pool for closing store files in parallel.
971        ThreadPoolExecutor storeFileCloserThreadPool = this.region
972            .getStoreFileOpenAndCloseThreadPool("StoreFileCloser-"
973              + this.region.getRegionInfo().getEncodedName() + "-" + this.getColumnFamilyName());
974
975        // close each store file in parallel
976        CompletionService<Void> completionService =
977          new ExecutorCompletionService<>(storeFileCloserThreadPool);
978        for (HStoreFile f : result) {
979          completionService.submit(new Callable<Void>() {
980            @Override
981            public Void call() throws IOException {
982              boolean evictOnClose =
983                  cacheConf != null? cacheConf.shouldEvictOnClose(): true;
984              f.closeStoreFile(evictOnClose);
985              return null;
986            }
987          });
988        }
989
990        IOException ioe = null;
991        try {
992          for (int i = 0; i < result.size(); i++) {
993            try {
994              Future<Void> future = completionService.take();
995              future.get();
996            } catch (InterruptedException e) {
997              if (ioe == null) {
998                ioe = new InterruptedIOException();
999                ioe.initCause(e);
1000              }
1001            } catch (ExecutionException e) {
1002              if (ioe == null) {
1003                ioe = new IOException(e.getCause());
1004              }
1005            }
1006          }
1007        } finally {
1008          storeFileCloserThreadPool.shutdownNow();
1009        }
1010        if (ioe != null) {
1011          throw ioe;
1012        }
1013      }
1014      LOG.trace("Closed {}", this);
1015      return result;
1016    } finally {
1017      this.lock.writeLock().unlock();
1018      this.archiveLock.unlock();
1019    }
1020  }
1021
1022  /**
1023   * Snapshot this stores memstore. Call before running
1024   * {@link #flushCache(long, MemStoreSnapshot, MonitoredTask, ThroughputController,
1025   * FlushLifeCycleTracker)}
1026   *  so it has some work to do.
1027   */
1028  void snapshot() {
1029    this.lock.writeLock().lock();
1030    try {
1031      this.memstore.snapshot();
1032    } finally {
1033      this.lock.writeLock().unlock();
1034    }
1035  }
1036
1037  /**
1038   * Write out current snapshot. Presumes {@link #snapshot()} has been called previously.
1039   * @param logCacheFlushId flush sequence number
1040   * @return The path name of the tmp file to which the store was flushed
1041   * @throws IOException if exception occurs during process
1042   */
1043  protected List<Path> flushCache(final long logCacheFlushId, MemStoreSnapshot snapshot,
1044      MonitoredTask status, ThroughputController throughputController,
1045      FlushLifeCycleTracker tracker) throws IOException {
1046    // If an exception happens flushing, we let it out without clearing
1047    // the memstore snapshot.  The old snapshot will be returned when we say
1048    // 'snapshot', the next time flush comes around.
1049    // Retry after catching exception when flushing, otherwise server will abort
1050    // itself
1051    StoreFlusher flusher = storeEngine.getStoreFlusher();
1052    IOException lastException = null;
1053    for (int i = 0; i < flushRetriesNumber; i++) {
1054      try {
1055        List<Path> pathNames =
1056            flusher.flushSnapshot(snapshot, logCacheFlushId, status, throughputController, tracker);
1057        Path lastPathName = null;
1058        try {
1059          for (Path pathName : pathNames) {
1060            lastPathName = pathName;
1061            validateStoreFile(pathName);
1062          }
1063          return pathNames;
1064        } catch (Exception e) {
1065          LOG.warn("Failed validating store file {}, retrying num={}", lastPathName, i, e);
1066          if (e instanceof IOException) {
1067            lastException = (IOException) e;
1068          } else {
1069            lastException = new IOException(e);
1070          }
1071        }
1072      } catch (IOException e) {
1073        LOG.warn("Failed flushing store file for {}, retrying num={}", this, i, e);
1074        lastException = e;
1075      }
1076      if (lastException != null && i < (flushRetriesNumber - 1)) {
1077        try {
1078          Thread.sleep(pauseTime);
1079        } catch (InterruptedException e) {
1080          IOException iie = new InterruptedIOException();
1081          iie.initCause(e);
1082          throw iie;
1083        }
1084      }
1085    }
1086    throw lastException;
1087  }
1088
1089  public HStoreFile tryCommitRecoveredHFile(Path path) throws IOException {
1090    LOG.info("Validating recovered hfile at {} for inclusion in store {}", path, this);
1091    FileSystem srcFs = path.getFileSystem(conf);
1092    srcFs.access(path, FsAction.READ_WRITE);
1093    try (HFile.Reader reader =
1094        HFile.createReader(srcFs, path, cacheConf, isPrimaryReplicaStore(), conf)) {
1095      Optional<byte[]> firstKey = reader.getFirstRowKey();
1096      Preconditions.checkState(firstKey.isPresent(), "First key can not be null");
1097      Optional<Cell> lk = reader.getLastKey();
1098      Preconditions.checkState(lk.isPresent(), "Last key can not be null");
1099      byte[] lastKey = CellUtil.cloneRow(lk.get());
1100      if (!this.getRegionInfo().containsRange(firstKey.get(), lastKey)) {
1101        throw new WrongRegionException("Recovered hfile " + path.toString() +
1102            " does not fit inside region " + this.getRegionInfo().getRegionNameAsString());
1103      }
1104    }
1105
1106    Path dstPath = fs.commitStoreFile(getColumnFamilyName(), path);
1107    HStoreFile sf = createStoreFileAndReader(dstPath);
1108    StoreFileReader r = sf.getReader();
1109    this.storeSize.addAndGet(r.length());
1110    this.totalUncompressedBytes.addAndGet(r.getTotalUncompressedBytes());
1111
1112    this.lock.writeLock().lock();
1113    try {
1114      this.storeEngine.getStoreFileManager().insertNewFiles(Lists.newArrayList(sf));
1115    } finally {
1116      this.lock.writeLock().unlock();
1117    }
1118
1119    LOG.info("Loaded recovered hfile to {}, entries={}, sequenceid={}, filesize={}", sf,
1120      r.getEntries(), r.getSequenceID(), TraditionalBinaryPrefix.long2String(r.length(), "B", 1));
1121    return sf;
1122  }
1123
1124  /**
1125   * @param path The pathname of the tmp file into which the store was flushed
1126   * @return store file created.
1127   */
1128  private HStoreFile commitFile(Path path, long logCacheFlushId, MonitoredTask status)
1129      throws IOException {
1130    // Write-out finished successfully, move into the right spot
1131    Path dstPath = fs.commitStoreFile(getColumnFamilyName(), path);
1132
1133    status.setStatus("Flushing " + this + ": reopening flushed file");
1134    HStoreFile sf = createStoreFileAndReader(dstPath);
1135
1136    StoreFileReader r = sf.getReader();
1137    this.storeSize.addAndGet(r.length());
1138    this.totalUncompressedBytes.addAndGet(r.getTotalUncompressedBytes());
1139
1140    if (LOG.isInfoEnabled()) {
1141      LOG.info("Added " + sf + ", entries=" + r.getEntries() +
1142        ", sequenceid=" + logCacheFlushId +
1143        ", filesize=" + TraditionalBinaryPrefix.long2String(r.length(), "", 1));
1144    }
1145    return sf;
1146  }
1147
1148  public StoreFileWriter createWriterInTmp(long maxKeyCount, Compression.Algorithm compression,
1149    boolean isCompaction, boolean includeMVCCReadpoint, boolean includesTag,
1150    boolean shouldDropBehind) throws IOException {
1151    return createWriterInTmp(maxKeyCount, compression, isCompaction, includeMVCCReadpoint,
1152      includesTag, shouldDropBehind, -1, HConstants.EMPTY_STRING);
1153  }
1154
1155  /**
1156   * @param compression Compression algorithm to use
1157   * @param isCompaction whether we are creating a new file in a compaction
1158   * @param includeMVCCReadpoint - whether to include MVCC or not
1159   * @param includesTag - includesTag or not
1160   * @return Writer for a new StoreFile in the tmp dir.
1161   */
1162  // TODO : allow the Writer factory to create Writers of ShipperListener type only in case of
1163  // compaction
1164  public StoreFileWriter createWriterInTmp(long maxKeyCount, Compression.Algorithm compression,
1165      boolean isCompaction, boolean includeMVCCReadpoint, boolean includesTag,
1166      boolean shouldDropBehind, long totalCompactedFilesSize, String fileStoragePolicy)
1167        throws IOException {
1168    // creating new cache config for each new writer
1169    final CacheConfig writerCacheConf = new CacheConfig(cacheConf);
1170    if (isCompaction) {
1171      // Don't cache data on write on compactions, unless specifically configured to do so
1172      // Cache only when total file size remains lower than configured threshold
1173      final boolean cacheCompactedBlocksOnWrite =
1174        cacheConf.shouldCacheCompactedBlocksOnWrite();
1175      // if data blocks are to be cached on write
1176      // during compaction, we should forcefully
1177      // cache index and bloom blocks as well
1178      if (cacheCompactedBlocksOnWrite && totalCompactedFilesSize <= cacheConf
1179        .getCacheCompactedBlocksOnWriteThreshold()) {
1180        writerCacheConf.enableCacheOnWrite();
1181        if (!cacheOnWriteLogged) {
1182          LOG.info("For {} , cacheCompactedBlocksOnWrite is true, hence enabled " +
1183              "cacheOnWrite for Data blocks, Index blocks and Bloom filter blocks", this);
1184          cacheOnWriteLogged = true;
1185        }
1186      } else {
1187        writerCacheConf.setCacheDataOnWrite(false);
1188        if (totalCompactedFilesSize > cacheConf.getCacheCompactedBlocksOnWriteThreshold()) {
1189          // checking condition once again for logging
1190          LOG.debug(
1191            "For {}, setting cacheCompactedBlocksOnWrite as false as total size of compacted "
1192              + "files - {}, is greater than cacheCompactedBlocksOnWriteThreshold - {}",
1193            this, totalCompactedFilesSize,
1194            cacheConf.getCacheCompactedBlocksOnWriteThreshold());
1195        }
1196      }
1197    } else {
1198      final boolean shouldCacheDataOnWrite = cacheConf.shouldCacheDataOnWrite();
1199      if (shouldCacheDataOnWrite) {
1200        writerCacheConf.enableCacheOnWrite();
1201        if (!cacheOnWriteLogged) {
1202          LOG.info("For {} , cacheDataOnWrite is true, hence enabled cacheOnWrite for " +
1203            "Index blocks and Bloom filter blocks", this);
1204          cacheOnWriteLogged = true;
1205        }
1206      }
1207    }
1208    InetSocketAddress[] favoredNodes = null;
1209    if (region.getRegionServerServices() != null) {
1210      favoredNodes = region.getRegionServerServices().getFavoredNodesForRegion(
1211          region.getRegionInfo().getEncodedName());
1212    }
1213    HFileContext hFileContext = createFileContext(compression, includeMVCCReadpoint, includesTag,
1214      cryptoContext);
1215    Path familyTempDir = new Path(fs.getTempDir(), family.getNameAsString());
1216    StoreFileWriter.Builder builder = new StoreFileWriter.Builder(conf, writerCacheConf,
1217        this.getFileSystem())
1218            .withOutputDir(familyTempDir)
1219            .withBloomType(family.getBloomFilterType())
1220            .withMaxKeyCount(maxKeyCount)
1221            .withFavoredNodes(favoredNodes)
1222            .withFileContext(hFileContext)
1223            .withShouldDropCacheBehind(shouldDropBehind)
1224            .withCompactedFilesSupplier(this::getCompactedFiles)
1225            .withFileStoragePolicy(fileStoragePolicy);
1226    return builder.build();
1227  }
1228
1229  private HFileContext createFileContext(Compression.Algorithm compression,
1230      boolean includeMVCCReadpoint, boolean includesTag, Encryption.Context cryptoContext) {
1231    if (compression == null) {
1232      compression = HFile.DEFAULT_COMPRESSION_ALGORITHM;
1233    }
1234    HFileContext hFileContext = new HFileContextBuilder()
1235                                .withIncludesMvcc(includeMVCCReadpoint)
1236                                .withIncludesTags(includesTag)
1237                                .withCompression(compression)
1238                                .withCompressTags(family.isCompressTags())
1239                                .withChecksumType(checksumType)
1240                                .withBytesPerCheckSum(bytesPerChecksum)
1241                                .withBlockSize(blocksize)
1242                                .withHBaseCheckSum(true)
1243                                .withDataBlockEncoding(family.getDataBlockEncoding())
1244                                .withEncryptionContext(cryptoContext)
1245                                .withCreateTime(EnvironmentEdgeManager.currentTime())
1246                                .withColumnFamily(family.getName())
1247                                .withTableName(region.getTableDescriptor()
1248                                    .getTableName().getName())
1249                                .withCellComparator(this.comparator)
1250                                .build();
1251    return hFileContext;
1252  }
1253
1254
1255  private long getTotalSize(Collection<HStoreFile> sfs) {
1256    return sfs.stream().mapToLong(sf -> sf.getReader().length()).sum();
1257  }
1258
1259  /**
1260   * Change storeFiles adding into place the Reader produced by this new flush.
1261   * @param sfs Store files
1262   * @return Whether compaction is required.
1263   */
1264  private boolean updateStorefiles(List<HStoreFile> sfs, long snapshotId) throws IOException {
1265    this.lock.writeLock().lock();
1266    try {
1267      this.storeEngine.getStoreFileManager().insertNewFiles(sfs);
1268      if (snapshotId > 0) {
1269        this.memstore.clearSnapshot(snapshotId);
1270      }
1271    } finally {
1272      // We need the lock, as long as we are updating the storeFiles
1273      // or changing the memstore. Let us release it before calling
1274      // notifyChangeReadersObservers. See HBASE-4485 for a possible
1275      // deadlock scenario that could have happened if continue to hold
1276      // the lock.
1277      this.lock.writeLock().unlock();
1278    }
1279    // notify to be called here - only in case of flushes
1280    notifyChangedReadersObservers(sfs);
1281    if (LOG.isTraceEnabled()) {
1282      long totalSize = getTotalSize(sfs);
1283      String traceMessage = "FLUSH time,count,size,store size,store files ["
1284          + EnvironmentEdgeManager.currentTime() + "," + sfs.size() + "," + totalSize
1285          + "," + storeSize + "," + storeEngine.getStoreFileManager().getStorefileCount() + "]";
1286      LOG.trace(traceMessage);
1287    }
1288    return needsCompaction();
1289  }
1290
1291  /**
1292   * Notify all observers that set of Readers has changed.
1293   */
1294  private void notifyChangedReadersObservers(List<HStoreFile> sfs) throws IOException {
1295    for (ChangedReadersObserver o : this.changedReaderObservers) {
1296      List<KeyValueScanner> memStoreScanners;
1297      this.lock.readLock().lock();
1298      try {
1299        memStoreScanners = this.memstore.getScanners(o.getReadPoint());
1300      } finally {
1301        this.lock.readLock().unlock();
1302      }
1303      o.updateReaders(sfs, memStoreScanners);
1304    }
1305  }
1306
1307  /**
1308   * Get all scanners with no filtering based on TTL (that happens further down the line).
1309   * @param cacheBlocks cache the blocks or not
1310   * @param usePread true to use pread, false if not
1311   * @param isCompaction true if the scanner is created for compaction
1312   * @param matcher the scan query matcher
1313   * @param startRow the start row
1314   * @param stopRow the stop row
1315   * @param readPt the read point of the current scan
1316   * @return all scanners for this store
1317   */
1318  public List<KeyValueScanner> getScanners(boolean cacheBlocks, boolean isGet, boolean usePread,
1319      boolean isCompaction, ScanQueryMatcher matcher, byte[] startRow, byte[] stopRow, long readPt)
1320      throws IOException {
1321    return getScanners(cacheBlocks, usePread, isCompaction, matcher, startRow, true, stopRow, false,
1322      readPt);
1323  }
1324
1325  /**
1326   * Get all scanners with no filtering based on TTL (that happens further down the line).
1327   * @param cacheBlocks cache the blocks or not
1328   * @param usePread true to use pread, false if not
1329   * @param isCompaction true if the scanner is created for compaction
1330   * @param matcher the scan query matcher
1331   * @param startRow the start row
1332   * @param includeStartRow true to include start row, false if not
1333   * @param stopRow the stop row
1334   * @param includeStopRow true to include stop row, false if not
1335   * @param readPt the read point of the current scan
1336   * @return all scanners for this store
1337   */
1338  public List<KeyValueScanner> getScanners(boolean cacheBlocks, boolean usePread,
1339      boolean isCompaction, ScanQueryMatcher matcher, byte[] startRow, boolean includeStartRow,
1340      byte[] stopRow, boolean includeStopRow, long readPt) throws IOException {
1341    Collection<HStoreFile> storeFilesToScan;
1342    List<KeyValueScanner> memStoreScanners;
1343    this.lock.readLock().lock();
1344    try {
1345      storeFilesToScan = this.storeEngine.getStoreFileManager().getFilesForScan(startRow,
1346        includeStartRow, stopRow, includeStopRow);
1347      memStoreScanners = this.memstore.getScanners(readPt);
1348    } finally {
1349      this.lock.readLock().unlock();
1350    }
1351
1352    try {
1353      // First the store file scanners
1354
1355      // TODO this used to get the store files in descending order,
1356      // but now we get them in ascending order, which I think is
1357      // actually more correct, since memstore get put at the end.
1358      List<StoreFileScanner> sfScanners = StoreFileScanner
1359        .getScannersForStoreFiles(storeFilesToScan, cacheBlocks, usePread, isCompaction, false,
1360          matcher, readPt);
1361      List<KeyValueScanner> scanners = new ArrayList<>(sfScanners.size() + 1);
1362      scanners.addAll(sfScanners);
1363      // Then the memstore scanners
1364      scanners.addAll(memStoreScanners);
1365      return scanners;
1366    } catch (Throwable t) {
1367      clearAndClose(memStoreScanners);
1368      throw t instanceof IOException ? (IOException) t : new IOException(t);
1369    }
1370  }
1371
1372  private static void clearAndClose(List<KeyValueScanner> scanners) {
1373    if (scanners == null) {
1374      return;
1375    }
1376    for (KeyValueScanner s : scanners) {
1377      s.close();
1378    }
1379    scanners.clear();
1380  }
1381
1382  /**
1383   * Create scanners on the given files and if needed on the memstore with no filtering based on TTL
1384   * (that happens further down the line).
1385   * @param files the list of files on which the scanners has to be created
1386   * @param cacheBlocks cache the blocks or not
1387   * @param usePread true to use pread, false if not
1388   * @param isCompaction true if the scanner is created for compaction
1389   * @param matcher the scan query matcher
1390   * @param startRow the start row
1391   * @param stopRow the stop row
1392   * @param readPt the read point of the current scan
1393   * @param includeMemstoreScanner true if memstore has to be included
1394   * @return scanners on the given files and on the memstore if specified
1395   */
1396  public List<KeyValueScanner> getScanners(List<HStoreFile> files, boolean cacheBlocks,
1397      boolean isGet, boolean usePread, boolean isCompaction, ScanQueryMatcher matcher,
1398      byte[] startRow, byte[] stopRow, long readPt, boolean includeMemstoreScanner)
1399      throws IOException {
1400    return getScanners(files, cacheBlocks, usePread, isCompaction, matcher, startRow, true, stopRow,
1401      false, readPt, includeMemstoreScanner);
1402  }
1403
1404  /**
1405   * Create scanners on the given files and if needed on the memstore with no filtering based on TTL
1406   * (that happens further down the line).
1407   * @param files the list of files on which the scanners has to be created
1408   * @param cacheBlocks ache the blocks or not
1409   * @param usePread true to use pread, false if not
1410   * @param isCompaction true if the scanner is created for compaction
1411   * @param matcher the scan query matcher
1412   * @param startRow the start row
1413   * @param includeStartRow true to include start row, false if not
1414   * @param stopRow the stop row
1415   * @param includeStopRow true to include stop row, false if not
1416   * @param readPt the read point of the current scan
1417   * @param includeMemstoreScanner true if memstore has to be included
1418   * @return scanners on the given files and on the memstore if specified
1419   */
1420  public List<KeyValueScanner> getScanners(List<HStoreFile> files, boolean cacheBlocks,
1421      boolean usePread, boolean isCompaction, ScanQueryMatcher matcher, byte[] startRow,
1422      boolean includeStartRow, byte[] stopRow, boolean includeStopRow, long readPt,
1423      boolean includeMemstoreScanner) throws IOException {
1424    List<KeyValueScanner> memStoreScanners = null;
1425    if (includeMemstoreScanner) {
1426      this.lock.readLock().lock();
1427      try {
1428        memStoreScanners = this.memstore.getScanners(readPt);
1429      } finally {
1430        this.lock.readLock().unlock();
1431      }
1432    }
1433    try {
1434      List<StoreFileScanner> sfScanners = StoreFileScanner
1435        .getScannersForStoreFiles(files, cacheBlocks, usePread, isCompaction, false, matcher,
1436          readPt);
1437      List<KeyValueScanner> scanners = new ArrayList<>(sfScanners.size() + 1);
1438      scanners.addAll(sfScanners);
1439      // Then the memstore scanners
1440      if (memStoreScanners != null) {
1441        scanners.addAll(memStoreScanners);
1442      }
1443      return scanners;
1444    } catch (Throwable t) {
1445      clearAndClose(memStoreScanners);
1446      throw t instanceof IOException ? (IOException) t : new IOException(t);
1447    }
1448  }
1449
1450  /**
1451   * @param o Observer who wants to know about changes in set of Readers
1452   */
1453  public void addChangedReaderObserver(ChangedReadersObserver o) {
1454    this.changedReaderObservers.add(o);
1455  }
1456
1457  /**
1458   * @param o Observer no longer interested in changes in set of Readers.
1459   */
1460  public void deleteChangedReaderObserver(ChangedReadersObserver o) {
1461    // We don't check if observer present; it may not be (legitimately)
1462    this.changedReaderObservers.remove(o);
1463  }
1464
1465  //////////////////////////////////////////////////////////////////////////////
1466  // Compaction
1467  //////////////////////////////////////////////////////////////////////////////
1468
1469  /**
1470   * Compact the StoreFiles.  This method may take some time, so the calling
1471   * thread must be able to block for long periods.
1472   *
1473   * <p>During this time, the Store can work as usual, getting values from
1474   * StoreFiles and writing new StoreFiles from the memstore.
1475   *
1476   * Existing StoreFiles are not destroyed until the new compacted StoreFile is
1477   * completely written-out to disk.
1478   *
1479   * <p>The compactLock prevents multiple simultaneous compactions.
1480   * The structureLock prevents us from interfering with other write operations.
1481   *
1482   * <p>We don't want to hold the structureLock for the whole time, as a compact()
1483   * can be lengthy and we want to allow cache-flushes during this period.
1484   *
1485   * <p> Compaction event should be idempotent, since there is no IO Fencing for
1486   * the region directory in hdfs. A region server might still try to complete the
1487   * compaction after it lost the region. That is why the following events are carefully
1488   * ordered for a compaction:
1489   *  1. Compaction writes new files under region/.tmp directory (compaction output)
1490   *  2. Compaction atomically moves the temporary file under region directory
1491   *  3. Compaction appends a WAL edit containing the compaction input and output files.
1492   *  Forces sync on WAL.
1493   *  4. Compaction deletes the input files from the region directory.
1494   *
1495   * Failure conditions are handled like this:
1496   *  - If RS fails before 2, compaction wont complete. Even if RS lives on and finishes
1497   *  the compaction later, it will only write the new data file to the region directory.
1498   *  Since we already have this data, this will be idempotent but we will have a redundant
1499   *  copy of the data.
1500   *  - If RS fails between 2 and 3, the region will have a redundant copy of the data. The
1501   *  RS that failed won't be able to finish sync() for WAL because of lease recovery in WAL.
1502   *  - If RS fails after 3, the region region server who opens the region will pick up the
1503   *  the compaction marker from the WAL and replay it by removing the compaction input files.
1504   *  Failed RS can also attempt to delete those files, but the operation will be idempotent
1505   *
1506   * See HBASE-2231 for details.
1507   *
1508   * @param compaction compaction details obtained from requestCompaction()
1509   * @return Storefile we compacted into or null if we failed or opted out early.
1510   */
1511  public List<HStoreFile> compact(CompactionContext compaction,
1512    ThroughputController throughputController, User user) throws IOException {
1513    assert compaction != null;
1514    CompactionRequestImpl cr = compaction.getRequest();
1515    try {
1516      // Do all sanity checking in here if we have a valid CompactionRequestImpl
1517      // because we need to clean up after it on the way out in a finally
1518      // block below
1519      long compactionStartTime = EnvironmentEdgeManager.currentTime();
1520      assert compaction.hasSelection();
1521      Collection<HStoreFile> filesToCompact = cr.getFiles();
1522      assert !filesToCompact.isEmpty();
1523      synchronized (filesCompacting) {
1524        // sanity check: we're compacting files that this store knows about
1525        // TODO: change this to LOG.error() after more debugging
1526        Preconditions.checkArgument(filesCompacting.containsAll(filesToCompact));
1527      }
1528
1529      // Ready to go. Have list of files to compact.
1530      LOG.info("Starting compaction of " + filesToCompact +
1531        " into tmpdir=" + fs.getTempDir() + ", totalSize=" +
1532          TraditionalBinaryPrefix.long2String(cr.getSize(), "", 1));
1533
1534      return doCompaction(cr, filesToCompact, user, compactionStartTime,
1535          compaction.compact(throughputController, user));
1536    } finally {
1537      finishCompactionRequest(cr);
1538    }
1539  }
1540
1541  @VisibleForTesting
1542  protected List<HStoreFile> doCompaction(CompactionRequestImpl cr,
1543      Collection<HStoreFile> filesToCompact, User user, long compactionStartTime,
1544      List<Path> newFiles) throws IOException {
1545    // Do the steps necessary to complete the compaction.
1546    setStoragePolicyFromFileName(newFiles);
1547    List<HStoreFile> sfs = moveCompactedFilesIntoPlace(cr, newFiles, user);
1548    writeCompactionWalRecord(filesToCompact, sfs);
1549    replaceStoreFiles(filesToCompact, sfs);
1550    if (cr.isMajor()) {
1551      majorCompactedCellsCount.addAndGet(getCompactionProgress().getTotalCompactingKVs());
1552      majorCompactedCellsSize.addAndGet(getCompactionProgress().totalCompactedSize);
1553    } else {
1554      compactedCellsCount.addAndGet(getCompactionProgress().getTotalCompactingKVs());
1555      compactedCellsSize.addAndGet(getCompactionProgress().totalCompactedSize);
1556    }
1557    long outputBytes = getTotalSize(sfs);
1558
1559    // At this point the store will use new files for all new scanners.
1560    completeCompaction(filesToCompact); // update store size.
1561
1562    long now = EnvironmentEdgeManager.currentTime();
1563    if (region.getRegionServerServices() != null
1564        && region.getRegionServerServices().getMetrics() != null) {
1565      region.getRegionServerServices().getMetrics().updateCompaction(
1566          region.getTableDescriptor().getTableName().getNameAsString(),
1567          cr.isMajor(), now - compactionStartTime, cr.getFiles().size(),
1568          newFiles.size(), cr.getSize(), outputBytes);
1569
1570    }
1571
1572    logCompactionEndMessage(cr, sfs, now, compactionStartTime);
1573    return sfs;
1574  }
1575
1576  // Set correct storage policy from the file name of DTCP.
1577  // Rename file will not change the storage policy.
1578  private void setStoragePolicyFromFileName(List<Path> newFiles) throws IOException {
1579    String prefix = HConstants.STORAGE_POLICY_PREFIX;
1580    for (Path newFile : newFiles) {
1581      if (newFile.getParent().getName().startsWith(prefix)) {
1582        CommonFSUtils.setStoragePolicy(fs.getFileSystem(), newFile,
1583            newFile.getParent().getName().substring(prefix.length()));
1584      }
1585    }
1586  }
1587
1588  private List<HStoreFile> moveCompactedFilesIntoPlace(CompactionRequestImpl cr,
1589      List<Path> newFiles, User user) throws IOException {
1590    List<HStoreFile> sfs = new ArrayList<>(newFiles.size());
1591    for (Path newFile : newFiles) {
1592      assert newFile != null;
1593      HStoreFile sf = moveFileIntoPlace(newFile);
1594      if (this.getCoprocessorHost() != null) {
1595        getCoprocessorHost().postCompact(this, sf, cr.getTracker(), cr, user);
1596      }
1597      assert sf != null;
1598      sfs.add(sf);
1599    }
1600    return sfs;
1601  }
1602
1603  // Package-visible for tests
1604  HStoreFile moveFileIntoPlace(Path newFile) throws IOException {
1605    validateStoreFile(newFile);
1606    // Move the file into the right spot
1607    Path destPath = fs.commitStoreFile(getColumnFamilyName(), newFile);
1608    return createStoreFileAndReader(destPath);
1609  }
1610
1611  /**
1612   * Writes the compaction WAL record.
1613   * @param filesCompacted Files compacted (input).
1614   * @param newFiles Files from compaction.
1615   */
1616  private void writeCompactionWalRecord(Collection<HStoreFile> filesCompacted,
1617      Collection<HStoreFile> newFiles) throws IOException {
1618    if (region.getWAL() == null) {
1619      return;
1620    }
1621    List<Path> inputPaths =
1622        filesCompacted.stream().map(HStoreFile::getPath).collect(Collectors.toList());
1623    List<Path> outputPaths =
1624        newFiles.stream().map(HStoreFile::getPath).collect(Collectors.toList());
1625    RegionInfo info = this.region.getRegionInfo();
1626    CompactionDescriptor compactionDescriptor = ProtobufUtil.toCompactionDescriptor(info,
1627        family.getName(), inputPaths, outputPaths,
1628      fs.getStoreDir(getColumnFamilyDescriptor().getNameAsString()));
1629    // Fix reaching into Region to get the maxWaitForSeqId.
1630    // Does this method belong in Region altogether given it is making so many references up there?
1631    // Could be Region#writeCompactionMarker(compactionDescriptor);
1632    WALUtil.writeCompactionMarker(this.region.getWAL(), this.region.getReplicationScope(),
1633        this.region.getRegionInfo(), compactionDescriptor, this.region.getMVCC());
1634  }
1635
1636  @VisibleForTesting
1637  void replaceStoreFiles(Collection<HStoreFile> compactedFiles, Collection<HStoreFile> result)
1638      throws IOException {
1639    this.lock.writeLock().lock();
1640    try {
1641      this.storeEngine.getStoreFileManager().addCompactionResults(compactedFiles, result);
1642      synchronized (filesCompacting) {
1643        filesCompacting.removeAll(compactedFiles);
1644      }
1645
1646      // These may be null when the RS is shutting down. The space quota Chores will fix the Region
1647      // sizes later so it's not super-critical if we miss these.
1648      RegionServerServices rsServices = region.getRegionServerServices();
1649      if (rsServices != null && rsServices.getRegionServerSpaceQuotaManager() != null) {
1650        updateSpaceQuotaAfterFileReplacement(
1651            rsServices.getRegionServerSpaceQuotaManager().getRegionSizeStore(), getRegionInfo(),
1652            compactedFiles, result);
1653      }
1654    } finally {
1655      this.lock.writeLock().unlock();
1656    }
1657  }
1658
1659  /**
1660   * Updates the space quota usage for this region, removing the size for files compacted away
1661   * and adding in the size for new files.
1662   *
1663   * @param sizeStore The object tracking changes in region size for space quotas.
1664   * @param regionInfo The identifier for the region whose size is being updated.
1665   * @param oldFiles Files removed from this store's region.
1666   * @param newFiles Files added to this store's region.
1667   */
1668  void updateSpaceQuotaAfterFileReplacement(
1669      RegionSizeStore sizeStore, RegionInfo regionInfo, Collection<HStoreFile> oldFiles,
1670      Collection<HStoreFile> newFiles) {
1671    long delta = 0;
1672    if (oldFiles != null) {
1673      for (HStoreFile compactedFile : oldFiles) {
1674        if (compactedFile.isHFile()) {
1675          delta -= compactedFile.getReader().length();
1676        }
1677      }
1678    }
1679    if (newFiles != null) {
1680      for (HStoreFile newFile : newFiles) {
1681        if (newFile.isHFile()) {
1682          delta += newFile.getReader().length();
1683        }
1684      }
1685    }
1686    sizeStore.incrementRegionSize(regionInfo, delta);
1687  }
1688
1689  /**
1690   * Log a very elaborate compaction completion message.
1691   * @param cr Request.
1692   * @param sfs Resulting files.
1693   * @param compactionStartTime Start time.
1694   */
1695  private void logCompactionEndMessage(
1696      CompactionRequestImpl cr, List<HStoreFile> sfs, long now, long compactionStartTime) {
1697    StringBuilder message = new StringBuilder(
1698      "Completed" + (cr.isMajor() ? " major" : "") + " compaction of "
1699      + cr.getFiles().size() + (cr.isAllFiles() ? " (all)" : "") + " file(s) in "
1700      + this + " of " + this.getRegionInfo().getShortNameToLog() + " into ");
1701    if (sfs.isEmpty()) {
1702      message.append("none, ");
1703    } else {
1704      for (HStoreFile sf: sfs) {
1705        message.append(sf.getPath().getName());
1706        message.append("(size=");
1707        message.append(TraditionalBinaryPrefix.long2String(sf.getReader().length(), "", 1));
1708        message.append("), ");
1709      }
1710    }
1711    message.append("total size for store is ")
1712      .append(StringUtils.TraditionalBinaryPrefix.long2String(storeSize.get(), "", 1))
1713      .append(". This selection was in queue for ")
1714      .append(StringUtils.formatTimeDiff(compactionStartTime, cr.getSelectionTime()))
1715      .append(", and took ").append(StringUtils.formatTimeDiff(now, compactionStartTime))
1716      .append(" to execute.");
1717    LOG.info(message.toString());
1718    if (LOG.isTraceEnabled()) {
1719      int fileCount = storeEngine.getStoreFileManager().getStorefileCount();
1720      long resultSize = getTotalSize(sfs);
1721      String traceMessage = "COMPACTION start,end,size out,files in,files out,store size,"
1722        + "store files [" + compactionStartTime + "," + now + "," + resultSize + ","
1723          + cr.getFiles().size() + "," + sfs.size() + "," +  storeSize + "," + fileCount + "]";
1724      LOG.trace(traceMessage);
1725    }
1726  }
1727
1728  /**
1729   * Call to complete a compaction. Its for the case where we find in the WAL a compaction
1730   * that was not finished.  We could find one recovering a WAL after a regionserver crash.
1731   * See HBASE-2231.
1732   */
1733  public void replayCompactionMarker(CompactionDescriptor compaction, boolean pickCompactionFiles,
1734      boolean removeFiles) throws IOException {
1735    LOG.debug("Completing compaction from the WAL marker");
1736    List<String> compactionInputs = compaction.getCompactionInputList();
1737    List<String> compactionOutputs = Lists.newArrayList(compaction.getCompactionOutputList());
1738
1739    // The Compaction Marker is written after the compaction is completed,
1740    // and the files moved into the region/family folder.
1741    //
1742    // If we crash after the entry is written, we may not have removed the
1743    // input files, but the output file is present.
1744    // (The unremoved input files will be removed by this function)
1745    //
1746    // If we scan the directory and the file is not present, it can mean that:
1747    //   - The file was manually removed by the user
1748    //   - The file was removed as consequence of subsequent compaction
1749    // so, we can't do anything with the "compaction output list" because those
1750    // files have already been loaded when opening the region (by virtue of
1751    // being in the store's folder) or they may be missing due to a compaction.
1752
1753    String familyName = this.getColumnFamilyName();
1754    Set<String> inputFiles = new HashSet<>();
1755    for (String compactionInput : compactionInputs) {
1756      Path inputPath = fs.getStoreFilePath(familyName, compactionInput);
1757      inputFiles.add(inputPath.getName());
1758    }
1759
1760    //some of the input files might already be deleted
1761    List<HStoreFile> inputStoreFiles = new ArrayList<>(compactionInputs.size());
1762    for (HStoreFile sf : this.getStorefiles()) {
1763      if (inputFiles.contains(sf.getPath().getName())) {
1764        inputStoreFiles.add(sf);
1765      }
1766    }
1767
1768    // check whether we need to pick up the new files
1769    List<HStoreFile> outputStoreFiles = new ArrayList<>(compactionOutputs.size());
1770
1771    if (pickCompactionFiles) {
1772      for (HStoreFile sf : this.getStorefiles()) {
1773        compactionOutputs.remove(sf.getPath().getName());
1774      }
1775      for (String compactionOutput : compactionOutputs) {
1776        StoreFileInfo storeFileInfo = fs.getStoreFileInfo(getColumnFamilyName(), compactionOutput);
1777        HStoreFile storeFile = createStoreFileAndReader(storeFileInfo);
1778        outputStoreFiles.add(storeFile);
1779      }
1780    }
1781
1782    if (!inputStoreFiles.isEmpty() || !outputStoreFiles.isEmpty()) {
1783      LOG.info("Replaying compaction marker, replacing input files: " +
1784          inputStoreFiles + " with output files : " + outputStoreFiles);
1785      this.replaceStoreFiles(inputStoreFiles, outputStoreFiles);
1786      this.completeCompaction(inputStoreFiles);
1787    }
1788  }
1789
1790  /**
1791   * This method tries to compact N recent files for testing.
1792   * Note that because compacting "recent" files only makes sense for some policies,
1793   * e.g. the default one, it assumes default policy is used. It doesn't use policy,
1794   * but instead makes a compaction candidate list by itself.
1795   * @param N Number of files.
1796   */
1797  @VisibleForTesting
1798  public void compactRecentForTestingAssumingDefaultPolicy(int N) throws IOException {
1799    List<HStoreFile> filesToCompact;
1800    boolean isMajor;
1801
1802    this.lock.readLock().lock();
1803    try {
1804      synchronized (filesCompacting) {
1805        filesToCompact = Lists.newArrayList(storeEngine.getStoreFileManager().getStorefiles());
1806        if (!filesCompacting.isEmpty()) {
1807          // exclude all files older than the newest file we're currently
1808          // compacting. this allows us to preserve contiguity (HBASE-2856)
1809          HStoreFile last = filesCompacting.get(filesCompacting.size() - 1);
1810          int idx = filesToCompact.indexOf(last);
1811          Preconditions.checkArgument(idx != -1);
1812          filesToCompact.subList(0, idx + 1).clear();
1813        }
1814        int count = filesToCompact.size();
1815        if (N > count) {
1816          throw new RuntimeException("Not enough files");
1817        }
1818
1819        filesToCompact = filesToCompact.subList(count - N, count);
1820        isMajor = (filesToCompact.size() == storeEngine.getStoreFileManager().getStorefileCount());
1821        filesCompacting.addAll(filesToCompact);
1822        Collections.sort(filesCompacting, storeEngine.getStoreFileManager()
1823            .getStoreFileComparator());
1824      }
1825    } finally {
1826      this.lock.readLock().unlock();
1827    }
1828
1829    try {
1830      // Ready to go. Have list of files to compact.
1831      List<Path> newFiles = ((DefaultCompactor)this.storeEngine.getCompactor())
1832          .compactForTesting(filesToCompact, isMajor);
1833      for (Path newFile: newFiles) {
1834        // Move the compaction into place.
1835        HStoreFile sf = moveFileIntoPlace(newFile);
1836        if (this.getCoprocessorHost() != null) {
1837          this.getCoprocessorHost().postCompact(this, sf, null, null, null);
1838        }
1839        replaceStoreFiles(filesToCompact, Collections.singletonList(sf));
1840        completeCompaction(filesToCompact);
1841      }
1842    } finally {
1843      synchronized (filesCompacting) {
1844        filesCompacting.removeAll(filesToCompact);
1845      }
1846    }
1847  }
1848
1849  @Override
1850  public boolean hasReferences() {
1851    // Grab the read lock here, because we need to ensure that: only when the atomic
1852    // replaceStoreFiles(..) finished, we can get all the complete store file list.
1853    this.lock.readLock().lock();
1854    try {
1855      // Merge the current store files with compacted files here due to HBASE-20940.
1856      Collection<HStoreFile> allStoreFiles = new ArrayList<>(getStorefiles());
1857      allStoreFiles.addAll(getCompactedFiles());
1858      return StoreUtils.hasReferences(allStoreFiles);
1859    } finally {
1860      this.lock.readLock().unlock();
1861    }
1862  }
1863
1864  /**
1865   * getter for CompactionProgress object
1866   * @return CompactionProgress object; can be null
1867   */
1868  public CompactionProgress getCompactionProgress() {
1869    return this.storeEngine.getCompactor().getProgress();
1870  }
1871
1872  @Override
1873  public boolean shouldPerformMajorCompaction() throws IOException {
1874    for (HStoreFile sf : this.storeEngine.getStoreFileManager().getStorefiles()) {
1875      // TODO: what are these reader checks all over the place?
1876      if (sf.getReader() == null) {
1877        LOG.debug("StoreFile {} has null Reader", sf);
1878        return false;
1879      }
1880    }
1881    return storeEngine.getCompactionPolicy().shouldPerformMajorCompaction(
1882        this.storeEngine.getStoreFileManager().getStorefiles());
1883  }
1884
1885  public Optional<CompactionContext> requestCompaction() throws IOException {
1886    return requestCompaction(NO_PRIORITY, CompactionLifeCycleTracker.DUMMY, null);
1887  }
1888
1889  public Optional<CompactionContext> requestCompaction(int priority,
1890      CompactionLifeCycleTracker tracker, User user) throws IOException {
1891    // don't even select for compaction if writes are disabled
1892    if (!this.areWritesEnabled()) {
1893      return Optional.empty();
1894    }
1895    // Before we do compaction, try to get rid of unneeded files to simplify things.
1896    removeUnneededFiles();
1897
1898    final CompactionContext compaction = storeEngine.createCompaction();
1899    CompactionRequestImpl request = null;
1900    this.lock.readLock().lock();
1901    try {
1902      synchronized (filesCompacting) {
1903        // First, see if coprocessor would want to override selection.
1904        if (this.getCoprocessorHost() != null) {
1905          final List<HStoreFile> candidatesForCoproc = compaction.preSelect(this.filesCompacting);
1906          boolean override = getCoprocessorHost().preCompactSelection(this,
1907              candidatesForCoproc, tracker, user);
1908          if (override) {
1909            // Coprocessor is overriding normal file selection.
1910            compaction.forceSelect(new CompactionRequestImpl(candidatesForCoproc));
1911          }
1912        }
1913
1914        // Normal case - coprocessor is not overriding file selection.
1915        if (!compaction.hasSelection()) {
1916          boolean isUserCompaction = priority == Store.PRIORITY_USER;
1917          boolean mayUseOffPeak = offPeakHours.isOffPeakHour() &&
1918              offPeakCompactionTracker.compareAndSet(false, true);
1919          try {
1920            compaction.select(this.filesCompacting, isUserCompaction,
1921              mayUseOffPeak, forceMajor && filesCompacting.isEmpty());
1922          } catch (IOException e) {
1923            if (mayUseOffPeak) {
1924              offPeakCompactionTracker.set(false);
1925            }
1926            throw e;
1927          }
1928          assert compaction.hasSelection();
1929          if (mayUseOffPeak && !compaction.getRequest().isOffPeak()) {
1930            // Compaction policy doesn't want to take advantage of off-peak.
1931            offPeakCompactionTracker.set(false);
1932          }
1933        }
1934        if (this.getCoprocessorHost() != null) {
1935          this.getCoprocessorHost().postCompactSelection(
1936              this, ImmutableList.copyOf(compaction.getRequest().getFiles()), tracker,
1937              compaction.getRequest(), user);
1938        }
1939        // Finally, we have the resulting files list. Check if we have any files at all.
1940        request = compaction.getRequest();
1941        Collection<HStoreFile> selectedFiles = request.getFiles();
1942        if (selectedFiles.isEmpty()) {
1943          return Optional.empty();
1944        }
1945
1946        addToCompactingFiles(selectedFiles);
1947
1948        // If we're enqueuing a major, clear the force flag.
1949        this.forceMajor = this.forceMajor && !request.isMajor();
1950
1951        // Set common request properties.
1952        // Set priority, either override value supplied by caller or from store.
1953        final int compactionPriority =
1954          (priority != Store.NO_PRIORITY) ? priority : getCompactPriority();
1955        request.setPriority(compactionPriority);
1956
1957        if (request.isAfterSplit()) {
1958          // If the store belongs to recently splitted daughter regions, better we consider
1959          // them with the higher priority in the compaction queue.
1960          // Override priority if it is lower (higher int value) than
1961          // SPLIT_REGION_COMPACTION_PRIORITY
1962          final int splitHousekeepingPriority =
1963            Math.min(compactionPriority, SPLIT_REGION_COMPACTION_PRIORITY);
1964          request.setPriority(splitHousekeepingPriority);
1965          LOG.info("Keeping/Overriding Compaction request priority to {} for CF {} since it"
1966              + " belongs to recently split daughter region {}", splitHousekeepingPriority,
1967            this.getColumnFamilyName(), getRegionInfo().getRegionNameAsString());
1968        }
1969        request.setDescription(getRegionInfo().getRegionNameAsString(), getColumnFamilyName());
1970        request.setTracker(tracker);
1971      }
1972    } finally {
1973      this.lock.readLock().unlock();
1974    }
1975
1976    if (LOG.isDebugEnabled()) {
1977      LOG.debug(this + " is initiating " + (request.isMajor() ? "major" : "minor") + " compaction"
1978          + (request.isAllFiles() ? " (all files)" : ""));
1979    }
1980    this.region.reportCompactionRequestStart(request.isMajor());
1981    return Optional.of(compaction);
1982  }
1983
1984  /** Adds the files to compacting files. filesCompacting must be locked. */
1985  private void addToCompactingFiles(Collection<HStoreFile> filesToAdd) {
1986    if (CollectionUtils.isEmpty(filesToAdd)) {
1987      return;
1988    }
1989    // Check that we do not try to compact the same StoreFile twice.
1990    if (!Collections.disjoint(filesCompacting, filesToAdd)) {
1991      Preconditions.checkArgument(false, "%s overlaps with %s", filesToAdd, filesCompacting);
1992    }
1993    filesCompacting.addAll(filesToAdd);
1994    Collections.sort(filesCompacting, storeEngine.getStoreFileManager().getStoreFileComparator());
1995  }
1996
1997  private void removeUnneededFiles() throws IOException {
1998    if (!conf.getBoolean("hbase.store.delete.expired.storefile", true)) {
1999      return;
2000    }
2001    if (getColumnFamilyDescriptor().getMinVersions() > 0) {
2002      LOG.debug("Skipping expired store file removal due to min version of {} being {}",
2003          this, getColumnFamilyDescriptor().getMinVersions());
2004      return;
2005    }
2006    this.lock.readLock().lock();
2007    Collection<HStoreFile> delSfs = null;
2008    try {
2009      synchronized (filesCompacting) {
2010        long cfTtl = getStoreFileTtl();
2011        if (cfTtl != Long.MAX_VALUE) {
2012          delSfs = storeEngine.getStoreFileManager().getUnneededFiles(
2013              EnvironmentEdgeManager.currentTime() - cfTtl, filesCompacting);
2014          addToCompactingFiles(delSfs);
2015        }
2016      }
2017    } finally {
2018      this.lock.readLock().unlock();
2019    }
2020
2021    if (CollectionUtils.isEmpty(delSfs)) {
2022      return;
2023    }
2024
2025    Collection<HStoreFile> newFiles = Collections.emptyList(); // No new files.
2026    writeCompactionWalRecord(delSfs, newFiles);
2027    replaceStoreFiles(delSfs, newFiles);
2028    completeCompaction(delSfs);
2029    LOG.info("Completed removal of " + delSfs.size() + " unnecessary (expired) file(s) in "
2030        + this + "; total size is "
2031        + TraditionalBinaryPrefix.long2String(storeSize.get(), "", 1));
2032  }
2033
2034  public void cancelRequestedCompaction(CompactionContext compaction) {
2035    finishCompactionRequest(compaction.getRequest());
2036  }
2037
2038  protected void finishCompactionRequest(CompactionRequestImpl cr) {
2039    this.region.reportCompactionRequestEnd(cr.isMajor(), cr.getFiles().size(), cr.getSize());
2040    if (cr.isOffPeak()) {
2041      offPeakCompactionTracker.set(false);
2042      cr.setOffPeak(false);
2043    }
2044    synchronized (filesCompacting) {
2045      filesCompacting.removeAll(cr.getFiles());
2046    }
2047  }
2048
2049  /**
2050   * Validates a store file by opening and closing it. In HFileV2 this should not be an expensive
2051   * operation.
2052   * @param path the path to the store file
2053   */
2054  private void validateStoreFile(Path path) throws IOException {
2055    HStoreFile storeFile = null;
2056    try {
2057      storeFile = createStoreFileAndReader(path);
2058    } catch (IOException e) {
2059      LOG.error("Failed to open store file : {}, keeping it in tmp location", path, e);
2060      throw e;
2061    } finally {
2062      if (storeFile != null) {
2063        storeFile.closeStoreFile(false);
2064      }
2065    }
2066  }
2067
2068  /**
2069   * Update counts.
2070   * @param compactedFiles list of files that were compacted
2071   */
2072  @VisibleForTesting
2073  protected void completeCompaction(Collection<HStoreFile> compactedFiles)
2074  // Rename this method! TODO.
2075    throws IOException {
2076    this.storeSize.set(0L);
2077    this.totalUncompressedBytes.set(0L);
2078    for (HStoreFile hsf : this.storeEngine.getStoreFileManager().getStorefiles()) {
2079      StoreFileReader r = hsf.getReader();
2080      if (r == null) {
2081        LOG.warn("StoreFile {} has a null Reader", hsf);
2082        continue;
2083      }
2084      this.storeSize.addAndGet(r.length());
2085      this.totalUncompressedBytes.addAndGet(r.getTotalUncompressedBytes());
2086    }
2087  }
2088
2089  /*
2090   * @param wantedVersions How many versions were asked for.
2091   * @return wantedVersions or this families' {@link HConstants#VERSIONS}.
2092   */
2093  int versionsToReturn(final int wantedVersions) {
2094    if (wantedVersions <= 0) {
2095      throw new IllegalArgumentException("Number of versions must be > 0");
2096    }
2097    // Make sure we do not return more than maximum versions for this store.
2098    int maxVersions = this.family.getMaxVersions();
2099    return wantedVersions > maxVersions ? maxVersions: wantedVersions;
2100  }
2101
2102  @Override
2103  public boolean canSplit() {
2104    // Not split-able if we find a reference store file present in the store.
2105    boolean result = !hasReferences();
2106    if (!result) {
2107      LOG.trace("Not splittable; has references: {}", this);
2108    }
2109    return result;
2110  }
2111
2112  /**
2113   * Determines if Store should be split.
2114   */
2115  public Optional<byte[]> getSplitPoint() {
2116    this.lock.readLock().lock();
2117    try {
2118      // Should already be enforced by the split policy!
2119      assert !this.getRegionInfo().isMetaRegion();
2120      // Not split-able if we find a reference store file present in the store.
2121      if (hasReferences()) {
2122        LOG.trace("Not splittable; has references: {}", this);
2123        return Optional.empty();
2124      }
2125      return this.storeEngine.getStoreFileManager().getSplitPoint();
2126    } catch(IOException e) {
2127      LOG.warn("Failed getting store size for {}", this, e);
2128    } finally {
2129      this.lock.readLock().unlock();
2130    }
2131    return Optional.empty();
2132  }
2133
2134  @Override
2135  public long getLastCompactSize() {
2136    return this.lastCompactSize;
2137  }
2138
2139  @Override
2140  public long getSize() {
2141    return storeSize.get();
2142  }
2143
2144  public void triggerMajorCompaction() {
2145    this.forceMajor = true;
2146  }
2147
2148  //////////////////////////////////////////////////////////////////////////////
2149  // File administration
2150  //////////////////////////////////////////////////////////////////////////////
2151
2152  /**
2153   * Return a scanner for both the memstore and the HStore files. Assumes we are not in a
2154   * compaction.
2155   * @param scan Scan to apply when scanning the stores
2156   * @param targetCols columns to scan
2157   * @return a scanner over the current key values
2158   * @throws IOException on failure
2159   */
2160  public KeyValueScanner getScanner(Scan scan, final NavigableSet<byte[]> targetCols, long readPt)
2161      throws IOException {
2162    lock.readLock().lock();
2163    try {
2164      ScanInfo scanInfo;
2165      if (this.getCoprocessorHost() != null) {
2166        scanInfo = this.getCoprocessorHost().preStoreScannerOpen(this, scan);
2167      } else {
2168        scanInfo = getScanInfo();
2169      }
2170      return createScanner(scan, scanInfo, targetCols, readPt);
2171    } finally {
2172      lock.readLock().unlock();
2173    }
2174  }
2175
2176  // HMobStore will override this method to return its own implementation.
2177  protected KeyValueScanner createScanner(Scan scan, ScanInfo scanInfo,
2178      NavigableSet<byte[]> targetCols, long readPt) throws IOException {
2179    return scan.isReversed() ? new ReversedStoreScanner(this, scanInfo, scan, targetCols, readPt)
2180        : new StoreScanner(this, scanInfo, scan, targetCols, readPt);
2181  }
2182
2183  /**
2184   * Recreates the scanners on the current list of active store file scanners
2185   * @param currentFileScanners the current set of active store file scanners
2186   * @param cacheBlocks cache the blocks or not
2187   * @param usePread use pread or not
2188   * @param isCompaction is the scanner for compaction
2189   * @param matcher the scan query matcher
2190   * @param startRow the scan's start row
2191   * @param includeStartRow should the scan include the start row
2192   * @param stopRow the scan's stop row
2193   * @param includeStopRow should the scan include the stop row
2194   * @param readPt the read point of the current scane
2195   * @param includeMemstoreScanner whether the current scanner should include memstorescanner
2196   * @return list of scanners recreated on the current Scanners
2197   */
2198  public List<KeyValueScanner> recreateScanners(List<KeyValueScanner> currentFileScanners,
2199      boolean cacheBlocks, boolean usePread, boolean isCompaction, ScanQueryMatcher matcher,
2200      byte[] startRow, boolean includeStartRow, byte[] stopRow, boolean includeStopRow, long readPt,
2201      boolean includeMemstoreScanner) throws IOException {
2202    this.lock.readLock().lock();
2203    try {
2204      Map<String, HStoreFile> name2File =
2205          new HashMap<>(getStorefilesCount() + getCompactedFilesCount());
2206      for (HStoreFile file : getStorefiles()) {
2207        name2File.put(file.getFileInfo().getActiveFileName(), file);
2208      }
2209      Collection<HStoreFile> compactedFiles = getCompactedFiles();
2210      for (HStoreFile file : IterableUtils.emptyIfNull(compactedFiles)) {
2211        name2File.put(file.getFileInfo().getActiveFileName(), file);
2212      }
2213      List<HStoreFile> filesToReopen = new ArrayList<>();
2214      for (KeyValueScanner kvs : currentFileScanners) {
2215        assert kvs.isFileScanner();
2216        if (kvs.peek() == null) {
2217          continue;
2218        }
2219        filesToReopen.add(name2File.get(kvs.getFilePath().getName()));
2220      }
2221      if (filesToReopen.isEmpty()) {
2222        return null;
2223      }
2224      return getScanners(filesToReopen, cacheBlocks, false, false, matcher, startRow,
2225        includeStartRow, stopRow, includeStopRow, readPt, false);
2226    } finally {
2227      this.lock.readLock().unlock();
2228    }
2229  }
2230
2231  @Override
2232  public String toString() {
2233    return this.getRegionInfo().getShortNameToLog()+ "/" + this.getColumnFamilyName();
2234  }
2235
2236  @Override
2237  public int getStorefilesCount() {
2238    return this.storeEngine.getStoreFileManager().getStorefileCount();
2239  }
2240
2241  @Override
2242  public int getCompactedFilesCount() {
2243    return this.storeEngine.getStoreFileManager().getCompactedFilesCount();
2244  }
2245
2246  private LongStream getStoreFileAgeStream() {
2247    return this.storeEngine.getStoreFileManager().getStorefiles().stream().filter(sf -> {
2248      if (sf.getReader() == null) {
2249        LOG.warn("StoreFile {} has a null Reader", sf);
2250        return false;
2251      } else {
2252        return true;
2253      }
2254    }).filter(HStoreFile::isHFile).mapToLong(sf -> sf.getFileInfo().getCreatedTimestamp())
2255        .map(t -> EnvironmentEdgeManager.currentTime() - t);
2256  }
2257
2258  @Override
2259  public OptionalLong getMaxStoreFileAge() {
2260    return getStoreFileAgeStream().max();
2261  }
2262
2263  @Override
2264  public OptionalLong getMinStoreFileAge() {
2265    return getStoreFileAgeStream().min();
2266  }
2267
2268  @Override
2269  public OptionalDouble getAvgStoreFileAge() {
2270    return getStoreFileAgeStream().average();
2271  }
2272
2273  @Override
2274  public long getNumReferenceFiles() {
2275    return this.storeEngine.getStoreFileManager().getStorefiles().stream()
2276        .filter(HStoreFile::isReference).count();
2277  }
2278
2279  @Override
2280  public long getNumHFiles() {
2281    return this.storeEngine.getStoreFileManager().getStorefiles().stream()
2282        .filter(HStoreFile::isHFile).count();
2283  }
2284
2285  @Override
2286  public long getStoreSizeUncompressed() {
2287    return this.totalUncompressedBytes.get();
2288  }
2289
2290  @Override
2291  public long getStorefilesSize() {
2292    // Include all StoreFiles
2293    return getStorefilesSize(this.storeEngine.getStoreFileManager().getStorefiles(), sf -> true);
2294  }
2295
2296  @Override
2297  public long getHFilesSize() {
2298    // Include only StoreFiles which are HFiles
2299    return getStorefilesSize(this.storeEngine.getStoreFileManager().getStorefiles(),
2300      HStoreFile::isHFile);
2301  }
2302
2303  private long getTotalUncompressedBytes(List<HStoreFile> files) {
2304    return files.stream()
2305      .mapToLong(file -> getStorefileFieldSize(file, StoreFileReader::getTotalUncompressedBytes))
2306      .sum();
2307  }
2308
2309  private long getStorefilesSize(Collection<HStoreFile> files, Predicate<HStoreFile> predicate) {
2310    return files.stream().filter(predicate)
2311      .mapToLong(file -> getStorefileFieldSize(file, StoreFileReader::length)).sum();
2312  }
2313
2314  private long getStorefileFieldSize(HStoreFile file, ToLongFunction<StoreFileReader> f) {
2315    if (file == null) {
2316      return 0L;
2317    }
2318    StoreFileReader reader = file.getReader();
2319    if (reader == null) {
2320      return 0L;
2321    }
2322    return f.applyAsLong(reader);
2323  }
2324
2325  private long getStorefilesFieldSize(ToLongFunction<StoreFileReader> f) {
2326    return this.storeEngine.getStoreFileManager().getStorefiles().stream()
2327      .mapToLong(file -> getStorefileFieldSize(file, f)).sum();
2328  }
2329
2330  @Override
2331  public long getStorefilesRootLevelIndexSize() {
2332    return getStorefilesFieldSize(StoreFileReader::indexSize);
2333  }
2334
2335  @Override
2336  public long getTotalStaticIndexSize() {
2337    return getStorefilesFieldSize(StoreFileReader::getUncompressedDataIndexSize);
2338  }
2339
2340  @Override
2341  public long getTotalStaticBloomSize() {
2342    return getStorefilesFieldSize(StoreFileReader::getTotalBloomSize);
2343  }
2344
2345  @Override
2346  public MemStoreSize getMemStoreSize() {
2347    return this.memstore.size();
2348  }
2349
2350  @Override
2351  public int getCompactPriority() {
2352    int priority = this.storeEngine.getStoreFileManager().getStoreCompactionPriority();
2353    if (priority == PRIORITY_USER) {
2354      LOG.warn("Compaction priority is USER despite there being no user compaction");
2355    }
2356    return priority;
2357  }
2358
2359  public boolean throttleCompaction(long compactionSize) {
2360    return storeEngine.getCompactionPolicy().throttleCompaction(compactionSize);
2361  }
2362
2363  public HRegion getHRegion() {
2364    return this.region;
2365  }
2366
2367  public RegionCoprocessorHost getCoprocessorHost() {
2368    return this.region.getCoprocessorHost();
2369  }
2370
2371  @Override
2372  public RegionInfo getRegionInfo() {
2373    return this.fs.getRegionInfo();
2374  }
2375
2376  @Override
2377  public boolean areWritesEnabled() {
2378    return this.region.areWritesEnabled();
2379  }
2380
2381  @Override
2382  public long getSmallestReadPoint() {
2383    return this.region.getSmallestReadPoint();
2384  }
2385
2386  /**
2387   * Adds or replaces the specified KeyValues.
2388   * <p>
2389   * For each KeyValue specified, if a cell with the same row, family, and qualifier exists in
2390   * MemStore, it will be replaced. Otherwise, it will just be inserted to MemStore.
2391   * <p>
2392   * This operation is atomic on each KeyValue (row/family/qualifier) but not necessarily atomic
2393   * across all of them.
2394   * @param readpoint readpoint below which we can safely remove duplicate KVs
2395   */
2396  public void upsert(Iterable<Cell> cells, long readpoint, MemStoreSizing memstoreSizing)
2397      throws IOException {
2398    this.lock.readLock().lock();
2399    try {
2400      this.memstore.upsert(cells, readpoint, memstoreSizing);
2401    } finally {
2402      this.lock.readLock().unlock();
2403    }
2404  }
2405
2406  public StoreFlushContext createFlushContext(long cacheFlushId, FlushLifeCycleTracker tracker) {
2407    return new StoreFlusherImpl(cacheFlushId, tracker);
2408  }
2409
2410  private final class StoreFlusherImpl implements StoreFlushContext {
2411
2412    private final FlushLifeCycleTracker tracker;
2413    private final long cacheFlushSeqNum;
2414    private MemStoreSnapshot snapshot;
2415    private List<Path> tempFiles;
2416    private List<Path> committedFiles;
2417    private long cacheFlushCount;
2418    private long cacheFlushSize;
2419    private long outputFileSize;
2420
2421    private StoreFlusherImpl(long cacheFlushSeqNum, FlushLifeCycleTracker tracker) {
2422      this.cacheFlushSeqNum = cacheFlushSeqNum;
2423      this.tracker = tracker;
2424    }
2425
2426    /**
2427     * This is not thread safe. The caller should have a lock on the region or the store.
2428     * If necessary, the lock can be added with the patch provided in HBASE-10087
2429     */
2430    @Override
2431    public MemStoreSize prepare() {
2432      // passing the current sequence number of the wal - to allow bookkeeping in the memstore
2433      this.snapshot = memstore.snapshot();
2434      this.cacheFlushCount = snapshot.getCellsCount();
2435      this.cacheFlushSize = snapshot.getDataSize();
2436      committedFiles = new ArrayList<>(1);
2437      return snapshot.getMemStoreSize();
2438    }
2439
2440    @Override
2441    public void flushCache(MonitoredTask status) throws IOException {
2442      RegionServerServices rsService = region.getRegionServerServices();
2443      ThroughputController throughputController =
2444          rsService == null ? null : rsService.getFlushThroughputController();
2445      tempFiles =
2446          HStore.this.flushCache(cacheFlushSeqNum, snapshot, status, throughputController, tracker);
2447    }
2448
2449    @Override
2450    public boolean commit(MonitoredTask status) throws IOException {
2451      if (CollectionUtils.isEmpty(this.tempFiles)) {
2452        return false;
2453      }
2454      List<HStoreFile> storeFiles = new ArrayList<>(this.tempFiles.size());
2455      for (Path storeFilePath : tempFiles) {
2456        try {
2457          HStoreFile sf = HStore.this.commitFile(storeFilePath, cacheFlushSeqNum, status);
2458          outputFileSize += sf.getReader().length();
2459          storeFiles.add(sf);
2460        } catch (IOException ex) {
2461          LOG.error("Failed to commit store file {}", storeFilePath, ex);
2462          // Try to delete the files we have committed before.
2463          for (HStoreFile sf : storeFiles) {
2464            Path pathToDelete = sf.getPath();
2465            try {
2466              sf.deleteStoreFile();
2467            } catch (IOException deleteEx) {
2468              LOG.error(HBaseMarkers.FATAL, "Failed to delete store file we committed, "
2469                  + "halting {}", pathToDelete, ex);
2470              Runtime.getRuntime().halt(1);
2471            }
2472          }
2473          throw new IOException("Failed to commit the flush", ex);
2474        }
2475      }
2476
2477      for (HStoreFile sf : storeFiles) {
2478        if (HStore.this.getCoprocessorHost() != null) {
2479          HStore.this.getCoprocessorHost().postFlush(HStore.this, sf, tracker);
2480        }
2481        committedFiles.add(sf.getPath());
2482      }
2483
2484      HStore.this.flushedCellsCount.addAndGet(cacheFlushCount);
2485      HStore.this.flushedCellsSize.addAndGet(cacheFlushSize);
2486      HStore.this.flushedOutputFileSize.addAndGet(outputFileSize);
2487
2488      // Add new file to store files.  Clear snapshot too while we have the Store write lock.
2489      return HStore.this.updateStorefiles(storeFiles, snapshot.getId());
2490    }
2491
2492    @Override
2493    public long getOutputFileSize() {
2494      return outputFileSize;
2495    }
2496
2497    @Override
2498    public List<Path> getCommittedFiles() {
2499      return committedFiles;
2500    }
2501
2502    /**
2503     * Similar to commit, but called in secondary region replicas for replaying the
2504     * flush cache from primary region. Adds the new files to the store, and drops the
2505     * snapshot depending on dropMemstoreSnapshot argument.
2506     * @param fileNames names of the flushed files
2507     * @param dropMemstoreSnapshot whether to drop the prepared memstore snapshot
2508     */
2509    @Override
2510    public void replayFlush(List<String> fileNames, boolean dropMemstoreSnapshot)
2511        throws IOException {
2512      List<HStoreFile> storeFiles = new ArrayList<>(fileNames.size());
2513      for (String file : fileNames) {
2514        // open the file as a store file (hfile link, etc)
2515        StoreFileInfo storeFileInfo = fs.getStoreFileInfo(getColumnFamilyName(), file);
2516        HStoreFile storeFile = createStoreFileAndReader(storeFileInfo);
2517        storeFiles.add(storeFile);
2518        HStore.this.storeSize.addAndGet(storeFile.getReader().length());
2519        HStore.this.totalUncompressedBytes
2520            .addAndGet(storeFile.getReader().getTotalUncompressedBytes());
2521        if (LOG.isInfoEnabled()) {
2522          LOG.info(this + " added " + storeFile + ", entries=" + storeFile.getReader().getEntries() +
2523              ", sequenceid=" + storeFile.getReader().getSequenceID() + ", filesize="
2524              + TraditionalBinaryPrefix.long2String(storeFile.getReader().length(), "", 1));
2525        }
2526      }
2527
2528      long snapshotId = -1; // -1 means do not drop
2529      if (dropMemstoreSnapshot && snapshot != null) {
2530        snapshotId = snapshot.getId();
2531        snapshot.close();
2532      }
2533      HStore.this.updateStorefiles(storeFiles, snapshotId);
2534    }
2535
2536    /**
2537     * Abort the snapshot preparation. Drops the snapshot if any.
2538     */
2539    @Override
2540    public void abort() throws IOException {
2541      if (snapshot != null) {
2542        //We need to close the snapshot when aborting, otherwise, the segment scanner
2543        //won't be closed. If we are using MSLAB, the chunk referenced by those scanners
2544        //can't be released, thus memory leak
2545        snapshot.close();
2546        HStore.this.updateStorefiles(Collections.emptyList(), snapshot.getId());
2547      }
2548    }
2549  }
2550
2551  @Override
2552  public boolean needsCompaction() {
2553    List<HStoreFile> filesCompactingClone = null;
2554    synchronized (filesCompacting) {
2555      filesCompactingClone = Lists.newArrayList(filesCompacting);
2556    }
2557    return this.storeEngine.needsCompaction(filesCompactingClone);
2558  }
2559
2560  /**
2561   * Used for tests.
2562   * @return cache configuration for this Store.
2563   */
2564  @VisibleForTesting
2565  public CacheConfig getCacheConfig() {
2566    return this.cacheConf;
2567  }
2568
2569  public static final long FIXED_OVERHEAD =
2570      ClassSize.align(ClassSize.OBJECT + (29 * ClassSize.REFERENCE) + (2 * Bytes.SIZEOF_LONG)
2571              + (6 * Bytes.SIZEOF_INT) + (2 * Bytes.SIZEOF_BOOLEAN));
2572
2573  public static final long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD
2574      + ClassSize.OBJECT + ClassSize.REENTRANT_LOCK
2575      + ClassSize.CONCURRENT_SKIPLISTMAP
2576      + ClassSize.CONCURRENT_SKIPLISTMAP_ENTRY + ClassSize.OBJECT
2577      + ScanInfo.FIXED_OVERHEAD);
2578
2579  @Override
2580  public long heapSize() {
2581    MemStoreSize memstoreSize = this.memstore.size();
2582    return DEEP_OVERHEAD + memstoreSize.getHeapSize();
2583  }
2584
2585  @Override
2586  public CellComparator getComparator() {
2587    return comparator;
2588  }
2589
2590  public ScanInfo getScanInfo() {
2591    return scanInfo;
2592  }
2593
2594  /**
2595   * Set scan info, used by test
2596   * @param scanInfo new scan info to use for test
2597   */
2598  void setScanInfo(ScanInfo scanInfo) {
2599    this.scanInfo = scanInfo;
2600  }
2601
2602  @Override
2603  public boolean hasTooManyStoreFiles() {
2604    return getStorefilesCount() > this.blockingFileCount;
2605  }
2606
2607  @Override
2608  public long getFlushedCellsCount() {
2609    return flushedCellsCount.get();
2610  }
2611
2612  @Override
2613  public long getFlushedCellsSize() {
2614    return flushedCellsSize.get();
2615  }
2616
2617  @Override
2618  public long getFlushedOutputFileSize() {
2619    return flushedOutputFileSize.get();
2620  }
2621
2622  @Override
2623  public long getCompactedCellsCount() {
2624    return compactedCellsCount.get();
2625  }
2626
2627  @Override
2628  public long getCompactedCellsSize() {
2629    return compactedCellsSize.get();
2630  }
2631
2632  @Override
2633  public long getMajorCompactedCellsCount() {
2634    return majorCompactedCellsCount.get();
2635  }
2636
2637  @Override
2638  public long getMajorCompactedCellsSize() {
2639    return majorCompactedCellsSize.get();
2640  }
2641
2642  /**
2643   * Returns the StoreEngine that is backing this concrete implementation of Store.
2644   * @return Returns the {@link StoreEngine} object used internally inside this HStore object.
2645   */
2646  @VisibleForTesting
2647  public StoreEngine<?, ?, ?, ?> getStoreEngine() {
2648    return this.storeEngine;
2649  }
2650
2651  protected OffPeakHours getOffPeakHours() {
2652    return this.offPeakHours;
2653  }
2654
2655  /**
2656   * {@inheritDoc}
2657   */
2658  @Override
2659  public void onConfigurationChange(Configuration conf) {
2660    this.conf = new CompoundConfiguration()
2661            .add(conf)
2662            .addBytesMap(family.getValues());
2663    this.storeEngine.compactionPolicy.setConf(conf);
2664    this.offPeakHours = OffPeakHours.getInstance(conf);
2665  }
2666
2667  /**
2668   * {@inheritDoc}
2669   */
2670  @Override
2671  public void registerChildren(ConfigurationManager manager) {
2672    // No children to register
2673  }
2674
2675  /**
2676   * {@inheritDoc}
2677   */
2678  @Override
2679  public void deregisterChildren(ConfigurationManager manager) {
2680    // No children to deregister
2681  }
2682
2683  @Override
2684  public double getCompactionPressure() {
2685    return storeEngine.getStoreFileManager().getCompactionPressure();
2686  }
2687
2688  @Override
2689  public boolean isPrimaryReplicaStore() {
2690    return getRegionInfo().getReplicaId() == RegionInfo.DEFAULT_REPLICA_ID;
2691  }
2692
2693  /**
2694   * Sets the store up for a region level snapshot operation.
2695   * @see #postSnapshotOperation()
2696   */
2697  public void preSnapshotOperation() {
2698    archiveLock.lock();
2699  }
2700
2701  /**
2702   * Perform tasks needed after the completion of snapshot operation.
2703   * @see #preSnapshotOperation()
2704   */
2705  public void postSnapshotOperation() {
2706    archiveLock.unlock();
2707  }
2708
2709  /**
2710   * Closes and archives the compacted files under this store
2711   */
2712  public synchronized void closeAndArchiveCompactedFiles() throws IOException {
2713    // ensure other threads do not attempt to archive the same files on close()
2714    archiveLock.lock();
2715    try {
2716      lock.readLock().lock();
2717      Collection<HStoreFile> copyCompactedfiles = null;
2718      try {
2719        Collection<HStoreFile> compactedfiles =
2720            this.getStoreEngine().getStoreFileManager().getCompactedfiles();
2721        if (CollectionUtils.isNotEmpty(compactedfiles)) {
2722          // Do a copy under read lock
2723          copyCompactedfiles = new ArrayList<>(compactedfiles);
2724        } else {
2725          LOG.trace("No compacted files to archive");
2726        }
2727      } finally {
2728        lock.readLock().unlock();
2729      }
2730      if (CollectionUtils.isNotEmpty(copyCompactedfiles)) {
2731        removeCompactedfiles(copyCompactedfiles, true);
2732      }
2733    } finally {
2734      archiveLock.unlock();
2735    }
2736  }
2737
2738  /**
2739   * Archives and removes the compacted files
2740   * @param compactedfiles The compacted files in this store that are not active in reads
2741   * @param evictOnClose true if blocks should be evicted from the cache when an HFile reader is
2742   *   closed, false if not
2743   */
2744  private void removeCompactedfiles(Collection<HStoreFile> compactedfiles, boolean evictOnClose)
2745      throws IOException {
2746    final List<HStoreFile> filesToRemove = new ArrayList<>(compactedfiles.size());
2747    final List<Long> storeFileSizes = new ArrayList<>(compactedfiles.size());
2748    for (final HStoreFile file : compactedfiles) {
2749      synchronized (file) {
2750        try {
2751          StoreFileReader r = file.getReader();
2752          if (r == null) {
2753            LOG.debug("The file {} was closed but still not archived", file);
2754            // HACK: Temporarily re-open the reader so we can get the size of the file. Ideally,
2755            // we should know the size of an HStoreFile without having to ask the HStoreFileReader
2756            // for that.
2757            long length = getStoreFileSize(file);
2758            filesToRemove.add(file);
2759            storeFileSizes.add(length);
2760            continue;
2761          }
2762
2763          if (file.isCompactedAway() && !file.isReferencedInReads()) {
2764            // Even if deleting fails we need not bother as any new scanners won't be
2765            // able to use the compacted file as the status is already compactedAway
2766            LOG.trace("Closing and archiving the file {}", file);
2767            // Copy the file size before closing the reader
2768            final long length = r.length();
2769            r.close(evictOnClose);
2770            // Just close and return
2771            filesToRemove.add(file);
2772            // Only add the length if we successfully added the file to `filesToRemove`
2773            storeFileSizes.add(length);
2774          } else {
2775            LOG.info("Can't archive compacted file " + file.getPath()
2776                + " because of either isCompactedAway=" + file.isCompactedAway()
2777                + " or file has reference, isReferencedInReads=" + file.isReferencedInReads()
2778                + ", refCount=" + r.getRefCount() + ", skipping for now.");
2779          }
2780        } catch (Exception e) {
2781          LOG.error("Exception while trying to close the compacted store file {}", file.getPath(),
2782              e);
2783        }
2784      }
2785    }
2786    if (this.isPrimaryReplicaStore()) {
2787      // Only the primary region is allowed to move the file to archive.
2788      // The secondary region does not move the files to archive. Any active reads from
2789      // the secondary region will still work because the file as such has active readers on it.
2790      if (!filesToRemove.isEmpty()) {
2791        LOG.debug("Moving the files {} to archive", filesToRemove);
2792        // Only if this is successful it has to be removed
2793        try {
2794          this.fs.removeStoreFiles(this.getColumnFamilyDescriptor().getNameAsString(),
2795            filesToRemove);
2796        } catch (FailedArchiveException fae) {
2797          // Even if archiving some files failed, we still need to clear out any of the
2798          // files which were successfully archived.  Otherwise we will receive a
2799          // FileNotFoundException when we attempt to re-archive them in the next go around.
2800          Collection<Path> failedFiles = fae.getFailedFiles();
2801          Iterator<HStoreFile> iter = filesToRemove.iterator();
2802          Iterator<Long> sizeIter = storeFileSizes.iterator();
2803          while (iter.hasNext()) {
2804            sizeIter.next();
2805            if (failedFiles.contains(iter.next().getPath())) {
2806              iter.remove();
2807              sizeIter.remove();
2808            }
2809          }
2810          if (!filesToRemove.isEmpty()) {
2811            clearCompactedfiles(filesToRemove);
2812          }
2813          throw fae;
2814        }
2815      }
2816    }
2817    if (!filesToRemove.isEmpty()) {
2818      // Clear the compactedfiles from the store file manager
2819      clearCompactedfiles(filesToRemove);
2820      // Try to send report of this archival to the Master for updating quota usage faster
2821      reportArchivedFilesForQuota(filesToRemove, storeFileSizes);
2822    }
2823  }
2824
2825  /**
2826   * Computes the length of a store file without succumbing to any errors along the way. If an
2827   * error is encountered, the implementation returns {@code 0} instead of the actual size.
2828   *
2829   * @param file The file to compute the size of.
2830   * @return The size in bytes of the provided {@code file}.
2831   */
2832  long getStoreFileSize(HStoreFile file) {
2833    long length = 0;
2834    try {
2835      file.initReader();
2836      length = file.getReader().length();
2837    } catch (IOException e) {
2838      LOG.trace("Failed to open reader when trying to compute store file size for {}, ignoring",
2839        file, e);
2840    } finally {
2841      try {
2842        file.closeStoreFile(
2843            file.getCacheConf() != null ? file.getCacheConf().shouldEvictOnClose() : true);
2844      } catch (IOException e) {
2845        LOG.trace("Failed to close reader after computing store file size for {}, ignoring",
2846          file, e);
2847      }
2848    }
2849    return length;
2850  }
2851
2852  public Long preFlushSeqIDEstimation() {
2853    return memstore.preFlushSeqIDEstimation();
2854  }
2855
2856  @Override
2857  public boolean isSloppyMemStore() {
2858    return this.memstore.isSloppy();
2859  }
2860
2861  private void clearCompactedfiles(List<HStoreFile> filesToRemove) throws IOException {
2862    LOG.trace("Clearing the compacted file {} from this store", filesToRemove);
2863    try {
2864      lock.writeLock().lock();
2865      this.getStoreEngine().getStoreFileManager().removeCompactedFiles(filesToRemove);
2866    } finally {
2867      lock.writeLock().unlock();
2868    }
2869  }
2870
2871  void reportArchivedFilesForQuota(List<? extends StoreFile> archivedFiles, List<Long> fileSizes) {
2872    // Sanity check from the caller
2873    if (archivedFiles.size() != fileSizes.size()) {
2874      throw new RuntimeException("Coding error: should never see lists of varying size");
2875    }
2876    RegionServerServices rss = this.region.getRegionServerServices();
2877    if (rss == null) {
2878      return;
2879    }
2880    List<Entry<String,Long>> filesWithSizes = new ArrayList<>(archivedFiles.size());
2881    Iterator<Long> fileSizeIter = fileSizes.iterator();
2882    for (StoreFile storeFile : archivedFiles) {
2883      final long fileSize = fileSizeIter.next();
2884      if (storeFile.isHFile() && fileSize != 0) {
2885        filesWithSizes.add(Maps.immutableEntry(storeFile.getPath().getName(), fileSize));
2886      }
2887    }
2888    if (LOG.isTraceEnabled()) {
2889      LOG.trace("Files archived: " + archivedFiles + ", reporting the following to the Master: "
2890          + filesWithSizes);
2891    }
2892    boolean success = rss.reportFileArchivalForQuotas(getTableName(), filesWithSizes);
2893    if (!success) {
2894      LOG.warn("Failed to report archival of files: " + filesWithSizes);
2895    }
2896  }
2897
2898  @Override
2899  public int getCurrentParallelPutCount() {
2900    return currentParallelPutCount.get();
2901  }
2902
2903  public int getStoreRefCount() {
2904    return this.storeEngine.getStoreFileManager().getStorefiles().stream()
2905      .filter(sf -> sf.getReader() != null).filter(HStoreFile::isHFile)
2906      .mapToInt(HStoreFile::getRefCount).sum();
2907  }
2908
2909  /**
2910   * @return get maximum ref count of storeFile among all compacted HStore Files for the HStore
2911   */
2912  public int getMaxCompactedStoreFileRefCount() {
2913    OptionalInt maxCompactedStoreFileRefCount = this.storeEngine.getStoreFileManager()
2914      .getCompactedfiles()
2915      .stream()
2916      .filter(sf -> sf.getReader() != null)
2917      .filter(HStoreFile::isHFile)
2918      .mapToInt(HStoreFile::getRefCount)
2919      .max();
2920    return maxCompactedStoreFileRefCount.isPresent()
2921      ? maxCompactedStoreFileRefCount.getAsInt() : 0;
2922  }
2923
2924  @Override
2925  public long getMemstoreOnlyRowReadsCount() {
2926    return memstoreOnlyRowReadsCount.sum();
2927  }
2928
2929  @Override
2930  public long getMixedRowReadsCount() {
2931    return mixedRowReadsCount.sum();
2932  }
2933
2934  void updateMetricsStore(boolean memstoreRead) {
2935    if (memstoreRead) {
2936      memstoreOnlyRowReadsCount.increment();
2937    } else {
2938      mixedRowReadsCount.increment();
2939    }
2940  }
2941}