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