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