View Javadoc

1   /**
2    *
3    * Licensed to the Apache Software Foundation (ASF) under one
4    * or more contributor license agreements.  See the NOTICE file
5    * distributed with this work for additional information
6    * regarding copyright ownership.  The ASF licenses this file
7    * to you under the Apache License, Version 2.0 (the
8    * "License"); you may not use this file except in compliance
9    * with the License.  You may obtain a copy of the License at
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
13   * Unless required by applicable law or agreed to in writing, software
14   * distributed under the License is distributed on an "AS IS" BASIS,
15   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16   * See the License for the specific language governing permissions and
17   * limitations under the License.
18   */
19  package org.apache.hadoop.hbase.regionserver;
20  
21  import java.io.IOException;
22  import java.io.InterruptedIOException;
23  import java.net.InetSocketAddress;
24  import java.security.Key;
25  import java.security.KeyException;
26  import java.util.ArrayList;
27  import java.util.Collection;
28  import java.util.Collections;
29  import java.util.HashMap;
30  import java.util.HashSet;
31  import java.util.Iterator;
32  import java.util.List;
33  import java.util.NavigableSet;
34  import java.util.Set;
35  import java.util.concurrent.Callable;
36  import java.util.concurrent.CompletionService;
37  import java.util.concurrent.ConcurrentHashMap;
38  import java.util.concurrent.ExecutionException;
39  import java.util.concurrent.ExecutorCompletionService;
40  import java.util.concurrent.Future;
41  import java.util.concurrent.ThreadPoolExecutor;
42  import java.util.concurrent.atomic.AtomicBoolean;
43  import java.util.concurrent.locks.ReentrantReadWriteLock;
44  
45  import org.apache.commons.logging.Log;
46  import org.apache.commons.logging.LogFactory;
47  import org.apache.hadoop.classification.InterfaceAudience;
48  import org.apache.hadoop.conf.Configuration;
49  import org.apache.hadoop.fs.FileSystem;
50  import org.apache.hadoop.fs.Path;
51  import org.apache.hadoop.hbase.Cell;
52  import org.apache.hadoop.hbase.CellUtil;
53  import org.apache.hadoop.hbase.CompoundConfiguration;
54  import org.apache.hadoop.hbase.HColumnDescriptor;
55  import org.apache.hadoop.hbase.HConstants;
56  import org.apache.hadoop.hbase.HRegionInfo;
57  import org.apache.hadoop.hbase.KeyValue;
58  import org.apache.hadoop.hbase.KeyValueUtil;
59  import org.apache.hadoop.hbase.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       .addWritableMap(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 KeyValue kv) {
635     lock.readLock().lock();
636     try {
637        return this.memstore.add(kv);
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 KeyValue kv) {
665     lock.readLock().lock();
666     try {
667       this.memstore.rollback(kv);
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         Cell prevKV = null;
711         HFileScanner scanner = reader.getScanner(false, false, false);
712         scanner.seekTo();
713         do {
714           Cell kv = scanner.getKeyValue();
715           if (prevKV != null) {
716             if (Bytes.compareTo(prevKV.getRowArray(), prevKV.getRowOffset(),
717                 prevKV.getRowLength(), kv.getRowArray(), kv.getRowOffset(),
718                 kv.getRowLength()) > 0) {
719               throw new InvalidHFileException("Previous row is greater than"
720                   + " current row: path=" + srcPath + " previous="
721                   + Bytes.toStringBinary(KeyValueUtil.ensureKeyValue(prevKV).getKey()) + " current="
722                   + Bytes.toStringBinary(KeyValueUtil.ensureKeyValue(kv).getKey()));
723             }
724             if (Bytes.compareTo(prevKV.getFamilyArray(), prevKV.getFamilyOffset(),
725                 prevKV.getFamilyLength(), kv.getFamilyArray(), kv.getFamilyOffset(),
726                 kv.getFamilyLength()) != 0) {
727               throw new InvalidHFileException("Previous key had different"
728                   + " family compared to current key: path=" + srcPath
729                   + " previous=" + Bytes.toStringBinary(prevKV.getFamily())
730                   + " current=" + Bytes.toStringBinary(kv.getFamily()));
731             }
732           }
733           prevKV = kv;
734         } while (scanner.next());
735       }
736     } finally {
737       if (reader != null) reader.close();
738     }
739   }
740 
741   @Override
742   public void bulkLoadHFile(String srcPathStr, long seqNum) throws IOException {
743     Path srcPath = new Path(srcPathStr);
744     Path dstPath = fs.bulkLoadStoreFile(getColumnFamilyName(), srcPath, seqNum);
745 
746     StoreFile sf = createStoreFileAndReader(dstPath);
747 
748     StoreFile.Reader r = sf.getReader();
749     this.storeSize += r.length();
750     this.totalUncompressedBytes += r.getTotalUncompressedBytes();
751 
752     LOG.info("Loaded HFile " + srcPath + " into store '" + getColumnFamilyName() +
753         "' as " + dstPath + " - updating store file list.");
754 
755     // Append the new storefile into the list
756     this.lock.writeLock().lock();
757     try {
758       this.storeEngine.getStoreFileManager().insertNewFiles(Lists.newArrayList(sf));
759     } finally {
760       // We need the lock, as long as we are updating the storeFiles
761       // or changing the memstore. Let us release it before calling
762       // notifyChangeReadersObservers. See HBASE-4485 for a possible
763       // deadlock scenario that could have happened if continue to hold
764       // the lock.
765       this.lock.writeLock().unlock();
766     }
767     notifyChangedReadersObservers();
768     LOG.info("Successfully loaded store file " + srcPath
769         + " into store " + this + " (new location: " + dstPath + ")");
770     if (LOG.isTraceEnabled()) {
771       String traceMessage = "BULK LOAD time,size,store size,store files ["
772           + EnvironmentEdgeManager.currentTimeMillis() + "," + r.length() + "," + storeSize
773           + "," + storeEngine.getStoreFileManager().getStorefileCount() + "]";
774       LOG.trace(traceMessage);
775     }
776   }
777 
778   @Override
779   public ImmutableCollection<StoreFile> close() throws IOException {
780     this.lock.writeLock().lock();
781     try {
782       // Clear so metrics doesn't find them.
783       ImmutableCollection<StoreFile> result = storeEngine.getStoreFileManager().clearFiles();
784 
785       if (!result.isEmpty()) {
786         // initialize the thread pool for closing store files in parallel.
787         ThreadPoolExecutor storeFileCloserThreadPool = this.region
788             .getStoreFileOpenAndCloseThreadPool("StoreFileCloserThread-"
789                 + this.getColumnFamilyName());
790 
791         // close each store file in parallel
792         CompletionService<Void> completionService =
793           new ExecutorCompletionService<Void>(storeFileCloserThreadPool);
794         for (final StoreFile f : result) {
795           completionService.submit(new Callable<Void>() {
796             @Override
797             public Void call() throws IOException {
798               f.closeReader(true);
799               return null;
800             }
801           });
802         }
803 
804         IOException ioe = null;
805         try {
806           for (int i = 0; i < result.size(); i++) {
807             try {
808               Future<Void> future = completionService.take();
809               future.get();
810             } catch (InterruptedException e) {
811               if (ioe == null) {
812                 ioe = new InterruptedIOException();
813                 ioe.initCause(e);
814               }
815             } catch (ExecutionException e) {
816               if (ioe == null) ioe = new IOException(e.getCause());
817             }
818           }
819         } finally {
820           storeFileCloserThreadPool.shutdownNow();
821         }
822         if (ioe != null) throw ioe;
823       }
824       LOG.info("Closed " + this);
825       return result;
826     } finally {
827       this.lock.writeLock().unlock();
828     }
829   }
830 
831   /**
832    * Snapshot this stores memstore. Call before running
833    * {@link #flushCache(long, MemStoreSnapshot, MonitoredTask)}
834    *  so it has some work to do.
835    */
836   void snapshot() {
837     this.lock.writeLock().lock();
838     try {
839       this.memstore.snapshot();
840     } finally {
841       this.lock.writeLock().unlock();
842     }
843   }
844 
845   /**
846    * Write out current snapshot.  Presumes {@link #snapshot()} has been called
847    * previously.
848    * @param logCacheFlushId flush sequence number
849    * @param snapshot
850    * @param status
851    * @return The path name of the tmp file to which the store was flushed
852    * @throws IOException
853    */
854   protected List<Path> flushCache(final long logCacheFlushId, MemStoreSnapshot snapshot,
855       MonitoredTask status) throws IOException {
856     // If an exception happens flushing, we let it out without clearing
857     // the memstore snapshot.  The old snapshot will be returned when we say
858     // 'snapshot', the next time flush comes around.
859     // Retry after catching exception when flushing, otherwise server will abort
860     // itself
861     StoreFlusher flusher = storeEngine.getStoreFlusher();
862     IOException lastException = null;
863     for (int i = 0; i < flushRetriesNumber; i++) {
864       try {
865         List<Path> pathNames = flusher.flushSnapshot(snapshot, logCacheFlushId, status);
866         Path lastPathName = null;
867         try {
868           for (Path pathName : pathNames) {
869             lastPathName = pathName;
870             validateStoreFile(pathName);
871           }
872           return pathNames;
873         } catch (Exception e) {
874           LOG.warn("Failed validating store file " + lastPathName + ", retrying num=" + i, e);
875           if (e instanceof IOException) {
876             lastException = (IOException) e;
877           } else {
878             lastException = new IOException(e);
879           }
880         }
881       } catch (IOException e) {
882         LOG.warn("Failed flushing store file, retrying num=" + i, e);
883         lastException = e;
884       }
885       if (lastException != null && i < (flushRetriesNumber - 1)) {
886         try {
887           Thread.sleep(pauseTime);
888         } catch (InterruptedException e) {
889           IOException iie = new InterruptedIOException();
890           iie.initCause(e);
891           throw iie;
892         }
893       }
894     }
895     throw lastException;
896   }
897 
898   /*
899    * @param path The pathname of the tmp file into which the store was flushed
900    * @param logCacheFlushId
901    * @param status
902    * @return StoreFile created.
903    * @throws IOException
904    */
905   private StoreFile commitFile(final Path path, final long logCacheFlushId, MonitoredTask status)
906       throws IOException {
907     // Write-out finished successfully, move into the right spot
908     Path dstPath = fs.commitStoreFile(getColumnFamilyName(), path);
909 
910     status.setStatus("Flushing " + this + ": reopening flushed file");
911     StoreFile sf = createStoreFileAndReader(dstPath);
912 
913     StoreFile.Reader r = sf.getReader();
914     this.storeSize += r.length();
915     this.totalUncompressedBytes += r.getTotalUncompressedBytes();
916 
917     if (LOG.isInfoEnabled()) {
918       LOG.info("Added " + sf + ", entries=" + r.getEntries() +
919         ", sequenceid=" + logCacheFlushId +
920         ", filesize=" + StringUtils.humanReadableInt(r.length()));
921     }
922     return sf;
923   }
924 
925   /*
926    * @param maxKeyCount
927    * @param compression Compression algorithm to use
928    * @param isCompaction whether we are creating a new file in a compaction
929    * @param includesMVCCReadPoint - whether to include MVCC or not
930    * @param includesTag - includesTag or not
931    * @return Writer for a new StoreFile in the tmp dir.
932    */
933   @Override
934   public StoreFile.Writer createWriterInTmp(long maxKeyCount, Compression.Algorithm compression,
935       boolean isCompaction, boolean includeMVCCReadpoint, boolean includesTag)
936   throws IOException {
937     final CacheConfig writerCacheConf;
938     if (isCompaction) {
939       // Don't cache data on write on compactions.
940       writerCacheConf = new CacheConfig(cacheConf);
941       writerCacheConf.setCacheDataOnWrite(false);
942     } else {
943       writerCacheConf = cacheConf;
944     }
945     InetSocketAddress[] favoredNodes = null;
946     if (region.getRegionServerServices() != null) {
947       favoredNodes = region.getRegionServerServices().getFavoredNodesForRegion(
948           region.getRegionInfo().getEncodedName());
949     }
950     HFileContext hFileContext = createFileContext(compression, includeMVCCReadpoint, includesTag,
951       cryptoContext);
952     StoreFile.Writer w = new StoreFile.WriterBuilder(conf, writerCacheConf,
953         this.getFileSystem())
954             .withFilePath(fs.createTempName())
955             .withComparator(comparator)
956             .withBloomType(family.getBloomFilterType())
957             .withMaxKeyCount(maxKeyCount)
958             .withFavoredNodes(favoredNodes)
959             .withFileContext(hFileContext)
960             .build();
961     return w;
962   }
963 
964   private HFileContext createFileContext(Compression.Algorithm compression,
965       boolean includeMVCCReadpoint, boolean includesTag, Encryption.Context cryptoContext) {
966     if (compression == null) {
967       compression = HFile.DEFAULT_COMPRESSION_ALGORITHM;
968     }
969     HFileContext hFileContext = new HFileContextBuilder()
970                                 .withIncludesMvcc(includeMVCCReadpoint)
971                                 .withIncludesTags(includesTag)
972                                 .withCompression(compression)
973                                 .withCompressTags(family.shouldCompressTags())
974                                 .withChecksumType(checksumType)
975                                 .withBytesPerCheckSum(bytesPerChecksum)
976                                 .withBlockSize(blocksize)
977                                 .withHBaseCheckSum(true)
978                                 .withDataBlockEncoding(family.getDataBlockEncoding())
979                                 .withEncryptionContext(cryptoContext)
980                                 .build();
981     return hFileContext;
982   }
983 
984 
985   /*
986    * Change storeFiles adding into place the Reader produced by this new flush.
987    * @param sfs Store files
988    * @param snapshotId
989    * @throws IOException
990    * @return Whether compaction is required.
991    */
992   private boolean updateStorefiles(final List<StoreFile> sfs, final long snapshotId)
993       throws IOException {
994     this.lock.writeLock().lock();
995     try {
996       this.storeEngine.getStoreFileManager().insertNewFiles(sfs);
997       this.memstore.clearSnapshot(snapshotId);
998     } finally {
999       // We need the lock, as long as we are updating the storeFiles
1000       // or changing the memstore. Let us release it before calling
1001       // notifyChangeReadersObservers. See HBASE-4485 for a possible
1002       // deadlock scenario that could have happened if continue to hold
1003       // the lock.
1004       this.lock.writeLock().unlock();
1005     }
1006 
1007     // Tell listeners of the change in readers.
1008     notifyChangedReadersObservers();
1009 
1010     if (LOG.isTraceEnabled()) {
1011       long totalSize = 0;
1012       for (StoreFile sf : sfs) {
1013         totalSize += sf.getReader().length();
1014       }
1015       String traceMessage = "FLUSH time,count,size,store size,store files ["
1016           + EnvironmentEdgeManager.currentTimeMillis() + "," + sfs.size() + "," + totalSize
1017           + "," + storeSize + "," + storeEngine.getStoreFileManager().getStorefileCount() + "]";
1018       LOG.trace(traceMessage);
1019     }
1020     return needsCompaction();
1021   }
1022 
1023   /*
1024    * Notify all observers that set of Readers has changed.
1025    * @throws IOException
1026    */
1027   private void notifyChangedReadersObservers() throws IOException {
1028     for (ChangedReadersObserver o: this.changedReaderObservers) {
1029       o.updateReaders();
1030     }
1031   }
1032 
1033   /**
1034    * Get all scanners with no filtering based on TTL (that happens further down
1035    * the line).
1036    * @return all scanners for this store
1037    */
1038   @Override
1039   public List<KeyValueScanner> getScanners(boolean cacheBlocks, boolean isGet,
1040       boolean usePread, boolean isCompaction, ScanQueryMatcher matcher, byte[] startRow,
1041       byte[] stopRow, long readPt) throws IOException {
1042     Collection<StoreFile> storeFilesToScan;
1043     List<KeyValueScanner> memStoreScanners;
1044     this.lock.readLock().lock();
1045     try {
1046       storeFilesToScan =
1047           this.storeEngine.getStoreFileManager().getFilesForScanOrGet(isGet, startRow, stopRow);
1048       memStoreScanners = this.memstore.getScanners(readPt);
1049     } finally {
1050       this.lock.readLock().unlock();
1051     }
1052 
1053     // First the store file scanners
1054 
1055     // TODO this used to get the store files in descending order,
1056     // but now we get them in ascending order, which I think is
1057     // actually more correct, since memstore get put at the end.
1058     List<StoreFileScanner> sfScanners = StoreFileScanner
1059       .getScannersForStoreFiles(storeFilesToScan, cacheBlocks, usePread, isCompaction, matcher,
1060         readPt);
1061     List<KeyValueScanner> scanners =
1062       new ArrayList<KeyValueScanner>(sfScanners.size()+1);
1063     scanners.addAll(sfScanners);
1064     // Then the memstore scanners
1065     scanners.addAll(memStoreScanners);
1066     return scanners;
1067   }
1068 
1069   @Override
1070   public void addChangedReaderObserver(ChangedReadersObserver o) {
1071     this.changedReaderObservers.add(o);
1072   }
1073 
1074   @Override
1075   public void deleteChangedReaderObserver(ChangedReadersObserver o) {
1076     // We don't check if observer present; it may not be (legitimately)
1077     this.changedReaderObservers.remove(o);
1078   }
1079 
1080   //////////////////////////////////////////////////////////////////////////////
1081   // Compaction
1082   //////////////////////////////////////////////////////////////////////////////
1083 
1084   /**
1085    * Compact the StoreFiles.  This method may take some time, so the calling
1086    * thread must be able to block for long periods.
1087    *
1088    * <p>During this time, the Store can work as usual, getting values from
1089    * StoreFiles and writing new StoreFiles from the memstore.
1090    *
1091    * Existing StoreFiles are not destroyed until the new compacted StoreFile is
1092    * completely written-out to disk.
1093    *
1094    * <p>The compactLock prevents multiple simultaneous compactions.
1095    * The structureLock prevents us from interfering with other write operations.
1096    *
1097    * <p>We don't want to hold the structureLock for the whole time, as a compact()
1098    * can be lengthy and we want to allow cache-flushes during this period.
1099    *
1100    * <p> Compaction event should be idempotent, since there is no IO Fencing for
1101    * the region directory in hdfs. A region server might still try to complete the
1102    * compaction after it lost the region. That is why the following events are carefully
1103    * ordered for a compaction:
1104    *  1. Compaction writes new files under region/.tmp directory (compaction output)
1105    *  2. Compaction atomically moves the temporary file under region directory
1106    *  3. Compaction appends a WAL edit containing the compaction input and output files.
1107    *  Forces sync on WAL.
1108    *  4. Compaction deletes the input files from the region directory.
1109    *
1110    * Failure conditions are handled like this:
1111    *  - If RS fails before 2, compaction wont complete. Even if RS lives on and finishes
1112    *  the compaction later, it will only write the new data file to the region directory.
1113    *  Since we already have this data, this will be idempotent but we will have a redundant
1114    *  copy of the data.
1115    *  - If RS fails between 2 and 3, the region will have a redundant copy of the data. The
1116    *  RS that failed won't be able to finish snyc() for WAL because of lease recovery in WAL.
1117    *  - If RS fails after 3, the region region server who opens the region will pick up the
1118    *  the compaction marker from the WAL and replay it by removing the compaction input files.
1119    *  Failed RS can also attempt to delete those files, but the operation will be idempotent
1120    *
1121    * See HBASE-2231 for details.
1122    *
1123    * @param compaction compaction details obtained from requestCompaction()
1124    * @throws IOException
1125    * @return Storefile we compacted into or null if we failed or opted out early.
1126    */
1127   @Override
1128   public List<StoreFile> compact(CompactionContext compaction) throws IOException {
1129     assert compaction != null && compaction.hasSelection();
1130     CompactionRequest cr = compaction.getRequest();
1131     Collection<StoreFile> filesToCompact = cr.getFiles();
1132     assert !filesToCompact.isEmpty();
1133     synchronized (filesCompacting) {
1134       // sanity check: we're compacting files that this store knows about
1135       // TODO: change this to LOG.error() after more debugging
1136       Preconditions.checkArgument(filesCompacting.containsAll(filesToCompact));
1137     }
1138 
1139     // Ready to go. Have list of files to compact.
1140     LOG.info("Starting compaction of " + filesToCompact.size() + " file(s) in "
1141         + this + " of " + this.getRegionInfo().getRegionNameAsString()
1142         + " into tmpdir=" + fs.getTempDir() + ", totalSize="
1143         + StringUtils.humanReadableInt(cr.getSize()));
1144 
1145     long compactionStartTime = EnvironmentEdgeManager.currentTimeMillis();
1146     List<StoreFile> sfs = null;
1147     try {
1148       // Commence the compaction.
1149       List<Path> newFiles = compaction.compact();
1150 
1151       // TODO: get rid of this!
1152       if (!this.conf.getBoolean("hbase.hstore.compaction.complete", true)) {
1153         LOG.warn("hbase.hstore.compaction.complete is set to false");
1154         sfs = new ArrayList<StoreFile>(newFiles.size());
1155         for (Path newFile : newFiles) {
1156           // Create storefile around what we wrote with a reader on it.
1157           StoreFile sf = createStoreFileAndReader(newFile);
1158           sf.closeReader(true);
1159           sfs.add(sf);
1160         }
1161         return sfs;
1162       }
1163       // Do the steps necessary to complete the compaction.
1164       sfs = moveCompatedFilesIntoPlace(cr, newFiles);
1165       writeCompactionWalRecord(filesToCompact, sfs);
1166       replaceStoreFiles(filesToCompact, sfs);
1167       if (cr.isMajor()) {
1168         majorCompactedCellsCount += getCompactionProgress().totalCompactingKVs;
1169         majorCompactedCellsSize += getCompactionProgress().totalCompactedSize;
1170       } else {
1171         compactedCellsCount += getCompactionProgress().totalCompactingKVs;
1172         compactedCellsSize += getCompactionProgress().totalCompactedSize;
1173       }
1174       // At this point the store will use new files for all new scanners.
1175       completeCompaction(filesToCompact, true); // Archive old files & update store size.
1176     } finally {
1177       finishCompactionRequest(cr);
1178     }
1179     logCompactionEndMessage(cr, sfs, compactionStartTime);
1180     return sfs;
1181   }
1182 
1183   private List<StoreFile> moveCompatedFilesIntoPlace(
1184       CompactionRequest cr, List<Path> newFiles) throws IOException {
1185     List<StoreFile> sfs = new ArrayList<StoreFile>(newFiles.size());
1186     for (Path newFile : newFiles) {
1187       assert newFile != null;
1188       StoreFile sf = moveFileIntoPlace(newFile);
1189       if (this.getCoprocessorHost() != null) {
1190         this.getCoprocessorHost().postCompact(this, sf, cr);
1191       }
1192       assert sf != null;
1193       sfs.add(sf);
1194     }
1195     return sfs;
1196   }
1197 
1198   // Package-visible for tests
1199   StoreFile moveFileIntoPlace(final Path newFile) throws IOException {
1200     validateStoreFile(newFile);
1201     // Move the file into the right spot
1202     Path destPath = fs.commitStoreFile(getColumnFamilyName(), newFile);
1203     return createStoreFileAndReader(destPath);
1204   }
1205 
1206   /**
1207    * Writes the compaction WAL record.
1208    * @param filesCompacted Files compacted (input).
1209    * @param newFiles Files from compaction.
1210    */
1211   private void writeCompactionWalRecord(Collection<StoreFile> filesCompacted,
1212       Collection<StoreFile> newFiles) throws IOException {
1213     if (region.getLog() == null) return;
1214     List<Path> inputPaths = new ArrayList<Path>(filesCompacted.size());
1215     for (StoreFile f : filesCompacted) {
1216       inputPaths.add(f.getPath());
1217     }
1218     List<Path> outputPaths = new ArrayList<Path>(newFiles.size());
1219     for (StoreFile f : newFiles) {
1220       outputPaths.add(f.getPath());
1221     }
1222     HRegionInfo info = this.region.getRegionInfo();
1223     CompactionDescriptor compactionDescriptor = ProtobufUtil.toCompactionDescriptor(info,
1224         family.getName(), inputPaths, outputPaths, fs.getStoreDir(getFamily().getNameAsString()));
1225     HLogUtil.writeCompactionMarker(region.getLog(), this.region.getTableDesc(),
1226         this.region.getRegionInfo(), compactionDescriptor, this.region.getSequenceId());
1227   }
1228 
1229   @VisibleForTesting
1230   void replaceStoreFiles(final Collection<StoreFile> compactedFiles,
1231       final Collection<StoreFile> result) throws IOException {
1232     this.lock.writeLock().lock();
1233     try {
1234       this.storeEngine.getStoreFileManager().addCompactionResults(compactedFiles, result);
1235       filesCompacting.removeAll(compactedFiles); // safe bc: lock.writeLock();
1236     } finally {
1237       this.lock.writeLock().unlock();
1238     }
1239   }
1240 
1241   /**
1242    * Log a very elaborate compaction completion message.
1243    * @param cr Request.
1244    * @param sfs Resulting files.
1245    * @param compactionStartTime Start time.
1246    */
1247   private void logCompactionEndMessage(
1248       CompactionRequest cr, List<StoreFile> sfs, long compactionStartTime) {
1249     long now = EnvironmentEdgeManager.currentTimeMillis();
1250     StringBuilder message = new StringBuilder(
1251       "Completed" + (cr.isMajor() ? " major" : "") + " compaction of "
1252       + cr.getFiles().size() + (cr.isAllFiles() ? " (all)" : "") + " file(s) in "
1253       + this + " of " + this.getRegionInfo().getRegionNameAsString() + " into ");
1254     if (sfs.isEmpty()) {
1255       message.append("none, ");
1256     } else {
1257       for (StoreFile sf: sfs) {
1258         message.append(sf.getPath().getName());
1259         message.append("(size=");
1260         message.append(StringUtils.humanReadableInt(sf.getReader().length()));
1261         message.append("), ");
1262       }
1263     }
1264     message.append("total size for store is ")
1265       .append(StringUtils.humanReadableInt(storeSize))
1266       .append(". This selection was in queue for ")
1267       .append(StringUtils.formatTimeDiff(compactionStartTime, cr.getSelectionTime()))
1268       .append(", and took ").append(StringUtils.formatTimeDiff(now, compactionStartTime))
1269       .append(" to execute.");
1270     LOG.info(message.toString());
1271     if (LOG.isTraceEnabled()) {
1272       int fileCount = storeEngine.getStoreFileManager().getStorefileCount();
1273       long resultSize = 0;
1274       for (StoreFile sf : sfs) {
1275         resultSize += sf.getReader().length();
1276       }
1277       String traceMessage = "COMPACTION start,end,size out,files in,files out,store size,"
1278         + "store files [" + compactionStartTime + "," + now + "," + resultSize + ","
1279           + cr.getFiles().size() + "," + sfs.size() + "," +  storeSize + "," + fileCount + "]";
1280       LOG.trace(traceMessage);
1281     }
1282   }
1283 
1284   /**
1285    * Call to complete a compaction. Its for the case where we find in the WAL a compaction
1286    * that was not finished.  We could find one recovering a WAL after a regionserver crash.
1287    * See HBASE-2231.
1288    * @param compaction
1289    */
1290   @Override
1291   public void completeCompactionMarker(CompactionDescriptor compaction)
1292       throws IOException {
1293     LOG.debug("Completing compaction from the WAL marker");
1294     List<String> compactionInputs = compaction.getCompactionInputList();
1295 
1296     // The Compaction Marker is written after the compaction is completed,
1297     // and the files moved into the region/family folder.
1298     //
1299     // If we crash after the entry is written, we may not have removed the
1300     // input files, but the output file is present.
1301     // (The unremoved input files will be removed by this function)
1302     //
1303     // If we scan the directory and the file is not present, it can mean that:
1304     //   - The file was manually removed by the user
1305     //   - The file was removed as consequence of subsequent compaction
1306     // so, we can't do anything with the "compaction output list" because those
1307     // files have already been loaded when opening the region (by virtue of
1308     // being in the store's folder) or they may be missing due to a compaction.
1309 
1310     String familyName = this.getColumnFamilyName();
1311     List<Path> inputPaths = new ArrayList<Path>(compactionInputs.size());
1312     for (String compactionInput : compactionInputs) {
1313       Path inputPath = fs.getStoreFilePath(familyName, compactionInput);
1314       inputPaths.add(inputPath);
1315     }
1316 
1317     //some of the input files might already be deleted
1318     List<StoreFile> inputStoreFiles = new ArrayList<StoreFile>(compactionInputs.size());
1319     for (StoreFile sf : this.getStorefiles()) {
1320       if (inputPaths.contains(sf.getQualifiedPath())) {
1321         inputStoreFiles.add(sf);
1322       }
1323     }
1324 
1325     this.replaceStoreFiles(inputStoreFiles, Collections.EMPTY_LIST);
1326     this.completeCompaction(inputStoreFiles);
1327   }
1328 
1329   /**
1330    * This method tries to compact N recent files for testing.
1331    * Note that because compacting "recent" files only makes sense for some policies,
1332    * e.g. the default one, it assumes default policy is used. It doesn't use policy,
1333    * but instead makes a compaction candidate list by itself.
1334    * @param N Number of files.
1335    */
1336   public void compactRecentForTestingAssumingDefaultPolicy(int N) throws IOException {
1337     List<StoreFile> filesToCompact;
1338     boolean isMajor;
1339 
1340     this.lock.readLock().lock();
1341     try {
1342       synchronized (filesCompacting) {
1343         filesToCompact = Lists.newArrayList(storeEngine.getStoreFileManager().getStorefiles());
1344         if (!filesCompacting.isEmpty()) {
1345           // exclude all files older than the newest file we're currently
1346           // compacting. this allows us to preserve contiguity (HBASE-2856)
1347           StoreFile last = filesCompacting.get(filesCompacting.size() - 1);
1348           int idx = filesToCompact.indexOf(last);
1349           Preconditions.checkArgument(idx != -1);
1350           filesToCompact.subList(0, idx + 1).clear();
1351         }
1352         int count = filesToCompact.size();
1353         if (N > count) {
1354           throw new RuntimeException("Not enough files");
1355         }
1356 
1357         filesToCompact = filesToCompact.subList(count - N, count);
1358         isMajor = (filesToCompact.size() == storeEngine.getStoreFileManager().getStorefileCount());
1359         filesCompacting.addAll(filesToCompact);
1360         Collections.sort(filesCompacting, StoreFile.Comparators.SEQ_ID);
1361       }
1362     } finally {
1363       this.lock.readLock().unlock();
1364     }
1365 
1366     try {
1367       // Ready to go. Have list of files to compact.
1368       List<Path> newFiles = ((DefaultCompactor)this.storeEngine.getCompactor())
1369           .compactForTesting(filesToCompact, isMajor);
1370       for (Path newFile: newFiles) {
1371         // Move the compaction into place.
1372         StoreFile sf = moveFileIntoPlace(newFile);
1373         if (this.getCoprocessorHost() != null) {
1374           this.getCoprocessorHost().postCompact(this, sf, null);
1375         }
1376         replaceStoreFiles(filesToCompact, Lists.newArrayList(sf));
1377         completeCompaction(filesToCompact, true);
1378       }
1379     } finally {
1380       synchronized (filesCompacting) {
1381         filesCompacting.removeAll(filesToCompact);
1382       }
1383     }
1384   }
1385 
1386   @Override
1387   public boolean hasReferences() {
1388     return StoreUtils.hasReferences(this.storeEngine.getStoreFileManager().getStorefiles());
1389   }
1390 
1391   @Override
1392   public CompactionProgress getCompactionProgress() {
1393     return this.storeEngine.getCompactor().getProgress();
1394   }
1395 
1396   @Override
1397   public boolean isMajorCompaction() throws IOException {
1398     for (StoreFile sf : this.storeEngine.getStoreFileManager().getStorefiles()) {
1399       // TODO: what are these reader checks all over the place?
1400       if (sf.getReader() == null) {
1401         LOG.debug("StoreFile " + sf + " has null Reader");
1402         return false;
1403       }
1404     }
1405     return storeEngine.getCompactionPolicy().isMajorCompaction(
1406         this.storeEngine.getStoreFileManager().getStorefiles());
1407   }
1408 
1409   @Override
1410   public CompactionContext requestCompaction() throws IOException {
1411     return requestCompaction(Store.NO_PRIORITY, null);
1412   }
1413 
1414   @Override
1415   public CompactionContext requestCompaction(int priority, CompactionRequest baseRequest)
1416       throws IOException {
1417     // don't even select for compaction if writes are disabled
1418     if (!this.areWritesEnabled()) {
1419       return null;
1420     }
1421 
1422     // Before we do compaction, try to get rid of unneeded files to simplify things.
1423     removeUnneededFiles();
1424 
1425     CompactionContext compaction = storeEngine.createCompaction();
1426     CompactionRequest request = null;
1427     this.lock.readLock().lock();
1428     try {
1429       synchronized (filesCompacting) {
1430         // First, see if coprocessor would want to override selection.
1431         if (this.getCoprocessorHost() != null) {
1432           List<StoreFile> candidatesForCoproc = compaction.preSelect(this.filesCompacting);
1433           boolean override = this.getCoprocessorHost().preCompactSelection(
1434               this, candidatesForCoproc, baseRequest);
1435           if (override) {
1436             // Coprocessor is overriding normal file selection.
1437             compaction.forceSelect(new CompactionRequest(candidatesForCoproc));
1438           }
1439         }
1440 
1441         // Normal case - coprocessor is not overriding file selection.
1442         if (!compaction.hasSelection()) {
1443           boolean isUserCompaction = priority == Store.PRIORITY_USER;
1444           boolean mayUseOffPeak = offPeakHours.isOffPeakHour() &&
1445               offPeakCompactionTracker.compareAndSet(false, true);
1446           try {
1447             compaction.select(this.filesCompacting, isUserCompaction,
1448               mayUseOffPeak, forceMajor && filesCompacting.isEmpty());
1449           } catch (IOException e) {
1450             if (mayUseOffPeak) {
1451               offPeakCompactionTracker.set(false);
1452             }
1453             throw e;
1454           }
1455           assert compaction.hasSelection();
1456           if (mayUseOffPeak && !compaction.getRequest().isOffPeak()) {
1457             // Compaction policy doesn't want to take advantage of off-peak.
1458             offPeakCompactionTracker.set(false);
1459           }
1460         }
1461         if (this.getCoprocessorHost() != null) {
1462           this.getCoprocessorHost().postCompactSelection(
1463               this, ImmutableList.copyOf(compaction.getRequest().getFiles()), baseRequest);
1464         }
1465 
1466         // Selected files; see if we have a compaction with some custom base request.
1467         if (baseRequest != null) {
1468           // Update the request with what the system thinks the request should be;
1469           // its up to the request if it wants to listen.
1470           compaction.forceSelect(
1471               baseRequest.combineWith(compaction.getRequest()));
1472         }
1473         // Finally, we have the resulting files list. Check if we have any files at all.
1474         request = compaction.getRequest();
1475         final Collection<StoreFile> selectedFiles = request.getFiles();
1476         if (selectedFiles.isEmpty()) {
1477           return null;
1478         }
1479 
1480         addToCompactingFiles(selectedFiles);
1481 
1482         // If we're enqueuing a major, clear the force flag.
1483         this.forceMajor = this.forceMajor && !request.isMajor();
1484 
1485         // Set common request properties.
1486         // Set priority, either override value supplied by caller or from store.
1487         request.setPriority((priority != Store.NO_PRIORITY) ? priority : getCompactPriority());
1488         request.setDescription(getRegionInfo().getRegionNameAsString(), getColumnFamilyName());
1489       }
1490     } finally {
1491       this.lock.readLock().unlock();
1492     }
1493 
1494     LOG.debug(getRegionInfo().getEncodedName() + " - "  + getColumnFamilyName()
1495         + ": Initiating " + (request.isMajor() ? "major" : "minor") + " compaction"
1496         + (request.isAllFiles() ? " (all files)" : ""));
1497     this.region.reportCompactionRequestStart(request.isMajor());
1498     return compaction;
1499   }
1500 
1501   /** Adds the files to compacting files. filesCompacting must be locked. */
1502   private void addToCompactingFiles(final Collection<StoreFile> filesToAdd) {
1503     if (filesToAdd == null) return;
1504     // Check that we do not try to compact the same StoreFile twice.
1505     if (!Collections.disjoint(filesCompacting, filesToAdd)) {
1506       Preconditions.checkArgument(false, "%s overlaps with %s", filesToAdd, filesCompacting);
1507     }
1508     filesCompacting.addAll(filesToAdd);
1509     Collections.sort(filesCompacting, StoreFile.Comparators.SEQ_ID);
1510   }
1511 
1512   private void removeUnneededFiles() throws IOException {
1513     if (!conf.getBoolean("hbase.store.delete.expired.storefile", true)) return;
1514     this.lock.readLock().lock();
1515     Collection<StoreFile> delSfs = null;
1516     try {
1517       synchronized (filesCompacting) {
1518         long cfTtl = getStoreFileTtl();
1519         if (cfTtl != Long.MAX_VALUE) {
1520           delSfs = storeEngine.getStoreFileManager().getUnneededFiles(
1521               EnvironmentEdgeManager.currentTimeMillis() - cfTtl, filesCompacting);
1522           addToCompactingFiles(delSfs);
1523         }
1524       }
1525     } finally {
1526       this.lock.readLock().unlock();
1527     }
1528     if (delSfs == null || delSfs.isEmpty()) return;
1529 
1530     Collection<StoreFile> newFiles = new ArrayList<StoreFile>(); // No new files.
1531     writeCompactionWalRecord(delSfs, newFiles);
1532     replaceStoreFiles(delSfs, newFiles);
1533     completeCompaction(delSfs);
1534     LOG.info("Completed removal of " + delSfs.size() + " unnecessary (expired) file(s) in "
1535         + this + " of " + this.getRegionInfo().getRegionNameAsString()
1536         + "; total size for store is " + StringUtils.humanReadableInt(storeSize));
1537   }
1538 
1539   @Override
1540   public void cancelRequestedCompaction(CompactionContext compaction) {
1541     finishCompactionRequest(compaction.getRequest());
1542   }
1543 
1544   private void finishCompactionRequest(CompactionRequest cr) {
1545     this.region.reportCompactionRequestEnd(cr.isMajor(), cr.getFiles().size(), cr.getSize());
1546     if (cr.isOffPeak()) {
1547       offPeakCompactionTracker.set(false);
1548       cr.setOffPeak(false);
1549     }
1550     synchronized (filesCompacting) {
1551       filesCompacting.removeAll(cr.getFiles());
1552     }
1553   }
1554 
1555   /**
1556    * Validates a store file by opening and closing it. In HFileV2 this should
1557    * not be an expensive operation.
1558    *
1559    * @param path the path to the store file
1560    */
1561   private void validateStoreFile(Path path)
1562       throws IOException {
1563     StoreFile storeFile = null;
1564     try {
1565       storeFile = createStoreFileAndReader(path);
1566     } catch (IOException e) {
1567       LOG.error("Failed to open store file : " + path
1568           + ", keeping it in tmp location", e);
1569       throw e;
1570     } finally {
1571       if (storeFile != null) {
1572         storeFile.closeReader(false);
1573       }
1574     }
1575   }
1576 
1577   /**
1578    * <p>It works by processing a compaction that's been written to disk.
1579    *
1580    * <p>It is usually invoked at the end of a compaction, but might also be
1581    * invoked at HStore startup, if the prior execution died midway through.
1582    *
1583    * <p>Moving the compacted TreeMap into place means:
1584    * <pre>
1585    * 1) Unload all replaced StoreFile, close and collect list to delete.
1586    * 2) Compute new store size
1587    * </pre>
1588    *
1589    * @param compactedFiles list of files that were compacted
1590    */
1591   @VisibleForTesting
1592   protected void completeCompaction(final Collection<StoreFile> compactedFiles)
1593     throws IOException {
1594     completeCompaction(compactedFiles, true);
1595   }
1596 
1597 
1598   /**
1599    * <p>It works by processing a compaction that's been written to disk.
1600    *
1601    * <p>It is usually invoked at the end of a compaction, but might also be
1602    * invoked at HStore startup, if the prior execution died midway through.
1603    *
1604    * <p>Moving the compacted TreeMap into place means:
1605    * <pre>
1606    * 1) Unload all replaced StoreFile, close and collect list to delete.
1607    * 2) Compute new store size
1608    * </pre>
1609    *
1610    * @param compactedFiles list of files that were compacted
1611    */
1612   @VisibleForTesting
1613   protected void completeCompaction(final Collection<StoreFile> compactedFiles, boolean removeFiles)
1614       throws IOException {
1615     try {
1616       // Do not delete old store files until we have sent out notification of
1617       // change in case old files are still being accessed by outstanding scanners.
1618       // Don't do this under writeLock; see HBASE-4485 for a possible deadlock
1619       // scenario that could have happened if continue to hold the lock.
1620       notifyChangedReadersObservers();
1621       // At this point the store will use new files for all scanners.
1622 
1623       // let the archive util decide if we should archive or delete the files
1624       LOG.debug("Removing store files after compaction...");
1625       for (StoreFile compactedFile : compactedFiles) {
1626         compactedFile.closeReader(true);
1627       }
1628       if (removeFiles) {
1629         this.fs.removeStoreFiles(this.getColumnFamilyName(), compactedFiles);
1630       }
1631     } catch (IOException e) {
1632       e = e instanceof RemoteException ?
1633                 ((RemoteException)e).unwrapRemoteException() : e;
1634       LOG.error("Failed removing compacted files in " + this +
1635         ". Files we were trying to remove are " + compactedFiles.toString() +
1636         "; some of them may have been already removed", e);
1637     }
1638 
1639     // 4. Compute new store size
1640     this.storeSize = 0L;
1641     this.totalUncompressedBytes = 0L;
1642     for (StoreFile hsf : this.storeEngine.getStoreFileManager().getStorefiles()) {
1643       StoreFile.Reader r = hsf.getReader();
1644       if (r == null) {
1645         LOG.warn("StoreFile " + hsf + " has a null Reader");
1646         continue;
1647       }
1648       this.storeSize += r.length();
1649       this.totalUncompressedBytes += r.getTotalUncompressedBytes();
1650     }
1651   }
1652 
1653   /*
1654    * @param wantedVersions How many versions were asked for.
1655    * @return wantedVersions or this families' {@link HConstants#VERSIONS}.
1656    */
1657   int versionsToReturn(final int wantedVersions) {
1658     if (wantedVersions <= 0) {
1659       throw new IllegalArgumentException("Number of versions must be > 0");
1660     }
1661     // Make sure we do not return more than maximum versions for this store.
1662     int maxVersions = this.family.getMaxVersions();
1663     return wantedVersions > maxVersions ? maxVersions: wantedVersions;
1664   }
1665 
1666   static boolean isExpired(final Cell key, final long oldestTimestamp) {
1667     return key.getTimestamp() < oldestTimestamp;
1668   }
1669 
1670   @Override
1671   public KeyValue getRowKeyAtOrBefore(final byte[] row) throws IOException {
1672     // If minVersions is set, we will not ignore expired KVs.
1673     // As we're only looking for the latest matches, that should be OK.
1674     // With minVersions > 0 we guarantee that any KV that has any version
1675     // at all (expired or not) has at least one version that will not expire.
1676     // Note that this method used to take a KeyValue as arguments. KeyValue
1677     // can be back-dated, a row key cannot.
1678     long ttlToUse = scanInfo.getMinVersions() > 0 ? Long.MAX_VALUE : this.scanInfo.getTtl();
1679 
1680     KeyValue kv = new KeyValue(row, HConstants.LATEST_TIMESTAMP);
1681 
1682     GetClosestRowBeforeTracker state = new GetClosestRowBeforeTracker(
1683       this.comparator, kv, ttlToUse, this.getRegionInfo().isMetaRegion());
1684     this.lock.readLock().lock();
1685     try {
1686       // First go to the memstore.  Pick up deletes and candidates.
1687       this.memstore.getRowKeyAtOrBefore(state);
1688       // Check if match, if we got a candidate on the asked for 'kv' row.
1689       // Process each relevant store file. Run through from newest to oldest.
1690       Iterator<StoreFile> sfIterator = this.storeEngine.getStoreFileManager()
1691           .getCandidateFilesForRowKeyBefore(state.getTargetKey());
1692       while (sfIterator.hasNext()) {
1693         StoreFile sf = sfIterator.next();
1694         sfIterator.remove(); // Remove sf from iterator.
1695         boolean haveNewCandidate = rowAtOrBeforeFromStoreFile(sf, state);
1696         KeyValue keyv = KeyValueUtil.ensureKeyValue(state.getCandidate());
1697         // we have an optimization here which stops the search if we find exact match.
1698         if (keyv != null && CellUtil.matchingRow(keyv, row)) {
1699           return KeyValueUtil.ensureKeyValue(state.getCandidate());
1700         }
1701         if (haveNewCandidate) {
1702           sfIterator = this.storeEngine.getStoreFileManager().updateCandidateFilesForRowKeyBefore(
1703               sfIterator, state.getTargetKey(), KeyValueUtil.ensureKeyValue(state.getCandidate()));
1704         }
1705       }
1706       return KeyValueUtil.ensureKeyValue(state.getCandidate());
1707     } finally {
1708       this.lock.readLock().unlock();
1709     }
1710   }
1711 
1712   /*
1713    * Check an individual MapFile for the row at or before a given row.
1714    * @param f
1715    * @param state
1716    * @throws IOException
1717    * @return True iff the candidate has been updated in the state.
1718    */
1719   private boolean rowAtOrBeforeFromStoreFile(final StoreFile f,
1720                                           final GetClosestRowBeforeTracker state)
1721       throws IOException {
1722     StoreFile.Reader r = f.getReader();
1723     if (r == null) {
1724       LOG.warn("StoreFile " + f + " has a null Reader");
1725       return false;
1726     }
1727     if (r.getEntries() == 0) {
1728       LOG.warn("StoreFile " + f + " is a empty store file");
1729       return false;
1730     }
1731     // TODO: Cache these keys rather than make each time?
1732     byte [] fk = r.getFirstKey();
1733     if (fk == null) return false;
1734     KeyValue firstKV = KeyValue.createKeyValueFromKey(fk, 0, fk.length);
1735     byte [] lk = r.getLastKey();
1736     KeyValue lastKV = KeyValue.createKeyValueFromKey(lk, 0, lk.length);
1737     KeyValue firstOnRow = state.getTargetKey();
1738     if (this.comparator.compareRows(lastKV, firstOnRow) < 0) {
1739       // If last key in file is not of the target table, no candidates in this
1740       // file.  Return.
1741       if (!state.isTargetTable(lastKV)) return false;
1742       // If the row we're looking for is past the end of file, set search key to
1743       // last key. TODO: Cache last and first key rather than make each time.
1744       firstOnRow = new KeyValue(lastKV.getRow(), HConstants.LATEST_TIMESTAMP);
1745     }
1746     // Get a scanner that caches blocks and that uses pread.
1747     HFileScanner scanner = r.getScanner(true, true, false);
1748     // Seek scanner.  If can't seek it, return.
1749     if (!seekToScanner(scanner, firstOnRow, firstKV)) return false;
1750     // If we found candidate on firstOnRow, just return. THIS WILL NEVER HAPPEN!
1751     // Unlikely that there'll be an instance of actual first row in table.
1752     if (walkForwardInSingleRow(scanner, firstOnRow, state)) return true;
1753     // If here, need to start backing up.
1754     while (scanner.seekBefore(firstOnRow.getBuffer(), firstOnRow.getKeyOffset(),
1755        firstOnRow.getKeyLength())) {
1756       Cell kv = scanner.getKeyValue();
1757       if (!state.isTargetTable(kv)) break;
1758       if (!state.isBetterCandidate(kv)) break;
1759       // Make new first on row.
1760       firstOnRow = new KeyValue(kv.getRow(), HConstants.LATEST_TIMESTAMP);
1761       // Seek scanner.  If can't seek it, break.
1762       if (!seekToScanner(scanner, firstOnRow, firstKV)) return false;
1763       // If we find something, break;
1764       if (walkForwardInSingleRow(scanner, firstOnRow, state)) return true;
1765     }
1766     return false;
1767   }
1768 
1769   /*
1770    * Seek the file scanner to firstOnRow or first entry in file.
1771    * @param scanner
1772    * @param firstOnRow
1773    * @param firstKV
1774    * @return True if we successfully seeked scanner.
1775    * @throws IOException
1776    */
1777   private boolean seekToScanner(final HFileScanner scanner,
1778                                 final KeyValue firstOnRow,
1779                                 final KeyValue firstKV)
1780       throws IOException {
1781     KeyValue kv = firstOnRow;
1782     // If firstOnRow < firstKV, set to firstKV
1783     if (this.comparator.compareRows(firstKV, firstOnRow) == 0) kv = firstKV;
1784     int result = scanner.seekTo(kv);
1785     return result != -1;
1786   }
1787 
1788   /*
1789    * When we come in here, we are probably at the kv just before we break into
1790    * the row that firstOnRow is on.  Usually need to increment one time to get
1791    * on to the row we are interested in.
1792    * @param scanner
1793    * @param firstOnRow
1794    * @param state
1795    * @return True we found a candidate.
1796    * @throws IOException
1797    */
1798   private boolean walkForwardInSingleRow(final HFileScanner scanner,
1799                                          final KeyValue firstOnRow,
1800                                          final GetClosestRowBeforeTracker state)
1801       throws IOException {
1802     boolean foundCandidate = false;
1803     do {
1804       Cell kv = scanner.getKeyValue();
1805       // If we are not in the row, skip.
1806       if (this.comparator.compareRows(kv, firstOnRow) < 0) continue;
1807       // Did we go beyond the target row? If so break.
1808       if (state.isTooFar(kv, firstOnRow)) break;
1809       if (state.isExpired(kv)) {
1810         continue;
1811       }
1812       // If we added something, this row is a contender. break.
1813       if (state.handle(kv)) {
1814         foundCandidate = true;
1815         break;
1816       }
1817     } while(scanner.next());
1818     return foundCandidate;
1819   }
1820 
1821   @Override
1822   public boolean canSplit() {
1823     this.lock.readLock().lock();
1824     try {
1825       // Not split-able if we find a reference store file present in the store.
1826       boolean result = !hasReferences();
1827       if (!result && LOG.isDebugEnabled()) {
1828         LOG.debug("Cannot split region due to reference files being there");
1829       }
1830       return result;
1831     } finally {
1832       this.lock.readLock().unlock();
1833     }
1834   }
1835 
1836   @Override
1837   public byte[] getSplitPoint() {
1838     this.lock.readLock().lock();
1839     try {
1840       // Should already be enforced by the split policy!
1841       assert !this.getRegionInfo().isMetaRegion();
1842       // Not split-able if we find a reference store file present in the store.
1843       if (hasReferences()) {
1844         return null;
1845       }
1846       return this.storeEngine.getStoreFileManager().getSplitPoint();
1847     } catch(IOException e) {
1848       LOG.warn("Failed getting store size for " + this, e);
1849     } finally {
1850       this.lock.readLock().unlock();
1851     }
1852     return null;
1853   }
1854 
1855   @Override
1856   public long getLastCompactSize() {
1857     return this.lastCompactSize;
1858   }
1859 
1860   @Override
1861   public long getSize() {
1862     return storeSize;
1863   }
1864 
1865   @Override
1866   public void triggerMajorCompaction() {
1867     this.forceMajor = true;
1868   }
1869 
1870 
1871   //////////////////////////////////////////////////////////////////////////////
1872   // File administration
1873   //////////////////////////////////////////////////////////////////////////////
1874 
1875   @Override
1876   public KeyValueScanner getScanner(Scan scan,
1877       final NavigableSet<byte []> targetCols, long readPt) throws IOException {
1878     lock.readLock().lock();
1879     try {
1880       KeyValueScanner scanner = null;
1881       if (this.getCoprocessorHost() != null) {
1882         scanner = this.getCoprocessorHost().preStoreScannerOpen(this, scan, targetCols);
1883       }
1884       if (scanner == null) {
1885         scanner = scan.isReversed() ? new ReversedStoreScanner(this,
1886             getScanInfo(), scan, targetCols, readPt) : new StoreScanner(this,
1887             getScanInfo(), scan, targetCols, readPt);
1888       }
1889       return scanner;
1890     } finally {
1891       lock.readLock().unlock();
1892     }
1893   }
1894 
1895   @Override
1896   public String toString() {
1897     return this.getColumnFamilyName();
1898   }
1899 
1900   @Override
1901   // TODO: why is there this and also getNumberOfStorefiles?! Remove one.
1902   public int getStorefilesCount() {
1903     return this.storeEngine.getStoreFileManager().getStorefileCount();
1904   }
1905 
1906   @Override
1907   public long getStoreSizeUncompressed() {
1908     return this.totalUncompressedBytes;
1909   }
1910 
1911   @Override
1912   public long getStorefilesSize() {
1913     long size = 0;
1914     for (StoreFile s: this.storeEngine.getStoreFileManager().getStorefiles()) {
1915       StoreFile.Reader r = s.getReader();
1916       if (r == null) {
1917         LOG.warn("StoreFile " + s + " has a null Reader");
1918         continue;
1919       }
1920       size += r.length();
1921     }
1922     return size;
1923   }
1924 
1925   @Override
1926   public long getStorefilesIndexSize() {
1927     long size = 0;
1928     for (StoreFile s: this.storeEngine.getStoreFileManager().getStorefiles()) {
1929       StoreFile.Reader r = s.getReader();
1930       if (r == null) {
1931         LOG.warn("StoreFile " + s + " has a null Reader");
1932         continue;
1933       }
1934       size += r.indexSize();
1935     }
1936     return size;
1937   }
1938 
1939   @Override
1940   public long getTotalStaticIndexSize() {
1941     long size = 0;
1942     for (StoreFile s : this.storeEngine.getStoreFileManager().getStorefiles()) {
1943       size += s.getReader().getUncompressedDataIndexSize();
1944     }
1945     return size;
1946   }
1947 
1948   @Override
1949   public long getTotalStaticBloomSize() {
1950     long size = 0;
1951     for (StoreFile s : this.storeEngine.getStoreFileManager().getStorefiles()) {
1952       StoreFile.Reader r = s.getReader();
1953       size += r.getTotalBloomSize();
1954     }
1955     return size;
1956   }
1957 
1958   @Override
1959   public long getMemStoreSize() {
1960     return this.memstore.size();
1961   }
1962 
1963   @Override
1964   public int getCompactPriority() {
1965     int priority = this.storeEngine.getStoreFileManager().getStoreCompactionPriority();
1966     if (priority == PRIORITY_USER) {
1967       LOG.warn("Compaction priority is USER despite there being no user compaction");
1968     }
1969     return priority;
1970   }
1971 
1972   @Override
1973   public boolean throttleCompaction(long compactionSize) {
1974     return storeEngine.getCompactionPolicy().throttleCompaction(compactionSize);
1975   }
1976 
1977   public HRegion getHRegion() {
1978     return this.region;
1979   }
1980 
1981   @Override
1982   public RegionCoprocessorHost getCoprocessorHost() {
1983     return this.region.getCoprocessorHost();
1984   }
1985 
1986   @Override
1987   public HRegionInfo getRegionInfo() {
1988     return this.fs.getRegionInfo();
1989   }
1990 
1991   @Override
1992   public boolean areWritesEnabled() {
1993     return this.region.areWritesEnabled();
1994   }
1995 
1996   @Override
1997   public long getSmallestReadPoint() {
1998     return this.region.getSmallestReadPoint();
1999   }
2000 
2001   /**
2002    * Used in tests. TODO: Remove
2003    *
2004    * Updates the value for the given row/family/qualifier. This function will always be seen as
2005    * atomic by other readers because it only puts a single KV to memstore. Thus no read/write
2006    * control necessary.
2007    * @param row row to update
2008    * @param f family to update
2009    * @param qualifier qualifier to update
2010    * @param newValue the new value to set into memstore
2011    * @return memstore size delta
2012    * @throws IOException
2013    */
2014   public long updateColumnValue(byte [] row, byte [] f,
2015                                 byte [] qualifier, long newValue)
2016       throws IOException {
2017 
2018     this.lock.readLock().lock();
2019     try {
2020       long now = EnvironmentEdgeManager.currentTimeMillis();
2021 
2022       return this.memstore.updateColumnValue(row,
2023           f,
2024           qualifier,
2025           newValue,
2026           now);
2027 
2028     } finally {
2029       this.lock.readLock().unlock();
2030     }
2031   }
2032 
2033   @Override
2034   public long upsert(Iterable<Cell> cells, long readpoint) throws IOException {
2035     this.lock.readLock().lock();
2036     try {
2037       return this.memstore.upsert(cells, readpoint);
2038     } finally {
2039       this.lock.readLock().unlock();
2040     }
2041   }
2042 
2043   @Override
2044   public StoreFlushContext createFlushContext(long cacheFlushId) {
2045     return new StoreFlusherImpl(cacheFlushId);
2046   }
2047 
2048   private class StoreFlusherImpl implements StoreFlushContext {
2049 
2050     private long cacheFlushSeqNum;
2051     private MemStoreSnapshot snapshot;
2052     private List<Path> tempFiles;
2053     private List<Path> committedFiles;
2054     private long cacheFlushCount;
2055     private long cacheFlushSize;
2056 
2057     private StoreFlusherImpl(long cacheFlushSeqNum) {
2058       this.cacheFlushSeqNum = cacheFlushSeqNum;
2059     }
2060 
2061     /**
2062      * This is not thread safe. The caller should have a lock on the region or the store.
2063      * If necessary, the lock can be added with the patch provided in HBASE-10087
2064      */
2065     @Override
2066     public void prepare() {
2067       this.snapshot = memstore.snapshot();
2068       this.cacheFlushCount = snapshot.getCellsCount();
2069       this.cacheFlushSize = snapshot.getSize();
2070       committedFiles = new ArrayList<Path>(1);
2071     }
2072 
2073     @Override
2074     public void flushCache(MonitoredTask status) throws IOException {
2075       tempFiles = HStore.this.flushCache(cacheFlushSeqNum, snapshot, status);
2076     }
2077 
2078     @Override
2079     public boolean commit(MonitoredTask status) throws IOException {
2080       if (this.tempFiles == null || this.tempFiles.isEmpty()) {
2081         return false;
2082       }
2083       List<StoreFile> storeFiles = new ArrayList<StoreFile>(this.tempFiles.size());
2084       for (Path storeFilePath : tempFiles) {
2085         try {
2086           storeFiles.add(HStore.this.commitFile(storeFilePath, cacheFlushSeqNum, status));
2087         } catch (IOException ex) {
2088           LOG.error("Failed to commit store file " + storeFilePath, ex);
2089           // Try to delete the files we have committed before.
2090           for (StoreFile sf : storeFiles) {
2091             Path pathToDelete = sf.getPath();
2092             try {
2093               sf.deleteReader();
2094             } catch (IOException deleteEx) {
2095               LOG.fatal("Failed to delete store file we committed, halting " + pathToDelete, ex);
2096               Runtime.getRuntime().halt(1);
2097             }
2098           }
2099           throw new IOException("Failed to commit the flush", ex);
2100         }
2101       }
2102 
2103       for (StoreFile sf : storeFiles) {
2104         if (HStore.this.getCoprocessorHost() != null) {
2105           HStore.this.getCoprocessorHost().postFlush(HStore.this, sf);
2106         }
2107         committedFiles.add(sf.getPath());
2108       }
2109 
2110       HStore.this.flushedCellsCount += cacheFlushCount;
2111       HStore.this.flushedCellsSize += cacheFlushSize;
2112 
2113       // Add new file to store files.  Clear snapshot too while we have the Store write lock.
2114       return HStore.this.updateStorefiles(storeFiles, snapshot.getId());
2115     }
2116 
2117     @Override
2118     public List<Path> getCommittedFiles() {
2119       return committedFiles;
2120     }
2121   }
2122 
2123   @Override
2124   public boolean needsCompaction() {
2125     return this.storeEngine.needsCompaction(this.filesCompacting);
2126   }
2127 
2128   @Override
2129   public CacheConfig getCacheConfig() {
2130     return this.cacheConf;
2131   }
2132 
2133   public static final long FIXED_OVERHEAD =
2134       ClassSize.align(ClassSize.OBJECT + (16 * ClassSize.REFERENCE) + (10 * Bytes.SIZEOF_LONG)
2135               + (5 * Bytes.SIZEOF_INT) + (2 * Bytes.SIZEOF_BOOLEAN));
2136 
2137   public static final long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD
2138       + ClassSize.OBJECT + ClassSize.REENTRANT_LOCK
2139       + ClassSize.CONCURRENT_SKIPLISTMAP
2140       + ClassSize.CONCURRENT_SKIPLISTMAP_ENTRY + ClassSize.OBJECT
2141       + ScanInfo.FIXED_OVERHEAD);
2142 
2143   @Override
2144   public long heapSize() {
2145     return DEEP_OVERHEAD + this.memstore.heapSize();
2146   }
2147 
2148   @Override
2149   public KeyValue.KVComparator getComparator() {
2150     return comparator;
2151   }
2152 
2153   @Override
2154   public ScanInfo getScanInfo() {
2155     return scanInfo;
2156   }
2157 
2158   /**
2159    * Set scan info, used by test
2160    * @param scanInfo new scan info to use for test
2161    */
2162   void setScanInfo(ScanInfo scanInfo) {
2163     this.scanInfo = scanInfo;
2164   }
2165 
2166   @Override
2167   public boolean hasTooManyStoreFiles() {
2168     return getStorefilesCount() > this.blockingFileCount;
2169   }
2170 
2171   @Override
2172   public long getFlushedCellsCount() {
2173     return flushedCellsCount;
2174   }
2175 
2176   @Override
2177   public long getFlushedCellsSize() {
2178     return flushedCellsSize;
2179   }
2180 
2181   @Override
2182   public long getCompactedCellsCount() {
2183     return compactedCellsCount;
2184   }
2185 
2186   @Override
2187   public long getCompactedCellsSize() {
2188     return compactedCellsSize;
2189   }
2190 
2191   @Override
2192   public long getMajorCompactedCellsCount() {
2193     return majorCompactedCellsCount;
2194   }
2195 
2196   @Override
2197   public long getMajorCompactedCellsSize() {
2198     return majorCompactedCellsSize;
2199   }
2200 }