View Javadoc

1   /**
2    *
3    * Licensed to the Apache Software Foundation (ASF) under one
4    * or more contributor license agreements.  See the NOTICE file
5    * distributed with this work for additional information
6    * regarding copyright ownership.  The ASF licenses this file
7    * to you under the Apache License, Version 2.0 (the
8    * "License"); you may not use this file except in compliance
9    * with the License.  You may obtain a copy of the License at
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
13   * Unless required by applicable law or agreed to in writing, software
14   * distributed under the License is distributed on an "AS IS" BASIS,
15   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16   * See the License for the specific language governing permissions and
17   * limitations under the License.
18   */
19  package org.apache.hadoop.hbase.regionserver;
20  
21  import java.io.IOException;
22  import java.io.InterruptedIOException;
23  import java.net.InetSocketAddress;
24  import java.security.Key;
25  import java.security.KeyException;
26  import java.security.PrivilegedExceptionAction;
27  import java.util.ArrayList;
28  import java.util.Collection;
29  import java.util.Collections;
30  import java.util.HashMap;
31  import java.util.HashSet;
32  import java.util.Iterator;
33  import java.util.List;
34  import java.util.NavigableSet;
35  import java.util.Set;
36  import java.util.concurrent.Callable;
37  import java.util.concurrent.CompletionService;
38  import java.util.concurrent.ConcurrentHashMap;
39  import java.util.concurrent.ExecutionException;
40  import java.util.concurrent.ExecutorCompletionService;
41  import java.util.concurrent.Future;
42  import java.util.concurrent.ThreadPoolExecutor;
43  import java.util.concurrent.atomic.AtomicBoolean;
44  import java.util.concurrent.locks.ReentrantReadWriteLock;
45  
46  import org.apache.commons.logging.Log;
47  import org.apache.commons.logging.LogFactory;
48  import org.apache.hadoop.conf.Configuration;
49  import org.apache.hadoop.fs.FileSystem;
50  import org.apache.hadoop.fs.Path;
51  import org.apache.hadoop.hbase.Cell;
52  import org.apache.hadoop.hbase.CellComparator;
53  import org.apache.hadoop.hbase.CellUtil;
54  import org.apache.hadoop.hbase.CompoundConfiguration;
55  import org.apache.hadoop.hbase.HColumnDescriptor;
56  import org.apache.hadoop.hbase.HConstants;
57  import org.apache.hadoop.hbase.HRegionInfo;
58  import org.apache.hadoop.hbase.KeyValue;
59  import org.apache.hadoop.hbase.RemoteExceptionHandler;
60  import org.apache.hadoop.hbase.TableName;
61  import org.apache.hadoop.hbase.Tag;
62  import org.apache.hadoop.hbase.TagType;
63  import org.apache.hadoop.hbase.classification.InterfaceAudience;
64  import org.apache.hadoop.hbase.client.Scan;
65  import org.apache.hadoop.hbase.conf.ConfigurationManager;
66  import org.apache.hadoop.hbase.io.compress.Compression;
67  import org.apache.hadoop.hbase.io.crypto.Cipher;
68  import org.apache.hadoop.hbase.io.crypto.Encryption;
69  import org.apache.hadoop.hbase.io.hfile.CacheConfig;
70  import org.apache.hadoop.hbase.io.hfile.HFile;
71  import org.apache.hadoop.hbase.io.hfile.HFileContext;
72  import org.apache.hadoop.hbase.io.hfile.HFileContextBuilder;
73  import org.apache.hadoop.hbase.io.hfile.HFileDataBlockEncoder;
74  import org.apache.hadoop.hbase.io.hfile.HFileDataBlockEncoderImpl;
75  import org.apache.hadoop.hbase.io.hfile.HFileScanner;
76  import org.apache.hadoop.hbase.io.hfile.InvalidHFileException;
77  import org.apache.hadoop.hbase.monitoring.MonitoredTask;
78  import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
79  import org.apache.hadoop.hbase.protobuf.generated.WALProtos.CompactionDescriptor;
80  import org.apache.hadoop.hbase.regionserver.compactions.CompactionContext;
81  import org.apache.hadoop.hbase.regionserver.compactions.CompactionProgress;
82  import org.apache.hadoop.hbase.regionserver.compactions.CompactionRequest;
83  import org.apache.hadoop.hbase.regionserver.compactions.DefaultCompactor;
84  import org.apache.hadoop.hbase.regionserver.compactions.OffPeakHours;
85  import org.apache.hadoop.hbase.regionserver.compactions.CompactionThroughputController;
86  import org.apache.hadoop.hbase.regionserver.wal.WALUtil;
87  import org.apache.hadoop.hbase.security.EncryptionUtil;
88  import org.apache.hadoop.hbase.security.User;
89  import org.apache.hadoop.hbase.util.Bytes;
90  import org.apache.hadoop.hbase.util.ChecksumType;
91  import org.apache.hadoop.hbase.util.ClassSize;
92  import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
93  import org.apache.hadoop.hbase.util.ReflectionUtils;
94  import org.apache.hadoop.util.StringUtils;
95  import org.apache.hadoop.util.StringUtils.TraditionalBinaryPrefix;
96  
97  import com.google.common.annotations.VisibleForTesting;
98  import com.google.common.base.Preconditions;
99  import com.google.common.collect.ImmutableCollection;
100 import com.google.common.collect.ImmutableList;
101 import com.google.common.collect.Lists;
102 import com.google.common.collect.Sets;
103 
104 /**
105  * A Store holds a column family in a Region.  Its a memstore and a set of zero
106  * or more StoreFiles, which stretch backwards over time.
107  *
108  * <p>There's no reason to consider append-logging at this level; all logging
109  * and locking is handled at the HRegion level.  Store just provides
110  * services to manage sets of StoreFiles.  One of the most important of those
111  * services is compaction services where files are aggregated once they pass
112  * a configurable threshold.
113  *
114  * <p>Locking and transactions are handled at a higher level.  This API should
115  * not be called directly but by an HRegion manager.
116  */
117 @InterfaceAudience.Private
118 public class HStore implements Store {
119   private static final String MEMSTORE_CLASS_NAME = "hbase.regionserver.memstore.class";
120   public static final String COMPACTCHECKER_INTERVAL_MULTIPLIER_KEY =
121       "hbase.server.compactchecker.interval.multiplier";
122   public static final String BLOCKING_STOREFILES_KEY = "hbase.hstore.blockingStoreFiles";
123   public static final int DEFAULT_COMPACTCHECKER_INTERVAL_MULTIPLIER = 1000;
124   public static final int DEFAULT_BLOCKING_STOREFILE_COUNT = 7;
125 
126   private static final Log LOG = LogFactory.getLog(HStore.class);
127 
128   protected final MemStore memstore;
129   // This stores directory in the filesystem.
130   private final HRegion region;
131   private final HColumnDescriptor family;
132   private final HRegionFileSystem fs;
133   private Configuration conf;
134   private final CacheConfig cacheConf;
135   private long lastCompactSize = 0;
136   volatile boolean forceMajor = false;
137   /* how many bytes to write between status checks */
138   static int closeCheckInterval = 0;
139   private volatile long storeSize = 0L;
140   private volatile long totalUncompressedBytes = 0L;
141 
142   /**
143    * RWLock for store operations.
144    * Locked in shared mode when the list of component stores is looked at:
145    *   - all reads/writes to table data
146    *   - checking for split
147    * Locked in exclusive mode when the list of component stores is modified:
148    *   - closing
149    *   - completing a compaction
150    */
151   final ReentrantReadWriteLock lock = new ReentrantReadWriteLock();
152   private final boolean verifyBulkLoads;
153 
154   private ScanInfo scanInfo;
155 
156   // All access must be synchronized.
157   // TODO: ideally, this should be part of storeFileManager, as we keep passing this to it.
158   private final List<StoreFile> filesCompacting = Lists.newArrayList();
159 
160   // All access must be synchronized.
161   private final Set<ChangedReadersObserver> changedReaderObservers =
162     Collections.newSetFromMap(new ConcurrentHashMap<ChangedReadersObserver, Boolean>());
163 
164   private final int blocksize;
165   private HFileDataBlockEncoder dataBlockEncoder;
166 
167   /** Checksum configuration */
168   private ChecksumType checksumType;
169   private int bytesPerChecksum;
170 
171   // Comparing KeyValues
172   private final KeyValue.KVComparator comparator;
173 
174   final StoreEngine<?, ?, ?, ?> storeEngine;
175 
176   private static final AtomicBoolean offPeakCompactionTracker = new AtomicBoolean();
177   private volatile OffPeakHours offPeakHours;
178 
179   private static final int DEFAULT_FLUSH_RETRIES_NUMBER = 10;
180   private int flushRetriesNumber;
181   private int pauseTime;
182 
183   private long blockingFileCount;
184   private int compactionCheckMultiplier;
185 
186   private Encryption.Context cryptoContext = Encryption.Context.NONE;
187 
188   private volatile long flushedCellsCount = 0;
189   private volatile long compactedCellsCount = 0;
190   private volatile long majorCompactedCellsCount = 0;
191   private volatile long flushedCellsSize = 0;
192   private volatile long compactedCellsSize = 0;
193   private volatile long majorCompactedCellsSize = 0;
194 
195   /**
196    * Constructor
197    * @param region
198    * @param family HColumnDescriptor for this column
199    * @param confParam configuration object
200    * failed.  Can be null.
201    * @throws IOException
202    */
203   protected HStore(final HRegion region, final HColumnDescriptor family,
204       final Configuration confParam) throws IOException {
205 
206     HRegionInfo info = region.getRegionInfo();
207     this.fs = region.getRegionFileSystem();
208 
209     // Assemble the store's home directory and Ensure it exists.
210     fs.createStoreDir(family.getNameAsString());
211     this.region = region;
212     this.family = family;
213     // 'conf' renamed to 'confParam' b/c we use this.conf in the constructor
214     // CompoundConfiguration will look for keys in reverse order of addition, so we'd
215     // add global config first, then table and cf overrides, then cf metadata.
216     this.conf = new CompoundConfiguration()
217       .add(confParam)
218       .addStringMap(region.getTableDesc().getConfiguration())
219       .addStringMap(family.getConfiguration())
220       .addWritableMap(family.getValues());
221     this.blocksize = family.getBlocksize();
222 
223     this.dataBlockEncoder =
224         new HFileDataBlockEncoderImpl(family.getDataBlockEncoding());
225 
226     this.comparator = info.getComparator();
227     // used by ScanQueryMatcher
228     long timeToPurgeDeletes =
229         Math.max(conf.getLong("hbase.hstore.time.to.purge.deletes", 0), 0);
230     LOG.trace("Time to purge deletes set to " + timeToPurgeDeletes +
231         "ms in store " + this);
232     // Get TTL
233     long ttl = determineTTLFromFamily(family);
234     // Why not just pass a HColumnDescriptor in here altogether?  Even if have
235     // to clone it?
236     scanInfo = new ScanInfo(conf, family, ttl, timeToPurgeDeletes, this.comparator);
237     String className = conf.get(MEMSTORE_CLASS_NAME, DefaultMemStore.class.getName());
238     this.memstore = ReflectionUtils.instantiateWithCustomCtor(className, new Class[] {
239         Configuration.class, KeyValue.KVComparator.class }, new Object[] { conf, this.comparator });
240     this.offPeakHours = OffPeakHours.getInstance(conf);
241 
242     // Setting up cache configuration for this family
243     this.cacheConf = new CacheConfig(conf, family);
244 
245     this.verifyBulkLoads = conf.getBoolean("hbase.hstore.bulkload.verify", false);
246 
247     this.blockingFileCount =
248         conf.getInt(BLOCKING_STOREFILES_KEY, DEFAULT_BLOCKING_STOREFILE_COUNT);
249     this.compactionCheckMultiplier = conf.getInt(
250         COMPACTCHECKER_INTERVAL_MULTIPLIER_KEY, DEFAULT_COMPACTCHECKER_INTERVAL_MULTIPLIER);
251     if (this.compactionCheckMultiplier <= 0) {
252       LOG.error("Compaction check period multiplier must be positive, setting default: "
253           + DEFAULT_COMPACTCHECKER_INTERVAL_MULTIPLIER);
254       this.compactionCheckMultiplier = DEFAULT_COMPACTCHECKER_INTERVAL_MULTIPLIER;
255     }
256 
257     if (HStore.closeCheckInterval == 0) {
258       HStore.closeCheckInterval = conf.getInt(
259           "hbase.hstore.close.check.interval", 10*1000*1000 /* 10 MB */);
260     }
261 
262     this.storeEngine = StoreEngine.create(this, this.conf, this.comparator);
263     this.storeEngine.getStoreFileManager().loadFiles(loadStoreFiles());
264 
265     // Initialize checksum type from name. The names are CRC32, CRC32C, etc.
266     this.checksumType = getChecksumType(conf);
267     // initilize bytes per checksum
268     this.bytesPerChecksum = getBytesPerChecksum(conf);
269     flushRetriesNumber = conf.getInt(
270         "hbase.hstore.flush.retries.number", DEFAULT_FLUSH_RETRIES_NUMBER);
271     pauseTime = conf.getInt(HConstants.HBASE_SERVER_PAUSE, HConstants.DEFAULT_HBASE_SERVER_PAUSE);
272     if (flushRetriesNumber <= 0) {
273       throw new IllegalArgumentException(
274           "hbase.hstore.flush.retries.number must be > 0, not "
275               + flushRetriesNumber);
276     }
277 
278     // Crypto context for new store files
279     String cipherName = family.getEncryptionType();
280     if (cipherName != null) {
281       Cipher cipher;
282       Key key;
283       byte[] keyBytes = family.getEncryptionKey();
284       if (keyBytes != null) {
285         // Family provides specific key material
286         String masterKeyName = conf.get(HConstants.CRYPTO_MASTERKEY_NAME_CONF_KEY,
287           User.getCurrent().getShortName());
288         try {
289           // First try the master key
290           key = EncryptionUtil.unwrapKey(conf, masterKeyName, keyBytes);
291         } catch (KeyException e) {
292           // If the current master key fails to unwrap, try the alternate, if
293           // one is configured
294           if (LOG.isDebugEnabled()) {
295             LOG.debug("Unable to unwrap key with current master key '" + masterKeyName + "'");
296           }
297           String alternateKeyName =
298             conf.get(HConstants.CRYPTO_MASTERKEY_ALTERNATE_NAME_CONF_KEY);
299           if (alternateKeyName != null) {
300             try {
301               key = EncryptionUtil.unwrapKey(conf, alternateKeyName, keyBytes);
302             } catch (KeyException ex) {
303               throw new IOException(ex);
304             }
305           } else {
306             throw new IOException(e);
307           }
308         }
309         // Use the algorithm the key wants
310         cipher = Encryption.getCipher(conf, key.getAlgorithm());
311         if (cipher == null) {
312           throw new RuntimeException("Cipher '" + key.getAlgorithm() + "' is not available");
313         }
314         // Fail if misconfigured
315         // We use the encryption type specified in the column schema as a sanity check on
316         // what the wrapped key is telling us
317         if (!cipher.getName().equalsIgnoreCase(cipherName)) {
318           throw new RuntimeException("Encryption for family '" + family.getNameAsString() +
319             "' configured with type '" + cipherName +
320             "' but key specifies algorithm '" + cipher.getName() + "'");
321         }
322       } else {
323         // Family does not provide key material, create a random key
324         cipher = Encryption.getCipher(conf, cipherName);
325         if (cipher == null) {
326           throw new RuntimeException("Cipher '" + cipherName + "' is not available");
327         }
328         key = cipher.getRandomKey();
329       }
330       cryptoContext = Encryption.newContext(conf);
331       cryptoContext.setCipher(cipher);
332       cryptoContext.setKey(key);
333     }
334   }
335 
336   /**
337    * @param family
338    * @return TTL in seconds of the specified family
339    */
340   private static long determineTTLFromFamily(final HColumnDescriptor family) {
341     // HCD.getTimeToLive returns ttl in seconds.  Convert to milliseconds.
342     long ttl = family.getTimeToLive();
343     if (ttl == HConstants.FOREVER) {
344       // Default is unlimited ttl.
345       ttl = Long.MAX_VALUE;
346     } else if (ttl == -1) {
347       ttl = Long.MAX_VALUE;
348     } else {
349       // Second -> ms adjust for user data
350       ttl *= 1000;
351     }
352     return ttl;
353   }
354 
355   @Override
356   public String getColumnFamilyName() {
357     return this.family.getNameAsString();
358   }
359 
360   @Override
361   public TableName getTableName() {
362     return this.getRegionInfo().getTable();
363   }
364 
365   @Override
366   public FileSystem getFileSystem() {
367     return this.fs.getFileSystem();
368   }
369 
370   public HRegionFileSystem getRegionFileSystem() {
371     return this.fs;
372   }
373 
374   /* Implementation of StoreConfigInformation */
375   @Override
376   public long getStoreFileTtl() {
377     // TTL only applies if there's no MIN_VERSIONs setting on the column.
378     return (this.scanInfo.getMinVersions() == 0) ? this.scanInfo.getTtl() : Long.MAX_VALUE;
379   }
380 
381   @Override
382   public long getMemstoreFlushSize() {
383     // TODO: Why is this in here?  The flushsize of the region rather than the store?  St.Ack
384     return this.region.memstoreFlushSize;
385   }
386 
387   @Override
388   public long getFlushableSize() {
389     return this.memstore.getFlushableSize();
390   }
391 
392   @Override
393   public long getSnapshotSize() {
394     return this.memstore.getSnapshotSize();
395   }
396 
397   @Override
398   public long getCompactionCheckMultiplier() {
399     return this.compactionCheckMultiplier;
400   }
401 
402   @Override
403   public long getBlockingFileCount() {
404     return blockingFileCount;
405   }
406   /* End implementation of StoreConfigInformation */
407 
408   /**
409    * Returns the configured bytesPerChecksum value.
410    * @param conf The configuration
411    * @return The bytesPerChecksum that is set in the configuration
412    */
413   public static int getBytesPerChecksum(Configuration conf) {
414     return conf.getInt(HConstants.BYTES_PER_CHECKSUM,
415                        HFile.DEFAULT_BYTES_PER_CHECKSUM);
416   }
417 
418   /**
419    * Returns the configured checksum algorithm.
420    * @param conf The configuration
421    * @return The checksum algorithm that is set in the configuration
422    */
423   public static ChecksumType getChecksumType(Configuration conf) {
424     String checksumName = conf.get(HConstants.CHECKSUM_TYPE_NAME);
425     if (checksumName == null) {
426       return ChecksumType.getDefaultChecksumType();
427     } else {
428       return ChecksumType.nameToType(checksumName);
429     }
430   }
431 
432   /**
433    * @return how many bytes to write between status checks
434    */
435   public static int getCloseCheckInterval() {
436     return closeCheckInterval;
437   }
438 
439   @Override
440   public HColumnDescriptor getFamily() {
441     return this.family;
442   }
443 
444   /**
445    * @return The maximum sequence id in all store files. Used for log replay.
446    */
447   @Override
448   public long getMaxSequenceId() {
449     return StoreFile.getMaxSequenceIdInList(this.getStorefiles());
450   }
451 
452   @Override
453   public long getMaxMemstoreTS() {
454     return StoreFile.getMaxMemstoreTSInList(this.getStorefiles());
455   }
456 
457   /**
458    * @param tabledir {@link Path} to where the table is being stored
459    * @param hri {@link HRegionInfo} for the region.
460    * @param family {@link HColumnDescriptor} describing the column family
461    * @return Path to family/Store home directory.
462    */
463   @Deprecated
464   public static Path getStoreHomedir(final Path tabledir,
465       final HRegionInfo hri, final byte[] family) {
466     return getStoreHomedir(tabledir, hri.getEncodedName(), family);
467   }
468 
469   /**
470    * @param tabledir {@link Path} to where the table is being stored
471    * @param encodedName Encoded region name.
472    * @param family {@link HColumnDescriptor} describing the column family
473    * @return Path to family/Store home directory.
474    */
475   @Deprecated
476   public static Path getStoreHomedir(final Path tabledir,
477       final String encodedName, final byte[] family) {
478     return new Path(tabledir, new Path(encodedName, Bytes.toString(family)));
479   }
480 
481   @Override
482   public HFileDataBlockEncoder getDataBlockEncoder() {
483     return dataBlockEncoder;
484   }
485 
486   /**
487    * Should be used only in tests.
488    * @param blockEncoder the block delta encoder to use
489    */
490   void setDataBlockEncoderInTest(HFileDataBlockEncoder blockEncoder) {
491     this.dataBlockEncoder = blockEncoder;
492   }
493 
494   /**
495    * Creates an unsorted list of StoreFile loaded in parallel
496    * from the given directory.
497    * @throws IOException
498    */
499   private List<StoreFile> loadStoreFiles() throws IOException {
500     Collection<StoreFileInfo> files = fs.getStoreFiles(getColumnFamilyName());
501     return openStoreFiles(files);
502   }
503 
504   private List<StoreFile> openStoreFiles(Collection<StoreFileInfo> files) throws IOException {
505     if (files == null || files.size() == 0) {
506       return new ArrayList<StoreFile>();
507     }
508     // initialize the thread pool for opening store files in parallel..
509     ThreadPoolExecutor storeFileOpenerThreadPool =
510       this.region.getStoreFileOpenAndCloseThreadPool("StoreFileOpenerThread-" +
511           this.getColumnFamilyName());
512     CompletionService<StoreFile> completionService =
513       new ExecutorCompletionService<StoreFile>(storeFileOpenerThreadPool);
514 
515     int totalValidStoreFile = 0;
516     for (final StoreFileInfo storeFileInfo: files) {
517       // open each store file in parallel
518       completionService.submit(new Callable<StoreFile>() {
519         @Override
520         public StoreFile call() throws IOException {
521           StoreFile storeFile = createStoreFileAndReader(storeFileInfo);
522           return storeFile;
523         }
524       });
525       totalValidStoreFile++;
526     }
527 
528     ArrayList<StoreFile> results = new ArrayList<StoreFile>(files.size());
529     IOException ioe = null;
530     try {
531       for (int i = 0; i < totalValidStoreFile; i++) {
532         try {
533           Future<StoreFile> future = completionService.take();
534           StoreFile storeFile = future.get();
535           long length = storeFile.getReader().length();
536           this.storeSize += length;
537           this.totalUncompressedBytes +=
538               storeFile.getReader().getTotalUncompressedBytes();
539           if (LOG.isDebugEnabled()) {
540             LOG.debug("loaded " + storeFile.toStringDetailed());
541           }
542           results.add(storeFile);
543         } catch (InterruptedException e) {
544           if (ioe == null) ioe = new InterruptedIOException(e.getMessage());
545         } catch (ExecutionException e) {
546           if (ioe == null) ioe = new IOException(e.getCause());
547         }
548       }
549     } finally {
550       storeFileOpenerThreadPool.shutdownNow();
551     }
552     if (ioe != null) {
553       // close StoreFile readers
554       boolean evictOnClose = 
555           cacheConf != null? cacheConf.shouldEvictOnClose(): true; 
556       for (StoreFile file : results) {
557         try {
558           if (file != null) file.closeReader(evictOnClose);
559         } catch (IOException e) {
560           LOG.warn(e.getMessage());
561         }
562       }
563       throw ioe;
564     }
565 
566     return results;
567   }
568 
569   /**
570    * Checks the underlying store files, and opens the files that  have not
571    * been opened, and removes the store file readers for store files no longer
572    * available. Mainly used by secondary region replicas to keep up to date with
573    * the primary region files.
574    * @throws IOException
575    */
576   @Override
577   public void refreshStoreFiles() throws IOException {
578     Collection<StoreFileInfo> newFiles = fs.getStoreFiles(getColumnFamilyName());
579     refreshStoreFilesInternal(newFiles);
580   }
581 
582   @Override
583   public void refreshStoreFiles(Collection<String> newFiles) throws IOException {
584     List<StoreFileInfo> storeFiles = new ArrayList<StoreFileInfo>(newFiles.size());
585     for (String file : newFiles) {
586       storeFiles.add(fs.getStoreFileInfo(getColumnFamilyName(), file));
587     }
588     refreshStoreFilesInternal(storeFiles);
589   }
590 
591   /**
592    * Checks the underlying store files, and opens the files that  have not
593    * been opened, and removes the store file readers for store files no longer
594    * available. Mainly used by secondary region replicas to keep up to date with
595    * the primary region files.
596    * @throws IOException
597    */
598   private void refreshStoreFilesInternal(Collection<StoreFileInfo> newFiles) throws IOException {
599     StoreFileManager sfm = storeEngine.getStoreFileManager();
600     Collection<StoreFile> currentFiles = sfm.getStorefiles();
601     if (currentFiles == null) currentFiles = new ArrayList<StoreFile>(0);
602 
603     if (newFiles == null) newFiles = new ArrayList<StoreFileInfo>(0);
604 
605     HashMap<StoreFileInfo, StoreFile> currentFilesSet =
606         new HashMap<StoreFileInfo, StoreFile>(currentFiles.size());
607     for (StoreFile sf : currentFiles) {
608       currentFilesSet.put(sf.getFileInfo(), sf);
609     }
610     HashSet<StoreFileInfo> newFilesSet = new HashSet<StoreFileInfo>(newFiles);
611 
612     Set<StoreFileInfo> toBeAddedFiles = Sets.difference(newFilesSet, currentFilesSet.keySet());
613     Set<StoreFileInfo> toBeRemovedFiles = Sets.difference(currentFilesSet.keySet(), newFilesSet);
614 
615     if (toBeAddedFiles.isEmpty() && toBeRemovedFiles.isEmpty()) {
616       return;
617     }
618 
619     LOG.info("Refreshing store files for region " + this.getRegionInfo().getRegionNameAsString()
620       + " files to add: " + toBeAddedFiles + " files to remove: " + toBeRemovedFiles);
621 
622     Set<StoreFile> toBeRemovedStoreFiles = new HashSet<StoreFile>(toBeRemovedFiles.size());
623     for (StoreFileInfo sfi : toBeRemovedFiles) {
624       toBeRemovedStoreFiles.add(currentFilesSet.get(sfi));
625     }
626 
627     // try to open the files
628     List<StoreFile> openedFiles = openStoreFiles(toBeAddedFiles);
629 
630     // propogate the file changes to the underlying store file manager
631     replaceStoreFiles(toBeRemovedStoreFiles, openedFiles); //won't throw an exception
632 
633     // Advance the memstore read point to be at least the new store files seqIds so that
634     // readers might pick it up. This assumes that the store is not getting any writes (otherwise
635     // in-flight transactions might be made visible)
636     if (!toBeAddedFiles.isEmpty()) {
637       region.getMVCC().advanceTo(this.getMaxSequenceId());
638     }
639 
640     // notify scanners, close file readers, and recompute store size
641     completeCompaction(toBeRemovedStoreFiles, false);
642   }
643 
644   private StoreFile createStoreFileAndReader(final Path p) throws IOException {
645     StoreFileInfo info = new StoreFileInfo(conf, this.getFileSystem(), p);
646     return createStoreFileAndReader(info);
647   }
648 
649   private StoreFile createStoreFileAndReader(final StoreFileInfo info)
650       throws IOException {
651     info.setRegionCoprocessorHost(this.region.getCoprocessorHost());
652     StoreFile storeFile = new StoreFile(this.getFileSystem(), info, this.conf, this.cacheConf,
653       this.family.getBloomFilterType());
654     StoreFile.Reader r = storeFile.createReader();
655     r.setReplicaStoreFile(isPrimaryReplicaStore());
656     return storeFile;
657   }
658 
659   @Override
660   public long add(final Cell cell) {
661     lock.readLock().lock();
662     try {
663        return this.memstore.add(cell);
664     } finally {
665       lock.readLock().unlock();
666     }
667   }
668 
669   @Override
670   public long timeOfOldestEdit() {
671     return memstore.timeOfOldestEdit();
672   }
673 
674   /**
675    * Adds a value to the memstore
676    *
677    * @param kv
678    * @return memstore size delta
679    */
680   protected long delete(final KeyValue kv) {
681     lock.readLock().lock();
682     try {
683       return this.memstore.delete(kv);
684     } finally {
685       lock.readLock().unlock();
686     }
687   }
688 
689   @Override
690   public void rollback(final Cell cell) {
691     lock.readLock().lock();
692     try {
693       this.memstore.rollback(cell);
694     } finally {
695       lock.readLock().unlock();
696     }
697   }
698 
699   /**
700    * @return All store files.
701    */
702   @Override
703   public Collection<StoreFile> getStorefiles() {
704     return this.storeEngine.getStoreFileManager().getStorefiles();
705   }
706 
707   @Override
708   public void assertBulkLoadHFileOk(Path srcPath) throws IOException {
709     HFile.Reader reader  = null;
710     try {
711       LOG.info("Validating hfile at " + srcPath + " for inclusion in "
712           + "store " + this + " region " + this.getRegionInfo().getRegionNameAsString());
713       reader = HFile.createReader(srcPath.getFileSystem(conf),
714           srcPath, cacheConf, conf);
715       reader.loadFileInfo();
716 
717       byte[] firstKey = reader.getFirstRowKey();
718       Preconditions.checkState(firstKey != null, "First key can not be null");
719       byte[] lk = reader.getLastKey();
720       Preconditions.checkState(lk != null, "Last key can not be null");
721       byte[] lastKey =  KeyValue.createKeyValueFromKey(lk).getRow();
722 
723       LOG.debug("HFile bounds: first=" + Bytes.toStringBinary(firstKey) +
724           " last=" + Bytes.toStringBinary(lastKey));
725       LOG.debug("Region bounds: first=" +
726           Bytes.toStringBinary(getRegionInfo().getStartKey()) +
727           " last=" + Bytes.toStringBinary(getRegionInfo().getEndKey()));
728 
729       if (!this.getRegionInfo().containsRange(firstKey, lastKey)) {
730         throw new WrongRegionException(
731             "Bulk load file " + srcPath.toString() + " does not fit inside region "
732             + this.getRegionInfo().getRegionNameAsString());
733       }
734 
735       if(reader.length() > conf.getLong(HConstants.HREGION_MAX_FILESIZE,
736           HConstants.DEFAULT_MAX_FILE_SIZE)) {
737         LOG.warn("Trying to bulk load hfile " + srcPath.toString() + " with size: " +
738             reader.length() + " bytes can be problematic as it may lead to oversplitting.");
739       }
740 
741       if (verifyBulkLoads) {
742         long verificationStartTime = EnvironmentEdgeManager.currentTime();
743         LOG.info("Full verification started for bulk load hfile: " + srcPath.toString());
744         Cell prevCell = null;
745         HFileScanner scanner = reader.getScanner(false, false, false);
746         scanner.seekTo();
747         do {
748           Cell cell = scanner.getKeyValue();
749           if (prevCell != null) {
750             if (CellComparator.compareRows(prevCell, cell) > 0) {
751               throw new InvalidHFileException("Previous row is greater than"
752                   + " current row: path=" + srcPath + " previous="
753                   + CellUtil.getCellKeyAsString(prevCell) + " current="
754                   + CellUtil.getCellKeyAsString(cell));
755             }
756             if (CellComparator.compareFamilies(prevCell, cell) != 0) {
757               throw new InvalidHFileException("Previous key had different"
758                   + " family compared to current key: path=" + srcPath
759                   + " previous="
760                   + Bytes.toStringBinary(prevCell.getFamilyArray(), prevCell.getFamilyOffset(),
761                       prevCell.getFamilyLength())
762                   + " current="
763                   + Bytes.toStringBinary(cell.getFamilyArray(), cell.getFamilyOffset(),
764                       cell.getFamilyLength()));
765             }
766           }
767           prevCell = cell;
768         } while (scanner.next());
769       LOG.info("Full verification complete for bulk load hfile: " + srcPath.toString()
770          + " took " + (EnvironmentEdgeManager.currentTime() - verificationStartTime)
771          + " ms");
772       }
773     } finally {
774       if (reader != null) reader.close();
775     }
776   }
777 
778   @Override
779   public Path bulkLoadHFile(String srcPathStr, long seqNum) throws IOException {
780     Path srcPath = new Path(srcPathStr);
781     Path dstPath = fs.bulkLoadStoreFile(getColumnFamilyName(), srcPath, seqNum);
782 
783     LOG.info("Loaded HFile " + srcPath + " into store '" + getColumnFamilyName() + "' as "
784         + dstPath + " - updating store file list.");
785 
786     StoreFile sf = createStoreFileAndReader(dstPath);
787     bulkLoadHFile(sf);
788 
789     LOG.info("Successfully loaded store file " + srcPath + " into store " + this
790         + " (new location: " + dstPath + ")");
791 
792     return dstPath;
793   }
794 
795   @Override
796   public void bulkLoadHFile(StoreFileInfo fileInfo) throws IOException {
797     StoreFile sf = createStoreFileAndReader(fileInfo);
798     bulkLoadHFile(sf);
799   }
800 
801   private void bulkLoadHFile(StoreFile sf) throws IOException {
802     StoreFile.Reader r = sf.getReader();
803     this.storeSize += r.length();
804     this.totalUncompressedBytes += r.getTotalUncompressedBytes();
805 
806     // Append the new storefile into the list
807     this.lock.writeLock().lock();
808     try {
809       this.storeEngine.getStoreFileManager().insertNewFiles(Lists.newArrayList(sf));
810     } finally {
811       // We need the lock, as long as we are updating the storeFiles
812       // or changing the memstore. Let us release it before calling
813       // notifyChangeReadersObservers. See HBASE-4485 for a possible
814       // deadlock scenario that could have happened if continue to hold
815       // the lock.
816       this.lock.writeLock().unlock();
817     }
818     notifyChangedReadersObservers();
819     LOG.info("Loaded HFile " + sf.getFileInfo() + " into store '" + getColumnFamilyName());
820     if (LOG.isTraceEnabled()) {
821       String traceMessage = "BULK LOAD time,size,store size,store files ["
822           + EnvironmentEdgeManager.currentTime() + "," + r.length() + "," + storeSize
823           + "," + storeEngine.getStoreFileManager().getStorefileCount() + "]";
824       LOG.trace(traceMessage);
825     }
826   }
827 
828   @Override
829   public ImmutableCollection<StoreFile> close() throws IOException {
830     this.lock.writeLock().lock();
831     try {
832       // Clear so metrics doesn't find them.
833       ImmutableCollection<StoreFile> result = storeEngine.getStoreFileManager().clearFiles();
834 
835       if (!result.isEmpty()) {
836         // initialize the thread pool for closing store files in parallel.
837         ThreadPoolExecutor storeFileCloserThreadPool = this.region
838             .getStoreFileOpenAndCloseThreadPool("StoreFileCloserThread-"
839                 + this.getColumnFamilyName());
840 
841         // close each store file in parallel
842         CompletionService<Void> completionService =
843           new ExecutorCompletionService<Void>(storeFileCloserThreadPool);
844         for (final StoreFile f : result) {
845           completionService.submit(new Callable<Void>() {
846             @Override
847             public Void call() throws IOException {
848               boolean evictOnClose = 
849                   cacheConf != null? cacheConf.shouldEvictOnClose(): true; 
850               f.closeReader(evictOnClose);
851               return null;
852             }
853           });
854         }
855 
856         IOException ioe = null;
857         try {
858           for (int i = 0; i < result.size(); i++) {
859             try {
860               Future<Void> future = completionService.take();
861               future.get();
862             } catch (InterruptedException e) {
863               if (ioe == null) {
864                 ioe = new InterruptedIOException();
865                 ioe.initCause(e);
866               }
867             } catch (ExecutionException e) {
868               if (ioe == null) ioe = new IOException(e.getCause());
869             }
870           }
871         } finally {
872           storeFileCloserThreadPool.shutdownNow();
873         }
874         if (ioe != null) throw ioe;
875       }
876       LOG.info("Closed " + this);
877       return result;
878     } finally {
879       this.lock.writeLock().unlock();
880     }
881   }
882 
883   /**
884    * Snapshot this stores memstore. Call before running
885    * {@link #flushCache(long, MemStoreSnapshot, MonitoredTask)}
886    *  so it has some work to do.
887    */
888   void snapshot() {
889     this.lock.writeLock().lock();
890     try {
891       this.memstore.snapshot();
892     } finally {
893       this.lock.writeLock().unlock();
894     }
895   }
896 
897   /**
898    * Write out current snapshot.  Presumes {@link #snapshot()} has been called previously.
899    * @param logCacheFlushId flush sequence number
900    * @param snapshot
901    * @param status
902    * @return The path name of the tmp file to which the store was flushed
903    * @throws IOException
904    */
905   protected List<Path> flushCache(final long logCacheFlushId, MemStoreSnapshot snapshot,
906       MonitoredTask status) throws IOException {
907     // If an exception happens flushing, we let it out without clearing
908     // the memstore snapshot.  The old snapshot will be returned when we say
909     // 'snapshot', the next time flush comes around.
910     // Retry after catching exception when flushing, otherwise server will abort
911     // itself
912     StoreFlusher flusher = storeEngine.getStoreFlusher();
913     IOException lastException = null;
914     for (int i = 0; i < flushRetriesNumber; i++) {
915       try {
916         List<Path> pathNames = flusher.flushSnapshot(snapshot, logCacheFlushId, status);
917         Path lastPathName = null;
918         try {
919           for (Path pathName : pathNames) {
920             lastPathName = pathName;
921             validateStoreFile(pathName);
922           }
923           return pathNames;
924         } catch (Exception e) {
925           LOG.warn("Failed validating store file " + lastPathName + ", retrying num=" + i, e);
926           if (e instanceof IOException) {
927             lastException = (IOException) e;
928           } else {
929             lastException = new IOException(e);
930           }
931         }
932       } catch (IOException e) {
933         LOG.warn("Failed flushing store file, retrying num=" + i, e);
934         lastException = e;
935       }
936       if (lastException != null && i < (flushRetriesNumber - 1)) {
937         try {
938           Thread.sleep(pauseTime);
939         } catch (InterruptedException e) {
940           IOException iie = new InterruptedIOException();
941           iie.initCause(e);
942           throw iie;
943         }
944       }
945     }
946     throw lastException;
947   }
948 
949   /*
950    * @param path The pathname of the tmp file into which the store was flushed
951    * @param logCacheFlushId
952    * @param status
953    * @return StoreFile created.
954    * @throws IOException
955    */
956   private StoreFile commitFile(final Path path, final long logCacheFlushId, MonitoredTask status)
957       throws IOException {
958     // Write-out finished successfully, move into the right spot
959     Path dstPath = fs.commitStoreFile(getColumnFamilyName(), path);
960 
961     status.setStatus("Flushing " + this + ": reopening flushed file");
962     StoreFile sf = createStoreFileAndReader(dstPath);
963 
964     StoreFile.Reader r = sf.getReader();
965     this.storeSize += r.length();
966     this.totalUncompressedBytes += r.getTotalUncompressedBytes();
967 
968     if (LOG.isInfoEnabled()) {
969       LOG.info("Added " + sf + ", entries=" + r.getEntries() +
970         ", sequenceid=" + logCacheFlushId +
971         ", filesize=" + TraditionalBinaryPrefix.long2String(r.length(), "", 1));
972     }
973     return sf;
974   }
975 
976   @Override
977   public StoreFile.Writer createWriterInTmp(long maxKeyCount, Compression.Algorithm compression,
978                                             boolean isCompaction, boolean includeMVCCReadpoint,
979                                             boolean includesTag)
980       throws IOException {
981     return createWriterInTmp(maxKeyCount, compression, isCompaction, includeMVCCReadpoint,
982         includesTag, false);
983   }
984 
985   /*
986    * @param maxKeyCount
987    * @param compression Compression algorithm to use
988    * @param isCompaction whether we are creating a new file in a compaction
989    * @param includesMVCCReadPoint - whether to include MVCC or not
990    * @param includesTag - includesTag or not
991    * @return Writer for a new StoreFile in the tmp dir.
992    */
993   @Override
994   public StoreFile.Writer createWriterInTmp(long maxKeyCount, Compression.Algorithm compression,
995       boolean isCompaction, boolean includeMVCCReadpoint, boolean includesTag,
996       boolean shouldDropBehind)
997   throws IOException {
998     final CacheConfig writerCacheConf;
999     if (isCompaction) {
1000       // Don't cache data on write on compactions.
1001       writerCacheConf = new CacheConfig(cacheConf);
1002       writerCacheConf.setCacheDataOnWrite(false);
1003     } else {
1004       writerCacheConf = cacheConf;
1005     }
1006     InetSocketAddress[] favoredNodes = null;
1007     if (region.getRegionServerServices() != null) {
1008       favoredNodes = region.getRegionServerServices().getFavoredNodesForRegion(
1009           region.getRegionInfo().getEncodedName());
1010     }
1011     HFileContext hFileContext = createFileContext(compression, includeMVCCReadpoint, includesTag,
1012       cryptoContext);
1013     StoreFile.Writer w = new StoreFile.WriterBuilder(conf, writerCacheConf,
1014         this.getFileSystem())
1015             .withFilePath(fs.createTempName())
1016             .withComparator(comparator)
1017             .withBloomType(family.getBloomFilterType())
1018             .withMaxKeyCount(maxKeyCount)
1019             .withFavoredNodes(favoredNodes)
1020             .withFileContext(hFileContext)
1021             .withShouldDropCacheBehind(shouldDropBehind)
1022             .build();
1023     return w;
1024   }
1025 
1026   private HFileContext createFileContext(Compression.Algorithm compression,
1027       boolean includeMVCCReadpoint, boolean includesTag, Encryption.Context cryptoContext) {
1028     if (compression == null) {
1029       compression = HFile.DEFAULT_COMPRESSION_ALGORITHM;
1030     }
1031     HFileContext hFileContext = new HFileContextBuilder()
1032                                 .withIncludesMvcc(includeMVCCReadpoint)
1033                                 .withIncludesTags(includesTag)
1034                                 .withCompression(compression)
1035                                 .withCompressTags(family.isCompressTags())
1036                                 .withChecksumType(checksumType)
1037                                 .withBytesPerCheckSum(bytesPerChecksum)
1038                                 .withBlockSize(blocksize)
1039                                 .withHBaseCheckSum(true)
1040                                 .withDataBlockEncoding(family.getDataBlockEncoding())
1041                                 .withEncryptionContext(cryptoContext)
1042                                 .withCreateTime(EnvironmentEdgeManager.currentTime())
1043                                 .build();
1044     return hFileContext;
1045   }
1046 
1047 
1048   /*
1049    * Change storeFiles adding into place the Reader produced by this new flush.
1050    * @param sfs Store files
1051    * @param snapshotId
1052    * @throws IOException
1053    * @return Whether compaction is required.
1054    */
1055   private boolean updateStorefiles(final List<StoreFile> sfs, final long snapshotId)
1056       throws IOException {
1057     this.lock.writeLock().lock();
1058     try {
1059       this.storeEngine.getStoreFileManager().insertNewFiles(sfs);
1060       if (snapshotId > 0) {
1061         this.memstore.clearSnapshot(snapshotId);
1062       }
1063     } finally {
1064       // We need the lock, as long as we are updating the storeFiles
1065       // or changing the memstore. Let us release it before calling
1066       // notifyChangeReadersObservers. See HBASE-4485 for a possible
1067       // deadlock scenario that could have happened if continue to hold
1068       // the lock.
1069       this.lock.writeLock().unlock();
1070     }
1071 
1072     // Tell listeners of the change in readers.
1073     notifyChangedReadersObservers();
1074 
1075     if (LOG.isTraceEnabled()) {
1076       long totalSize = 0;
1077       for (StoreFile sf : sfs) {
1078         totalSize += sf.getReader().length();
1079       }
1080       String traceMessage = "FLUSH time,count,size,store size,store files ["
1081           + EnvironmentEdgeManager.currentTime() + "," + sfs.size() + "," + totalSize
1082           + "," + storeSize + "," + storeEngine.getStoreFileManager().getStorefileCount() + "]";
1083       LOG.trace(traceMessage);
1084     }
1085     return needsCompaction();
1086   }
1087 
1088   /*
1089    * Notify all observers that set of Readers has changed.
1090    * @throws IOException
1091    */
1092   private void notifyChangedReadersObservers() throws IOException {
1093     for (ChangedReadersObserver o: this.changedReaderObservers) {
1094       o.updateReaders();
1095     }
1096   }
1097 
1098   /**
1099    * Get all scanners with no filtering based on TTL (that happens further down
1100    * the line).
1101    * @return all scanners for this store
1102    */
1103   @Override
1104   public List<KeyValueScanner> getScanners(boolean cacheBlocks, boolean isGet,
1105       boolean usePread, boolean isCompaction, ScanQueryMatcher matcher, byte[] startRow,
1106       byte[] stopRow, long readPt) throws IOException {
1107     Collection<StoreFile> storeFilesToScan;
1108     List<KeyValueScanner> memStoreScanners;
1109     this.lock.readLock().lock();
1110     try {
1111       storeFilesToScan =
1112           this.storeEngine.getStoreFileManager().getFilesForScanOrGet(isGet, startRow, stopRow);
1113       memStoreScanners = this.memstore.getScanners(readPt);
1114     } finally {
1115       this.lock.readLock().unlock();
1116     }
1117 
1118     // First the store file scanners
1119 
1120     // TODO this used to get the store files in descending order,
1121     // but now we get them in ascending order, which I think is
1122     // actually more correct, since memstore get put at the end.
1123     List<StoreFileScanner> sfScanners = StoreFileScanner.getScannersForStoreFiles(storeFilesToScan,
1124         cacheBlocks, usePread, isCompaction, false, matcher, readPt, isPrimaryReplicaStore());
1125     List<KeyValueScanner> scanners =
1126       new ArrayList<KeyValueScanner>(sfScanners.size()+1);
1127     scanners.addAll(sfScanners);
1128     // Then the memstore scanners
1129     if (memStoreScanners != null) {
1130       scanners.addAll(memStoreScanners);
1131     }
1132     return scanners;
1133   }
1134 
1135   @Override
1136   public void addChangedReaderObserver(ChangedReadersObserver o) {
1137     this.changedReaderObservers.add(o);
1138   }
1139 
1140   @Override
1141   public void deleteChangedReaderObserver(ChangedReadersObserver o) {
1142     // We don't check if observer present; it may not be (legitimately)
1143     this.changedReaderObservers.remove(o);
1144   }
1145 
1146   //////////////////////////////////////////////////////////////////////////////
1147   // Compaction
1148   //////////////////////////////////////////////////////////////////////////////
1149 
1150   /**
1151    * Compact the StoreFiles.  This method may take some time, so the calling
1152    * thread must be able to block for long periods.
1153    *
1154    * <p>During this time, the Store can work as usual, getting values from
1155    * StoreFiles and writing new StoreFiles from the memstore.
1156    *
1157    * Existing StoreFiles are not destroyed until the new compacted StoreFile is
1158    * completely written-out to disk.
1159    *
1160    * <p>The compactLock prevents multiple simultaneous compactions.
1161    * The structureLock prevents us from interfering with other write operations.
1162    *
1163    * <p>We don't want to hold the structureLock for the whole time, as a compact()
1164    * can be lengthy and we want to allow cache-flushes during this period.
1165    *
1166    * <p> Compaction event should be idempotent, since there is no IO Fencing for
1167    * the region directory in hdfs. A region server might still try to complete the
1168    * compaction after it lost the region. That is why the following events are carefully
1169    * ordered for a compaction:
1170    *  1. Compaction writes new files under region/.tmp directory (compaction output)
1171    *  2. Compaction atomically moves the temporary file under region directory
1172    *  3. Compaction appends a WAL edit containing the compaction input and output files.
1173    *  Forces sync on WAL.
1174    *  4. Compaction deletes the input files from the region directory.
1175    *
1176    * Failure conditions are handled like this:
1177    *  - If RS fails before 2, compaction wont complete. Even if RS lives on and finishes
1178    *  the compaction later, it will only write the new data file to the region directory.
1179    *  Since we already have this data, this will be idempotent but we will have a redundant
1180    *  copy of the data.
1181    *  - If RS fails between 2 and 3, the region will have a redundant copy of the data. The
1182    *  RS that failed won't be able to finish snyc() for WAL because of lease recovery in WAL.
1183    *  - If RS fails after 3, the region region server who opens the region will pick up the
1184    *  the compaction marker from the WAL and replay it by removing the compaction input files.
1185    *  Failed RS can also attempt to delete those files, but the operation will be idempotent
1186    *
1187    * See HBASE-2231 for details.
1188    *
1189    * @param compaction compaction details obtained from requestCompaction()
1190    * @throws IOException
1191    * @return Storefile we compacted into or null if we failed or opted out early.
1192    */
1193   @Override
1194   public List<StoreFile> compact(CompactionContext compaction,
1195       CompactionThroughputController throughputController) throws IOException {
1196     return compact(compaction, throughputController, null);
1197   }
1198 
1199   @Override
1200   public List<StoreFile> compact(CompactionContext compaction,
1201     CompactionThroughputController throughputController, User user) throws IOException {
1202     assert compaction != null;
1203     List<StoreFile> sfs = null;
1204     CompactionRequest cr = compaction.getRequest();
1205     try {
1206       // Do all sanity checking in here if we have a valid CompactionRequest
1207       // because we need to clean up after it on the way out in a finally
1208       // block below
1209       long compactionStartTime = EnvironmentEdgeManager.currentTime();
1210       assert compaction.hasSelection();
1211       Collection<StoreFile> filesToCompact = cr.getFiles();
1212       assert !filesToCompact.isEmpty();
1213       synchronized (filesCompacting) {
1214         // sanity check: we're compacting files that this store knows about
1215         // TODO: change this to LOG.error() after more debugging
1216         Preconditions.checkArgument(filesCompacting.containsAll(filesToCompact));
1217       }
1218 
1219       // Ready to go. Have list of files to compact.
1220       LOG.info("Starting compaction of " + filesToCompact.size() + " file(s) in "
1221           + this + " of " + this.getRegionInfo().getRegionNameAsString()
1222           + " into tmpdir=" + fs.getTempDir() + ", totalSize="
1223           + TraditionalBinaryPrefix.long2String(cr.getSize(), "", 1));
1224 
1225       // Commence the compaction.
1226       List<Path> newFiles = compaction.compact(throughputController, user);
1227 
1228       // TODO: get rid of this!
1229       if (!this.conf.getBoolean("hbase.hstore.compaction.complete", true)) {
1230         LOG.warn("hbase.hstore.compaction.complete is set to false");
1231         sfs = new ArrayList<StoreFile>(newFiles.size());
1232         final boolean evictOnClose =
1233             cacheConf != null? cacheConf.shouldEvictOnClose(): true;
1234         for (Path newFile : newFiles) {
1235           // Create storefile around what we wrote with a reader on it.
1236           StoreFile sf = createStoreFileAndReader(newFile);
1237           sf.closeReader(evictOnClose);
1238           sfs.add(sf);
1239         }
1240         return sfs;
1241       }
1242       // Do the steps necessary to complete the compaction.
1243       sfs = moveCompatedFilesIntoPlace(cr, newFiles, user);
1244       writeCompactionWalRecord(filesToCompact, sfs);
1245       replaceStoreFiles(filesToCompact, sfs);
1246       if (cr.isMajor()) {
1247         majorCompactedCellsCount += getCompactionProgress().totalCompactingKVs;
1248         majorCompactedCellsSize += getCompactionProgress().totalCompactedSize;
1249       } else {
1250         compactedCellsCount += getCompactionProgress().totalCompactingKVs;
1251         compactedCellsSize += getCompactionProgress().totalCompactedSize;
1252       }
1253       // At this point the store will use new files for all new scanners.
1254       completeCompaction(filesToCompact, true); // Archive old files & update store size.
1255 
1256       logCompactionEndMessage(cr, sfs, compactionStartTime);
1257       return sfs;
1258     } finally {
1259       finishCompactionRequest(cr);
1260     }
1261   }
1262 
1263   private List<StoreFile> moveCompatedFilesIntoPlace(
1264       final CompactionRequest cr, List<Path> newFiles, User user) throws IOException {
1265     List<StoreFile> sfs = new ArrayList<StoreFile>(newFiles.size());
1266     for (Path newFile : newFiles) {
1267       assert newFile != null;
1268       final StoreFile sf = moveFileIntoPlace(newFile);
1269       if (this.getCoprocessorHost() != null) {
1270         final Store thisStore = this;
1271         if (user == null) {
1272           getCoprocessorHost().postCompact(thisStore, sf, cr);
1273         } else {
1274           try {
1275             user.getUGI().doAs(new PrivilegedExceptionAction<Void>() {
1276               @Override
1277               public Void run() throws Exception {
1278                 getCoprocessorHost().postCompact(thisStore, sf, cr);
1279                 return null;
1280               }
1281             });
1282           } catch (InterruptedException ie) {
1283             InterruptedIOException iioe = new InterruptedIOException();
1284             iioe.initCause(ie);
1285             throw iioe;
1286           }
1287         }
1288       }
1289       assert sf != null;
1290       sfs.add(sf);
1291     }
1292     return sfs;
1293   }
1294 
1295   // Package-visible for tests
1296   StoreFile moveFileIntoPlace(final Path newFile) throws IOException {
1297     validateStoreFile(newFile);
1298     // Move the file into the right spot
1299     Path destPath = fs.commitStoreFile(getColumnFamilyName(), newFile);
1300     return createStoreFileAndReader(destPath);
1301   }
1302 
1303   /**
1304    * Writes the compaction WAL record.
1305    * @param filesCompacted Files compacted (input).
1306    * @param newFiles Files from compaction.
1307    */
1308   private void writeCompactionWalRecord(Collection<StoreFile> filesCompacted,
1309       Collection<StoreFile> newFiles) throws IOException {
1310     if (region.getWAL() == null) return;
1311     List<Path> inputPaths = new ArrayList<Path>(filesCompacted.size());
1312     for (StoreFile f : filesCompacted) {
1313       inputPaths.add(f.getPath());
1314     }
1315     List<Path> outputPaths = new ArrayList<Path>(newFiles.size());
1316     for (StoreFile f : newFiles) {
1317       outputPaths.add(f.getPath());
1318     }
1319     HRegionInfo info = this.region.getRegionInfo();
1320     CompactionDescriptor compactionDescriptor = ProtobufUtil.toCompactionDescriptor(info,
1321         family.getName(), inputPaths, outputPaths, fs.getStoreDir(getFamily().getNameAsString()));
1322     WALUtil.writeCompactionMarker(region.getWAL(), this.region.getTableDesc(),
1323         this.region.getRegionInfo(), compactionDescriptor, region.getMVCC());
1324   }
1325 
1326   @VisibleForTesting
1327   void replaceStoreFiles(final Collection<StoreFile> compactedFiles,
1328       final Collection<StoreFile> result) throws IOException {
1329     this.lock.writeLock().lock();
1330     try {
1331       this.storeEngine.getStoreFileManager().addCompactionResults(compactedFiles, result);
1332       synchronized (filesCompacting) {
1333         filesCompacting.removeAll(compactedFiles);
1334       }
1335     } finally {
1336       this.lock.writeLock().unlock();
1337     }
1338   }
1339 
1340   /**
1341    * Log a very elaborate compaction completion message.
1342    * @param cr Request.
1343    * @param sfs Resulting files.
1344    * @param compactionStartTime Start time.
1345    */
1346   private void logCompactionEndMessage(
1347       CompactionRequest cr, List<StoreFile> sfs, long compactionStartTime) {
1348     long now = EnvironmentEdgeManager.currentTime();
1349     StringBuilder message = new StringBuilder(
1350       "Completed" + (cr.isMajor() ? " major" : "") + " compaction of "
1351       + cr.getFiles().size() + (cr.isAllFiles() ? " (all)" : "") + " file(s) in "
1352       + this + " of " + this.getRegionInfo().getRegionNameAsString() + " into ");
1353     if (sfs.isEmpty()) {
1354       message.append("none, ");
1355     } else {
1356       for (StoreFile sf: sfs) {
1357         message.append(sf.getPath().getName());
1358         message.append("(size=");
1359         message.append(TraditionalBinaryPrefix.long2String(sf.getReader().length(), "", 1));
1360         message.append("), ");
1361       }
1362     }
1363     message.append("total size for store is ")
1364       .append(StringUtils.TraditionalBinaryPrefix.long2String(storeSize, "", 1))
1365       .append(". This selection was in queue for ")
1366       .append(StringUtils.formatTimeDiff(compactionStartTime, cr.getSelectionTime()))
1367       .append(", and took ").append(StringUtils.formatTimeDiff(now, compactionStartTime))
1368       .append(" to execute.");
1369     LOG.info(message.toString());
1370     if (LOG.isTraceEnabled()) {
1371       int fileCount = storeEngine.getStoreFileManager().getStorefileCount();
1372       long resultSize = 0;
1373       for (StoreFile sf : sfs) {
1374         resultSize += sf.getReader().length();
1375       }
1376       String traceMessage = "COMPACTION start,end,size out,files in,files out,store size,"
1377         + "store files [" + compactionStartTime + "," + now + "," + resultSize + ","
1378           + cr.getFiles().size() + "," + sfs.size() + "," +  storeSize + "," + fileCount + "]";
1379       LOG.trace(traceMessage);
1380     }
1381   }
1382 
1383   /**
1384    * Call to complete a compaction. Its for the case where we find in the WAL a compaction
1385    * that was not finished.  We could find one recovering a WAL after a regionserver crash.
1386    * See HBASE-2231.
1387    * @param compaction
1388    */
1389   @Override
1390   public void replayCompactionMarker(CompactionDescriptor compaction,
1391       boolean pickCompactionFiles, boolean removeFiles)
1392       throws IOException {
1393     LOG.debug("Completing compaction from the WAL marker");
1394     List<String> compactionInputs = compaction.getCompactionInputList();
1395     List<String> compactionOutputs = Lists.newArrayList(compaction.getCompactionOutputList());
1396 
1397     // The Compaction Marker is written after the compaction is completed,
1398     // and the files moved into the region/family folder.
1399     //
1400     // If we crash after the entry is written, we may not have removed the
1401     // input files, but the output file is present.
1402     // (The unremoved input files will be removed by this function)
1403     //
1404     // If we scan the directory and the file is not present, it can mean that:
1405     //   - The file was manually removed by the user
1406     //   - The file was removed as consequence of subsequent compaction
1407     // so, we can't do anything with the "compaction output list" because those
1408     // files have already been loaded when opening the region (by virtue of
1409     // being in the store's folder) or they may be missing due to a compaction.
1410 
1411     String familyName = this.getColumnFamilyName();
1412     List<String> inputFiles = new ArrayList<String>(compactionInputs.size());
1413     for (String compactionInput : compactionInputs) {
1414       Path inputPath = fs.getStoreFilePath(familyName, compactionInput);
1415       inputFiles.add(inputPath.getName());
1416     }
1417 
1418     //some of the input files might already be deleted
1419     List<StoreFile> inputStoreFiles = new ArrayList<StoreFile>(compactionInputs.size());
1420     for (StoreFile sf : this.getStorefiles()) {
1421       if (inputFiles.contains(sf.getPath().getName())) {
1422         inputStoreFiles.add(sf);
1423       }
1424     }
1425 
1426     // check whether we need to pick up the new files
1427     List<StoreFile> outputStoreFiles = new ArrayList<StoreFile>(compactionOutputs.size());
1428 
1429     if (pickCompactionFiles) {
1430       for (StoreFile sf : this.getStorefiles()) {
1431         compactionOutputs.remove(sf.getPath().getName());
1432       }
1433       for (String compactionOutput : compactionOutputs) {
1434         StoreFileInfo storeFileInfo = fs.getStoreFileInfo(getColumnFamilyName(), compactionOutput);
1435         StoreFile storeFile = createStoreFileAndReader(storeFileInfo);
1436         outputStoreFiles.add(storeFile);
1437       }
1438     }
1439 
1440     if (!inputStoreFiles.isEmpty() || !outputStoreFiles.isEmpty()) {
1441       LOG.info("Replaying compaction marker, replacing input files: " +
1442           inputStoreFiles + " with output files : " + outputStoreFiles);
1443       this.replaceStoreFiles(inputStoreFiles, outputStoreFiles);
1444       this.completeCompaction(inputStoreFiles, removeFiles);
1445     }
1446   }
1447 
1448   /**
1449    * This method tries to compact N recent files for testing.
1450    * Note that because compacting "recent" files only makes sense for some policies,
1451    * e.g. the default one, it assumes default policy is used. It doesn't use policy,
1452    * but instead makes a compaction candidate list by itself.
1453    * @param N Number of files.
1454    */
1455   public void compactRecentForTestingAssumingDefaultPolicy(int N) throws IOException {
1456     List<StoreFile> filesToCompact;
1457     boolean isMajor;
1458 
1459     this.lock.readLock().lock();
1460     try {
1461       synchronized (filesCompacting) {
1462         filesToCompact = Lists.newArrayList(storeEngine.getStoreFileManager().getStorefiles());
1463         if (!filesCompacting.isEmpty()) {
1464           // exclude all files older than the newest file we're currently
1465           // compacting. this allows us to preserve contiguity (HBASE-2856)
1466           StoreFile last = filesCompacting.get(filesCompacting.size() - 1);
1467           int idx = filesToCompact.indexOf(last);
1468           Preconditions.checkArgument(idx != -1);
1469           filesToCompact.subList(0, idx + 1).clear();
1470         }
1471         int count = filesToCompact.size();
1472         if (N > count) {
1473           throw new RuntimeException("Not enough files");
1474         }
1475 
1476         filesToCompact = filesToCompact.subList(count - N, count);
1477         isMajor = (filesToCompact.size() == storeEngine.getStoreFileManager().getStorefileCount());
1478         filesCompacting.addAll(filesToCompact);
1479         Collections.sort(filesCompacting, StoreFile.Comparators.SEQ_ID);
1480       }
1481     } finally {
1482       this.lock.readLock().unlock();
1483     }
1484 
1485     try {
1486       // Ready to go. Have list of files to compact.
1487       List<Path> newFiles = ((DefaultCompactor)this.storeEngine.getCompactor())
1488           .compactForTesting(filesToCompact, isMajor);
1489       for (Path newFile: newFiles) {
1490         // Move the compaction into place.
1491         StoreFile sf = moveFileIntoPlace(newFile);
1492         if (this.getCoprocessorHost() != null) {
1493           this.getCoprocessorHost().postCompact(this, sf, null);
1494         }
1495         replaceStoreFiles(filesToCompact, Lists.newArrayList(sf));
1496         completeCompaction(filesToCompact, true);
1497       }
1498     } finally {
1499       synchronized (filesCompacting) {
1500         filesCompacting.removeAll(filesToCompact);
1501       }
1502     }
1503   }
1504 
1505   @Override
1506   public boolean hasReferences() {
1507     return StoreUtils.hasReferences(this.storeEngine.getStoreFileManager().getStorefiles());
1508   }
1509 
1510   @Override
1511   public CompactionProgress getCompactionProgress() {
1512     return this.storeEngine.getCompactor().getProgress();
1513   }
1514 
1515   @Override
1516   public boolean isMajorCompaction() throws IOException {
1517     for (StoreFile sf : this.storeEngine.getStoreFileManager().getStorefiles()) {
1518       // TODO: what are these reader checks all over the place?
1519       if (sf.getReader() == null) {
1520         LOG.debug("StoreFile " + sf + " has null Reader");
1521         return false;
1522       }
1523     }
1524     return storeEngine.getCompactionPolicy().isMajorCompaction(
1525         this.storeEngine.getStoreFileManager().getStorefiles());
1526   }
1527 
1528   @Override
1529   public CompactionContext requestCompaction() throws IOException {
1530     return requestCompaction(Store.NO_PRIORITY, null);
1531   }
1532 
1533   @Override
1534   public CompactionContext requestCompaction(int priority, CompactionRequest baseRequest)
1535       throws IOException {
1536     return requestCompaction(priority, baseRequest, null);
1537   }
1538   @Override
1539   public CompactionContext requestCompaction(int priority, final CompactionRequest baseRequest,
1540       User user) throws IOException {
1541     // don't even select for compaction if writes are disabled
1542     if (!this.areWritesEnabled()) {
1543       return null;
1544     }
1545 
1546     // Before we do compaction, try to get rid of unneeded files to simplify things.
1547     removeUnneededFiles();
1548 
1549     final CompactionContext compaction = storeEngine.createCompaction();
1550     CompactionRequest request = null;
1551     this.lock.readLock().lock();
1552     try {
1553       synchronized (filesCompacting) {
1554         final Store thisStore = this;
1555         // First, see if coprocessor would want to override selection.
1556         if (this.getCoprocessorHost() != null) {
1557           final List<StoreFile> candidatesForCoproc = compaction.preSelect(this.filesCompacting);
1558           boolean override = false;
1559           if (user == null) {
1560             override = getCoprocessorHost().preCompactSelection(this, candidatesForCoproc,
1561               baseRequest);
1562           } else {
1563             try {
1564               override = user.getUGI().doAs(new PrivilegedExceptionAction<Boolean>() {
1565                 @Override
1566                 public Boolean run() throws Exception {
1567                   return getCoprocessorHost().preCompactSelection(thisStore, candidatesForCoproc,
1568                     baseRequest);
1569                 }
1570               });
1571             } catch (InterruptedException ie) {
1572               InterruptedIOException iioe = new InterruptedIOException();
1573               iioe.initCause(ie);
1574               throw iioe;
1575             }
1576           }
1577           if (override) {
1578             // Coprocessor is overriding normal file selection.
1579             compaction.forceSelect(new CompactionRequest(candidatesForCoproc));
1580           }
1581         }
1582 
1583         // Normal case - coprocessor is not overriding file selection.
1584         if (!compaction.hasSelection()) {
1585           boolean isUserCompaction = priority == Store.PRIORITY_USER;
1586           boolean mayUseOffPeak = offPeakHours.isOffPeakHour() &&
1587               offPeakCompactionTracker.compareAndSet(false, true);
1588           try {
1589             compaction.select(this.filesCompacting, isUserCompaction,
1590               mayUseOffPeak, forceMajor && filesCompacting.isEmpty());
1591           } catch (IOException e) {
1592             if (mayUseOffPeak) {
1593               offPeakCompactionTracker.set(false);
1594             }
1595             throw e;
1596           }
1597           assert compaction.hasSelection();
1598           if (mayUseOffPeak && !compaction.getRequest().isOffPeak()) {
1599             // Compaction policy doesn't want to take advantage of off-peak.
1600             offPeakCompactionTracker.set(false);
1601           }
1602         }
1603         if (this.getCoprocessorHost() != null) {
1604           if (user == null) {
1605             this.getCoprocessorHost().postCompactSelection(
1606               this, ImmutableList.copyOf(compaction.getRequest().getFiles()), baseRequest);
1607           } else {
1608             try {
1609               user.getUGI().doAs(new PrivilegedExceptionAction<Void>() {
1610                 @Override
1611                 public Void run() throws Exception {
1612                   getCoprocessorHost().postCompactSelection(
1613                     thisStore,ImmutableList.copyOf(compaction.getRequest().getFiles()),baseRequest);
1614                   return null;
1615                 }
1616               });
1617             } catch (InterruptedException ie) {
1618               InterruptedIOException iioe = new InterruptedIOException();
1619               iioe.initCause(ie);
1620               throw iioe;
1621             }
1622           }
1623         }
1624 
1625         // Selected files; see if we have a compaction with some custom base request.
1626         if (baseRequest != null) {
1627           // Update the request with what the system thinks the request should be;
1628           // its up to the request if it wants to listen.
1629           compaction.forceSelect(
1630               baseRequest.combineWith(compaction.getRequest()));
1631         }
1632         // Finally, we have the resulting files list. Check if we have any files at all.
1633         request = compaction.getRequest();
1634         final Collection<StoreFile> selectedFiles = request.getFiles();
1635         if (selectedFiles.isEmpty()) {
1636           return null;
1637         }
1638 
1639         addToCompactingFiles(selectedFiles);
1640 
1641         // If we're enqueuing a major, clear the force flag.
1642         this.forceMajor = this.forceMajor && !request.isMajor();
1643 
1644         // Set common request properties.
1645         // Set priority, either override value supplied by caller or from store.
1646         request.setPriority((priority != Store.NO_PRIORITY) ? priority : getCompactPriority());
1647         request.setDescription(getRegionInfo().getRegionNameAsString(), getColumnFamilyName());
1648       }
1649     } finally {
1650       this.lock.readLock().unlock();
1651     }
1652 
1653     LOG.debug(getRegionInfo().getEncodedName() + " - "  + getColumnFamilyName()
1654         + ": Initiating " + (request.isMajor() ? "major" : "minor") + " compaction"
1655         + (request.isAllFiles() ? " (all files)" : ""));
1656     this.region.reportCompactionRequestStart(request.isMajor());
1657     return compaction;
1658   }
1659 
1660   /** Adds the files to compacting files. filesCompacting must be locked. */
1661   private void addToCompactingFiles(final Collection<StoreFile> filesToAdd) {
1662     if (filesToAdd == null) return;
1663     // Check that we do not try to compact the same StoreFile twice.
1664     if (!Collections.disjoint(filesCompacting, filesToAdd)) {
1665       Preconditions.checkArgument(false, "%s overlaps with %s", filesToAdd, filesCompacting);
1666     }
1667     filesCompacting.addAll(filesToAdd);
1668     Collections.sort(filesCompacting, StoreFile.Comparators.SEQ_ID);
1669   }
1670 
1671   private void removeUnneededFiles() throws IOException {
1672     if (!conf.getBoolean("hbase.store.delete.expired.storefile", true)) return;
1673     if (getFamily().getMinVersions() > 0) {
1674       LOG.debug("Skipping expired store file removal due to min version being " +
1675           getFamily().getMinVersions());
1676       return;
1677     }
1678     this.lock.readLock().lock();
1679     Collection<StoreFile> delSfs = null;
1680     try {
1681       synchronized (filesCompacting) {
1682         long cfTtl = getStoreFileTtl();
1683         if (cfTtl != Long.MAX_VALUE) {
1684           delSfs = storeEngine.getStoreFileManager().getUnneededFiles(
1685               EnvironmentEdgeManager.currentTime() - cfTtl, filesCompacting);
1686           addToCompactingFiles(delSfs);
1687         }
1688       }
1689     } finally {
1690       this.lock.readLock().unlock();
1691     }
1692     if (delSfs == null || delSfs.isEmpty()) return;
1693 
1694     Collection<StoreFile> newFiles = new ArrayList<StoreFile>(); // No new files.
1695     writeCompactionWalRecord(delSfs, newFiles);
1696     replaceStoreFiles(delSfs, newFiles);
1697     completeCompaction(delSfs);
1698     LOG.info("Completed removal of " + delSfs.size() + " unnecessary (expired) file(s) in "
1699         + this + " of " + this.getRegionInfo().getRegionNameAsString()
1700         + "; total size for store is " + TraditionalBinaryPrefix.long2String(storeSize, "", 1));
1701   }
1702 
1703   @Override
1704   public void cancelRequestedCompaction(CompactionContext compaction) {
1705     finishCompactionRequest(compaction.getRequest());
1706   }
1707 
1708   private void finishCompactionRequest(CompactionRequest cr) {
1709     this.region.reportCompactionRequestEnd(cr.isMajor(), cr.getFiles().size(), cr.getSize());
1710     if (cr.isOffPeak()) {
1711       offPeakCompactionTracker.set(false);
1712       cr.setOffPeak(false);
1713     }
1714     synchronized (filesCompacting) {
1715       filesCompacting.removeAll(cr.getFiles());
1716     }
1717   }
1718 
1719   /**
1720    * Validates a store file by opening and closing it. In HFileV2 this should
1721    * not be an expensive operation.
1722    *
1723    * @param path the path to the store file
1724    */
1725   private void validateStoreFile(Path path)
1726       throws IOException {
1727     StoreFile storeFile = null;
1728     try {
1729       storeFile = createStoreFileAndReader(path);
1730     } catch (IOException e) {
1731       LOG.error("Failed to open store file : " + path
1732           + ", keeping it in tmp location", e);
1733       throw e;
1734     } finally {
1735       if (storeFile != null) {
1736         storeFile.closeReader(false);
1737       }
1738     }
1739   }
1740 
1741   /**
1742    * <p>It works by processing a compaction that's been written to disk.
1743    *
1744    * <p>It is usually invoked at the end of a compaction, but might also be
1745    * invoked at HStore startup, if the prior execution died midway through.
1746    *
1747    * <p>Moving the compacted TreeMap into place means:
1748    * <pre>
1749    * 1) Unload all replaced StoreFile, close and collect list to delete.
1750    * 2) Compute new store size
1751    * </pre>
1752    *
1753    * @param compactedFiles list of files that were compacted
1754    */
1755   @VisibleForTesting
1756   protected void completeCompaction(final Collection<StoreFile> compactedFiles)
1757     throws IOException {
1758     completeCompaction(compactedFiles, true);
1759   }
1760 
1761 
1762   /**
1763    * <p>It works by processing a compaction that's been written to disk.
1764    *
1765    * <p>It is usually invoked at the end of a compaction, but might also be
1766    * invoked at HStore startup, if the prior execution died midway through.
1767    *
1768    * <p>Moving the compacted TreeMap into place means:
1769    * <pre>
1770    * 1) Unload all replaced StoreFile, close and collect list to delete.
1771    * 2) Compute new store size
1772    * </pre>
1773    *
1774    * @param compactedFiles list of files that were compacted
1775    */
1776   @VisibleForTesting
1777   protected void completeCompaction(final Collection<StoreFile> compactedFiles, boolean removeFiles)
1778       throws IOException {
1779     try {
1780       // Do not delete old store files until we have sent out notification of
1781       // change in case old files are still being accessed by outstanding scanners.
1782       // Don't do this under writeLock; see HBASE-4485 for a possible deadlock
1783       // scenario that could have happened if continue to hold the lock.
1784       notifyChangedReadersObservers();
1785       // At this point the store will use new files for all scanners.
1786 
1787       // let the archive util decide if we should archive or delete the files
1788       LOG.debug("Removing store files after compaction...");
1789       boolean evictOnClose = 
1790           cacheConf != null? cacheConf.shouldEvictOnClose(): true; 
1791       for (StoreFile compactedFile : compactedFiles) {
1792         compactedFile.closeReader(evictOnClose);
1793       }
1794       if (removeFiles) {
1795         this.fs.removeStoreFiles(this.getColumnFamilyName(), compactedFiles);
1796       }
1797     } catch (IOException e) {
1798       e = RemoteExceptionHandler.checkIOException(e);
1799       LOG.error("Failed removing compacted files in " + this +
1800         ". Files we were trying to remove are " + compactedFiles.toString() +
1801         "; some of them may have been already removed", e);
1802     }
1803 
1804     // 4. Compute new store size
1805     this.storeSize = 0L;
1806     this.totalUncompressedBytes = 0L;
1807     for (StoreFile hsf : this.storeEngine.getStoreFileManager().getStorefiles()) {
1808       StoreFile.Reader r = hsf.getReader();
1809       if (r == null) {
1810         LOG.warn("StoreFile " + hsf + " has a null Reader");
1811         continue;
1812       }
1813       this.storeSize += r.length();
1814       this.totalUncompressedBytes += r.getTotalUncompressedBytes();
1815     }
1816   }
1817 
1818   /*
1819    * @param wantedVersions How many versions were asked for.
1820    * @return wantedVersions or this families' {@link HConstants#VERSIONS}.
1821    */
1822   int versionsToReturn(final int wantedVersions) {
1823     if (wantedVersions <= 0) {
1824       throw new IllegalArgumentException("Number of versions must be > 0");
1825     }
1826     // Make sure we do not return more than maximum versions for this store.
1827     int maxVersions = this.family.getMaxVersions();
1828     return wantedVersions > maxVersions ? maxVersions: wantedVersions;
1829   }
1830 
1831   /**
1832    * @param cell
1833    * @param oldestTimestamp
1834    * @return true if the cell is expired
1835    */
1836   static boolean isCellTTLExpired(final Cell cell, final long oldestTimestamp, final long now) {
1837     // Do not create an Iterator or Tag objects unless the cell actually has tags.
1838     // TODO: This check for tags is really expensive. We decode an int for key and value. Costs.
1839     if (cell.getTagsLength() > 0) {
1840       // Look for a TTL tag first. Use it instead of the family setting if
1841       // found. If a cell has multiple TTLs, resolve the conflict by using the
1842       // first tag encountered.
1843       Iterator<Tag> i = CellUtil.tagsIterator(cell.getTagsArray(), cell.getTagsOffset(),
1844         cell.getTagsLength());
1845       while (i.hasNext()) {
1846         Tag t = i.next();
1847         if (TagType.TTL_TAG_TYPE == t.getType()) {
1848           // Unlike in schema cell TTLs are stored in milliseconds, no need
1849           // to convert
1850           long ts = cell.getTimestamp();
1851           assert t.getTagLength() == Bytes.SIZEOF_LONG;
1852           long ttl = Bytes.toLong(t.getBuffer(), t.getTagOffset(), t.getTagLength());
1853           if (ts + ttl < now) {
1854             return true;
1855           }
1856           // Per cell TTLs cannot extend lifetime beyond family settings, so
1857           // fall through to check that
1858           break;
1859         }
1860       }
1861     }
1862     return false;
1863   }
1864 
1865   @Override
1866   public Cell getRowKeyAtOrBefore(final byte[] row) throws IOException {
1867     // If minVersions is set, we will not ignore expired KVs.
1868     // As we're only looking for the latest matches, that should be OK.
1869     // With minVersions > 0 we guarantee that any KV that has any version
1870     // at all (expired or not) has at least one version that will not expire.
1871     // Note that this method used to take a KeyValue as arguments. KeyValue
1872     // can be back-dated, a row key cannot.
1873     long ttlToUse = scanInfo.getMinVersions() > 0 ? Long.MAX_VALUE : this.scanInfo.getTtl();
1874 
1875     KeyValue kv = new KeyValue(row, HConstants.LATEST_TIMESTAMP);
1876 
1877     GetClosestRowBeforeTracker state = new GetClosestRowBeforeTracker(
1878       this.comparator, kv, ttlToUse, this.getRegionInfo().isMetaRegion());
1879     this.lock.readLock().lock();
1880     try {
1881       // First go to the memstore.  Pick up deletes and candidates.
1882       this.memstore.getRowKeyAtOrBefore(state);
1883       // Check if match, if we got a candidate on the asked for 'kv' row.
1884       // Process each relevant store file. Run through from newest to oldest.
1885       Iterator<StoreFile> sfIterator = this.storeEngine.getStoreFileManager()
1886           .getCandidateFilesForRowKeyBefore(state.getTargetKey());
1887       while (sfIterator.hasNext()) {
1888         StoreFile sf = sfIterator.next();
1889         sfIterator.remove(); // Remove sf from iterator.
1890         boolean haveNewCandidate = rowAtOrBeforeFromStoreFile(sf, state);
1891         Cell candidate = state.getCandidate();
1892         // we have an optimization here which stops the search if we find exact match.
1893         if (candidate != null && CellUtil.matchingRow(candidate, row)) {
1894           return candidate;
1895         }
1896         if (haveNewCandidate) {
1897           sfIterator = this.storeEngine.getStoreFileManager().updateCandidateFilesForRowKeyBefore(
1898               sfIterator, state.getTargetKey(), candidate);
1899         }
1900       }
1901       return state.getCandidate();
1902     } finally {
1903       this.lock.readLock().unlock();
1904     }
1905   }
1906 
1907   /*
1908    * Check an individual MapFile for the row at or before a given row.
1909    * @param f
1910    * @param state
1911    * @throws IOException
1912    * @return True iff the candidate has been updated in the state.
1913    */
1914   private boolean rowAtOrBeforeFromStoreFile(final StoreFile f,
1915                                           final GetClosestRowBeforeTracker state)
1916       throws IOException {
1917     StoreFile.Reader r = f.getReader();
1918     if (r == null) {
1919       LOG.warn("StoreFile " + f + " has a null Reader");
1920       return false;
1921     }
1922     if (r.getEntries() == 0) {
1923       LOG.warn("StoreFile " + f + " is a empty store file");
1924       return false;
1925     }
1926     // TODO: Cache these keys rather than make each time?
1927     byte [] fk = r.getFirstKey();
1928     if (fk == null) return false;
1929     KeyValue firstKV = KeyValue.createKeyValueFromKey(fk, 0, fk.length);
1930     byte [] lk = r.getLastKey();
1931     KeyValue lastKV = KeyValue.createKeyValueFromKey(lk, 0, lk.length);
1932     KeyValue firstOnRow = state.getTargetKey();
1933     if (this.comparator.compareRows(lastKV, firstOnRow) < 0) {
1934       // If last key in file is not of the target table, no candidates in this
1935       // file.  Return.
1936       if (!state.isTargetTable(lastKV)) return false;
1937       // If the row we're looking for is past the end of file, set search key to
1938       // last key. TODO: Cache last and first key rather than make each time.
1939       firstOnRow = new KeyValue(lastKV.getRow(), HConstants.LATEST_TIMESTAMP);
1940     }
1941     // Get a scanner that caches blocks and that uses pread.
1942     HFileScanner scanner = r.getScanner(true, true, false);
1943     // Seek scanner.  If can't seek it, return.
1944     if (!seekToScanner(scanner, firstOnRow, firstKV)) return false;
1945     // If we found candidate on firstOnRow, just return. THIS WILL NEVER HAPPEN!
1946     // Unlikely that there'll be an instance of actual first row in table.
1947     if (walkForwardInSingleRow(scanner, firstOnRow, state)) return true;
1948     // If here, need to start backing up.
1949     while (scanner.seekBefore(firstOnRow.getBuffer(), firstOnRow.getKeyOffset(),
1950        firstOnRow.getKeyLength())) {
1951       Cell kv = scanner.getKeyValue();
1952       if (!state.isTargetTable(kv)) break;
1953       if (!state.isBetterCandidate(kv)) break;
1954       // Make new first on row.
1955       firstOnRow = new KeyValue(kv.getRow(), HConstants.LATEST_TIMESTAMP);
1956       // Seek scanner.  If can't seek it, break.
1957       if (!seekToScanner(scanner, firstOnRow, firstKV)) return false;
1958       // If we find something, break;
1959       if (walkForwardInSingleRow(scanner, firstOnRow, state)) return true;
1960     }
1961     return false;
1962   }
1963 
1964   /*
1965    * Seek the file scanner to firstOnRow or first entry in file.
1966    * @param scanner
1967    * @param firstOnRow
1968    * @param firstKV
1969    * @return True if we successfully seeked scanner.
1970    * @throws IOException
1971    */
1972   private boolean seekToScanner(final HFileScanner scanner,
1973                                 final KeyValue firstOnRow,
1974                                 final KeyValue firstKV)
1975       throws IOException {
1976     KeyValue kv = firstOnRow;
1977     // If firstOnRow < firstKV, set to firstKV
1978     if (this.comparator.compareRows(firstKV, firstOnRow) == 0) kv = firstKV;
1979     int result = scanner.seekTo(kv);
1980     return result != -1;
1981   }
1982 
1983   /*
1984    * When we come in here, we are probably at the kv just before we break into
1985    * the row that firstOnRow is on.  Usually need to increment one time to get
1986    * on to the row we are interested in.
1987    * @param scanner
1988    * @param firstOnRow
1989    * @param state
1990    * @return True we found a candidate.
1991    * @throws IOException
1992    */
1993   private boolean walkForwardInSingleRow(final HFileScanner scanner,
1994                                          final KeyValue firstOnRow,
1995                                          final GetClosestRowBeforeTracker state)
1996       throws IOException {
1997     boolean foundCandidate = false;
1998     do {
1999       Cell kv = scanner.getKeyValue();
2000       // If we are not in the row, skip.
2001       if (this.comparator.compareRows(kv, firstOnRow) < 0) continue;
2002       // Did we go beyond the target row? If so break.
2003       if (state.isTooFar(kv, firstOnRow)) break;
2004       if (state.isExpired(kv)) {
2005         continue;
2006       }
2007       // If we added something, this row is a contender. break.
2008       if (state.handle(kv)) {
2009         foundCandidate = true;
2010         break;
2011       }
2012     } while(scanner.next());
2013     return foundCandidate;
2014   }
2015 
2016   @Override
2017   public boolean canSplit() {
2018     this.lock.readLock().lock();
2019     try {
2020       // Not split-able if we find a reference store file present in the store.
2021       boolean result = !hasReferences();
2022       if (!result && LOG.isDebugEnabled()) {
2023         LOG.debug("Cannot split region due to reference files being there");
2024       }
2025       return result;
2026     } finally {
2027       this.lock.readLock().unlock();
2028     }
2029   }
2030 
2031   @Override
2032   public byte[] getSplitPoint() {
2033     this.lock.readLock().lock();
2034     try {
2035       // Should already be enforced by the split policy!
2036       assert !this.getRegionInfo().isMetaRegion();
2037       // Not split-able if we find a reference store file present in the store.
2038       if (hasReferences()) {
2039         return null;
2040       }
2041       return this.storeEngine.getStoreFileManager().getSplitPoint();
2042     } catch(IOException e) {
2043       LOG.warn("Failed getting store size for " + this, e);
2044     } finally {
2045       this.lock.readLock().unlock();
2046     }
2047     return null;
2048   }
2049 
2050   @Override
2051   public long getLastCompactSize() {
2052     return this.lastCompactSize;
2053   }
2054 
2055   @Override
2056   public long getSize() {
2057     return storeSize;
2058   }
2059 
2060   @Override
2061   public void triggerMajorCompaction() {
2062     this.forceMajor = true;
2063   }
2064 
2065 
2066   //////////////////////////////////////////////////////////////////////////////
2067   // File administration
2068   //////////////////////////////////////////////////////////////////////////////
2069 
2070   @Override
2071   public KeyValueScanner getScanner(Scan scan,
2072       final NavigableSet<byte []> targetCols, long readPt) throws IOException {
2073     lock.readLock().lock();
2074     try {
2075       KeyValueScanner scanner = null;
2076       if (this.getCoprocessorHost() != null) {
2077         scanner = this.getCoprocessorHost().preStoreScannerOpen(this, scan, targetCols);
2078       }
2079       if (scanner == null) {
2080         scanner = scan.isReversed() ? new ReversedStoreScanner(this,
2081             getScanInfo(), scan, targetCols, readPt) : new StoreScanner(this,
2082             getScanInfo(), scan, targetCols, readPt);
2083       }
2084       return scanner;
2085     } finally {
2086       lock.readLock().unlock();
2087     }
2088   }
2089 
2090   @Override
2091   public String toString() {
2092     return this.getColumnFamilyName();
2093   }
2094 
2095   @Override
2096   public int getStorefilesCount() {
2097     return this.storeEngine.getStoreFileManager().getStorefileCount();
2098   }
2099 
2100   @Override
2101   public long getStoreSizeUncompressed() {
2102     return this.totalUncompressedBytes;
2103   }
2104 
2105   @Override
2106   public long getStorefilesSize() {
2107     long size = 0;
2108     for (StoreFile s: this.storeEngine.getStoreFileManager().getStorefiles()) {
2109       StoreFile.Reader r = s.getReader();
2110       if (r == null) {
2111         LOG.warn("StoreFile " + s + " has a null Reader");
2112         continue;
2113       }
2114       size += r.length();
2115     }
2116     return size;
2117   }
2118 
2119   @Override
2120   public long getStorefilesIndexSize() {
2121     long size = 0;
2122     for (StoreFile s: this.storeEngine.getStoreFileManager().getStorefiles()) {
2123       StoreFile.Reader r = s.getReader();
2124       if (r == null) {
2125         LOG.warn("StoreFile " + s + " has a null Reader");
2126         continue;
2127       }
2128       size += r.indexSize();
2129     }
2130     return size;
2131   }
2132 
2133   @Override
2134   public long getTotalStaticIndexSize() {
2135     long size = 0;
2136     for (StoreFile s : this.storeEngine.getStoreFileManager().getStorefiles()) {
2137       StoreFile.Reader r = s.getReader();
2138       if (r == null) {
2139         continue;
2140       }
2141       size += r.getUncompressedDataIndexSize();
2142     }
2143     return size;
2144   }
2145 
2146   @Override
2147   public long getTotalStaticBloomSize() {
2148     long size = 0;
2149     for (StoreFile s : this.storeEngine.getStoreFileManager().getStorefiles()) {
2150       StoreFile.Reader r = s.getReader();
2151       if (r == null) {
2152         continue;
2153       }
2154       size += r.getTotalBloomSize();
2155     }
2156     return size;
2157   }
2158 
2159   @Override
2160   public long getMemStoreSize() {
2161     return this.memstore.size();
2162   }
2163 
2164   @Override
2165   public int getCompactPriority() {
2166     int priority = this.storeEngine.getStoreFileManager().getStoreCompactionPriority();
2167     if (priority == PRIORITY_USER) {
2168       LOG.warn("Compaction priority is USER despite there being no user compaction");
2169     }
2170     return priority;
2171   }
2172 
2173   @Override
2174   public boolean throttleCompaction(long compactionSize) {
2175     return storeEngine.getCompactionPolicy().throttleCompaction(compactionSize);
2176   }
2177 
2178   public HRegion getHRegion() {
2179     return this.region;
2180   }
2181 
2182   @Override
2183   public RegionCoprocessorHost getCoprocessorHost() {
2184     return this.region.getCoprocessorHost();
2185   }
2186 
2187   @Override
2188   public HRegionInfo getRegionInfo() {
2189     return this.fs.getRegionInfo();
2190   }
2191 
2192   @Override
2193   public boolean areWritesEnabled() {
2194     return this.region.areWritesEnabled();
2195   }
2196 
2197   @Override
2198   public long getSmallestReadPoint() {
2199     return this.region.getSmallestReadPoint();
2200   }
2201 
2202   /**
2203    * Used in tests. TODO: Remove
2204    *
2205    * Updates the value for the given row/family/qualifier. This function will always be seen as
2206    * atomic by other readers because it only puts a single KV to memstore. Thus no read/write
2207    * control necessary.
2208    * @param row row to update
2209    * @param f family to update
2210    * @param qualifier qualifier to update
2211    * @param newValue the new value to set into memstore
2212    * @return memstore size delta
2213    * @throws IOException
2214    */
2215   public long updateColumnValue(byte [] row, byte [] f,
2216                                 byte [] qualifier, long newValue)
2217       throws IOException {
2218 
2219     this.lock.readLock().lock();
2220     try {
2221       long now = EnvironmentEdgeManager.currentTime();
2222 
2223       return this.memstore.updateColumnValue(row,
2224           f,
2225           qualifier,
2226           newValue,
2227           now);
2228 
2229     } finally {
2230       this.lock.readLock().unlock();
2231     }
2232   }
2233 
2234   @Override
2235   public long upsert(Iterable<Cell> cells, long readpoint) throws IOException {
2236     this.lock.readLock().lock();
2237     try {
2238       return this.memstore.upsert(cells, readpoint);
2239     } finally {
2240       this.lock.readLock().unlock();
2241     }
2242   }
2243 
2244   @Override
2245   public StoreFlushContext createFlushContext(long cacheFlushId) {
2246     return new StoreFlusherImpl(cacheFlushId);
2247   }
2248 
2249   private final class StoreFlusherImpl implements StoreFlushContext {
2250 
2251     private long cacheFlushSeqNum;
2252     private MemStoreSnapshot snapshot;
2253     private List<Path> tempFiles;
2254     private List<Path> committedFiles;
2255     private long cacheFlushCount;
2256     private long cacheFlushSize;
2257 
2258     private StoreFlusherImpl(long cacheFlushSeqNum) {
2259       this.cacheFlushSeqNum = cacheFlushSeqNum;
2260     }
2261 
2262     /**
2263      * This is not thread safe. The caller should have a lock on the region or the store.
2264      * If necessary, the lock can be added with the patch provided in HBASE-10087
2265      */
2266     @Override
2267     public void prepare() {
2268       this.snapshot = memstore.snapshot();
2269       this.cacheFlushCount = snapshot.getCellsCount();
2270       this.cacheFlushSize = snapshot.getSize();
2271       committedFiles = new ArrayList<Path>(1);
2272     }
2273 
2274     @Override
2275     public void flushCache(MonitoredTask status) throws IOException {
2276       tempFiles = HStore.this.flushCache(cacheFlushSeqNum, snapshot, status);
2277     }
2278 
2279     @Override
2280     public boolean commit(MonitoredTask status) throws IOException {
2281       if (this.tempFiles == null || this.tempFiles.isEmpty()) {
2282         return false;
2283       }
2284       List<StoreFile> storeFiles = new ArrayList<StoreFile>(this.tempFiles.size());
2285       for (Path storeFilePath : tempFiles) {
2286         try {
2287           storeFiles.add(HStore.this.commitFile(storeFilePath, cacheFlushSeqNum, status));
2288         } catch (IOException ex) {
2289           LOG.error("Failed to commit store file " + storeFilePath, ex);
2290           // Try to delete the files we have committed before.
2291           for (StoreFile sf : storeFiles) {
2292             Path pathToDelete = sf.getPath();
2293             try {
2294               sf.deleteReader();
2295             } catch (IOException deleteEx) {
2296               LOG.fatal("Failed to delete store file we committed, halting " + pathToDelete, ex);
2297               Runtime.getRuntime().halt(1);
2298             }
2299           }
2300           throw new IOException("Failed to commit the flush", ex);
2301         }
2302       }
2303 
2304       for (StoreFile sf : storeFiles) {
2305         if (HStore.this.getCoprocessorHost() != null) {
2306           HStore.this.getCoprocessorHost().postFlush(HStore.this, sf);
2307         }
2308         committedFiles.add(sf.getPath());
2309       }
2310 
2311       HStore.this.flushedCellsCount += cacheFlushCount;
2312       HStore.this.flushedCellsSize += cacheFlushSize;
2313 
2314       // Add new file to store files.  Clear snapshot too while we have the Store write lock.
2315       return HStore.this.updateStorefiles(storeFiles, snapshot.getId());
2316     }
2317 
2318     @Override
2319     public List<Path> getCommittedFiles() {
2320       return committedFiles;
2321     }
2322 
2323     /**
2324      * Similar to commit, but called in secondary region replicas for replaying the
2325      * flush cache from primary region. Adds the new files to the store, and drops the
2326      * snapshot depending on dropMemstoreSnapshot argument.
2327      * @param fileNames names of the flushed files
2328      * @param dropMemstoreSnapshot whether to drop the prepared memstore snapshot
2329      * @throws IOException
2330      */
2331     @Override
2332     public void replayFlush(List<String> fileNames, boolean dropMemstoreSnapshot)
2333         throws IOException {
2334       List<StoreFile> storeFiles = new ArrayList<StoreFile>(fileNames.size());
2335       for (String file : fileNames) {
2336         // open the file as a store file (hfile link, etc)
2337         StoreFileInfo storeFileInfo = fs.getStoreFileInfo(getColumnFamilyName(), file);
2338         StoreFile storeFile = createStoreFileAndReader(storeFileInfo);
2339         storeFiles.add(storeFile);
2340         HStore.this.storeSize += storeFile.getReader().length();
2341         HStore.this.totalUncompressedBytes += storeFile.getReader().getTotalUncompressedBytes();
2342         if (LOG.isInfoEnabled()) {
2343           LOG.info("Region: " + HStore.this.getRegionInfo().getEncodedName() +
2344             " added " + storeFile + ", entries=" + storeFile.getReader().getEntries() +
2345             ", sequenceid=" +  + storeFile.getReader().getSequenceID() +
2346             ", filesize=" + StringUtils.humanReadableInt(storeFile.getReader().length()));
2347         }
2348       }
2349 
2350       long snapshotId = -1; // -1 means do not drop
2351       if (dropMemstoreSnapshot && snapshot != null) {
2352         snapshotId = snapshot.getId();
2353       }
2354       HStore.this.updateStorefiles(storeFiles, snapshotId);
2355     }
2356 
2357     /**
2358      * Abort the snapshot preparation. Drops the snapshot if any.
2359      * @throws IOException
2360      */
2361     @Override
2362     public void abort() throws IOException {
2363       if (snapshot == null) {
2364         return;
2365       }
2366       HStore.this.updateStorefiles(new ArrayList<StoreFile>(0), snapshot.getId());
2367     }
2368   }
2369 
2370   @Override
2371   public boolean needsCompaction() {
2372     List<StoreFile> filesCompactingClone = null;
2373     synchronized (filesCompacting) {
2374       filesCompactingClone = Lists.newArrayList(filesCompacting);
2375     }
2376     return this.storeEngine.needsCompaction(filesCompactingClone);
2377   }
2378 
2379   @Override
2380   public CacheConfig getCacheConfig() {
2381     return this.cacheConf;
2382   }
2383 
2384   public static final long FIXED_OVERHEAD =
2385       ClassSize.align(ClassSize.OBJECT + (16 * ClassSize.REFERENCE) + (10 * Bytes.SIZEOF_LONG)
2386               + (5 * Bytes.SIZEOF_INT) + (2 * Bytes.SIZEOF_BOOLEAN));
2387 
2388   public static final long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD
2389       + ClassSize.OBJECT + ClassSize.REENTRANT_LOCK
2390       + ClassSize.CONCURRENT_SKIPLISTMAP
2391       + ClassSize.CONCURRENT_SKIPLISTMAP_ENTRY + ClassSize.OBJECT
2392       + ScanInfo.FIXED_OVERHEAD);
2393 
2394   @Override
2395   public long heapSize() {
2396     return DEEP_OVERHEAD + this.memstore.heapSize();
2397   }
2398 
2399   @Override
2400   public KeyValue.KVComparator getComparator() {
2401     return comparator;
2402   }
2403 
2404   @Override
2405   public ScanInfo getScanInfo() {
2406     return scanInfo;
2407   }
2408 
2409   /**
2410    * Set scan info, used by test
2411    * @param scanInfo new scan info to use for test
2412    */
2413   void setScanInfo(ScanInfo scanInfo) {
2414     this.scanInfo = scanInfo;
2415   }
2416 
2417   @Override
2418   public boolean hasTooManyStoreFiles() {
2419     return getStorefilesCount() > this.blockingFileCount;
2420   }
2421 
2422   @Override
2423   public long getFlushedCellsCount() {
2424     return flushedCellsCount;
2425   }
2426 
2427   @Override
2428   public long getFlushedCellsSize() {
2429     return flushedCellsSize;
2430   }
2431 
2432   @Override
2433   public long getCompactedCellsCount() {
2434     return compactedCellsCount;
2435   }
2436 
2437   @Override
2438   public long getCompactedCellsSize() {
2439     return compactedCellsSize;
2440   }
2441 
2442   @Override
2443   public long getMajorCompactedCellsCount() {
2444     return majorCompactedCellsCount;
2445   }
2446 
2447   @Override
2448   public long getMajorCompactedCellsSize() {
2449     return majorCompactedCellsSize;
2450   }
2451 
2452   /**
2453    * Returns the StoreEngine that is backing this concrete implementation of Store.
2454    * @return Returns the {@link StoreEngine} object used internally inside this HStore object.
2455    */
2456   @VisibleForTesting
2457   public StoreEngine<?, ?, ?, ?> getStoreEngine() {
2458     return this.storeEngine;
2459   }
2460 
2461   protected OffPeakHours getOffPeakHours() {
2462     return this.offPeakHours;
2463   }
2464 
2465   /**
2466    * {@inheritDoc}
2467    */
2468   @Override
2469   public void onConfigurationChange(Configuration conf) {
2470     this.conf = new CompoundConfiguration()
2471             .add(conf)
2472             .addWritableMap(family.getValues());
2473     this.storeEngine.compactionPolicy.setConf(conf);
2474     this.offPeakHours = OffPeakHours.getInstance(conf);
2475   }
2476 
2477   /**
2478    * {@inheritDoc}
2479    */
2480   @Override
2481   public void registerChildren(ConfigurationManager manager) {
2482     // No children to register
2483   }
2484 
2485   /**
2486    * {@inheritDoc}
2487    */
2488   @Override
2489   public void deregisterChildren(ConfigurationManager manager) {
2490     // No children to deregister
2491   }
2492 
2493   @Override
2494   public double getCompactionPressure() {
2495     return storeEngine.getStoreFileManager().getCompactionPressure();
2496   }
2497 
2498   @Override
2499   public boolean isPrimaryReplicaStore() {
2500 	   return getRegionInfo().getReplicaId() == HRegionInfo.DEFAULT_REPLICA_ID;
2501   }
2502 }