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