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