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