View Javadoc

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