001/**
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.io.hfile;
019
020import java.util.Optional;
021
022import org.apache.hadoop.conf.Configuration;
023import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
024import org.apache.hadoop.hbase.io.ByteBuffAllocator;
025import org.apache.hadoop.hbase.io.hfile.BlockType.BlockCategory;
026import org.apache.yetus.audience.InterfaceAudience;
027import org.slf4j.Logger;
028import org.slf4j.LoggerFactory;
029
030/**
031 * Stores all of the cache objects and configuration for a single HFile.
032 */
033@InterfaceAudience.Private
034public class CacheConfig {
035  private static final Logger LOG = LoggerFactory.getLogger(CacheConfig.class.getName());
036
037  /**
038   * Disabled cache configuration
039   */
040  public static final CacheConfig DISABLED = new CacheConfig();
041
042  /**
043   * Configuration key to cache data blocks on read. Bloom blocks and index blocks are always be
044   * cached if the block cache is enabled.
045   */
046  public static final String CACHE_DATA_ON_READ_KEY = "hbase.block.data.cacheonread";
047
048  /**
049   * Configuration key to cache data blocks on write. There are separate
050   * switches for bloom blocks and non-root index blocks.
051   */
052  public static final String CACHE_BLOCKS_ON_WRITE_KEY = "hbase.rs.cacheblocksonwrite";
053
054  /**
055   * Configuration key to cache leaf and intermediate-level index blocks on
056   * write.
057   */
058  public static final String CACHE_INDEX_BLOCKS_ON_WRITE_KEY = "hfile.block.index.cacheonwrite";
059
060  /**
061   * Configuration key to cache compound bloom filter blocks on write.
062   */
063  public static final String CACHE_BLOOM_BLOCKS_ON_WRITE_KEY = "hfile.block.bloom.cacheonwrite";
064
065  /**
066   * Configuration key to cache data blocks in compressed and/or encrypted format.
067   */
068  public static final String CACHE_DATA_BLOCKS_COMPRESSED_KEY = "hbase.block.data.cachecompressed";
069
070  /**
071   * Configuration key to evict all blocks of a given file from the block cache
072   * when the file is closed.
073   */
074  public static final String EVICT_BLOCKS_ON_CLOSE_KEY = "hbase.rs.evictblocksonclose";
075
076  /**
077   * Configuration key to prefetch all blocks of a given file into the block cache
078   * when the file is opened.
079   */
080  public static final String PREFETCH_BLOCKS_ON_OPEN_KEY = "hbase.rs.prefetchblocksonopen";
081
082  /**
083   * Configuration key to cache blocks when a compacted file is written
084   */
085  public static final String CACHE_COMPACTED_BLOCKS_ON_WRITE_KEY =
086      "hbase.rs.cachecompactedblocksonwrite";
087
088  /**
089   * Configuration key to determine total size in bytes of compacted files beyond which we do not
090   * cache blocks on compaction
091   */
092  public static final String CACHE_COMPACTED_BLOCKS_ON_WRITE_THRESHOLD_KEY =
093      "hbase.rs.cachecompactedblocksonwrite.threshold";
094
095  public static final String DROP_BEHIND_CACHE_COMPACTION_KEY =
096      "hbase.hfile.drop.behind.compaction";
097
098  // Defaults
099  public static final boolean DEFAULT_CACHE_DATA_ON_READ = true;
100  public static final boolean DEFAULT_CACHE_DATA_ON_WRITE = false;
101  public static final boolean DEFAULT_IN_MEMORY = false;
102  public static final boolean DEFAULT_CACHE_INDEXES_ON_WRITE = false;
103  public static final boolean DEFAULT_CACHE_BLOOMS_ON_WRITE = false;
104  public static final boolean DEFAULT_EVICT_ON_CLOSE = false;
105  public static final boolean DEFAULT_CACHE_DATA_COMPRESSED = false;
106  public static final boolean DEFAULT_PREFETCH_ON_OPEN = false;
107  public static final boolean DEFAULT_CACHE_COMPACTED_BLOCKS_ON_WRITE = false;
108  public static final boolean DROP_BEHIND_CACHE_COMPACTION_DEFAULT = true;
109  public static final long DEFAULT_CACHE_COMPACTED_BLOCKS_ON_WRITE_THRESHOLD = Long.MAX_VALUE;
110
111  /**
112   * Whether blocks should be cached on read (default is on if there is a
113   * cache but this can be turned off on a per-family or per-request basis).
114   * If off we will STILL cache meta blocks; i.e. INDEX and BLOOM types.
115   * This cannot be disabled.
116   */
117  private final boolean cacheDataOnRead;
118
119  /** Whether blocks should be flagged as in-memory when being cached */
120  private final boolean inMemory;
121
122  /** Whether data blocks should be cached when new files are written */
123  private boolean cacheDataOnWrite;
124
125  /** Whether index blocks should be cached when new files are written */
126  private boolean cacheIndexesOnWrite;
127
128  /** Whether compound bloom filter blocks should be cached on write */
129  private boolean cacheBloomsOnWrite;
130
131  /** Whether blocks of a file should be evicted when the file is closed */
132  private volatile boolean evictOnClose;
133
134  /** Whether data blocks should be stored in compressed and/or encrypted form in the cache */
135  private final boolean cacheDataCompressed;
136
137  /** Whether data blocks should be prefetched into the cache */
138  private final boolean prefetchOnOpen;
139
140  /**
141   * Whether data blocks should be cached when compacted file is written
142   */
143  private final boolean cacheCompactedDataOnWrite;
144
145  /**
146   * Determine threshold beyond which we do not cache blocks on compaction
147   */
148  private long cacheCompactedDataOnWriteThreshold;
149
150  private final boolean dropBehindCompaction;
151
152  // Local reference to the block cache
153  private final BlockCache blockCache;
154
155  private final ByteBuffAllocator byteBuffAllocator;
156
157  /**
158   * Create a cache configuration using the specified configuration object and
159   * defaults for family level settings. Only use if no column family context.
160   * @param conf hbase configuration
161   */
162  public CacheConfig(Configuration conf) {
163    this(conf, null);
164  }
165
166  public CacheConfig(Configuration conf, BlockCache blockCache) {
167    this(conf, null, blockCache, ByteBuffAllocator.HEAP);
168  }
169
170  /**
171   * Create a cache configuration using the specified configuration object and
172   * family descriptor.
173   * @param conf hbase configuration
174   * @param family column family configuration
175   */
176  public CacheConfig(Configuration conf, ColumnFamilyDescriptor family, BlockCache blockCache,
177      ByteBuffAllocator byteBuffAllocator) {
178    this.cacheDataOnRead = conf.getBoolean(CACHE_DATA_ON_READ_KEY, DEFAULT_CACHE_DATA_ON_READ) &&
179        (family == null ? true : family.isBlockCacheEnabled());
180    this.inMemory = family == null ? DEFAULT_IN_MEMORY : family.isInMemory();
181    this.cacheDataCompressed =
182        conf.getBoolean(CACHE_DATA_BLOCKS_COMPRESSED_KEY, DEFAULT_CACHE_DATA_COMPRESSED);
183    this.dropBehindCompaction =
184        conf.getBoolean(DROP_BEHIND_CACHE_COMPACTION_KEY, DROP_BEHIND_CACHE_COMPACTION_DEFAULT);
185    // For the following flags we enable them regardless of per-schema settings
186    // if they are enabled in the global configuration.
187    this.cacheDataOnWrite =
188        conf.getBoolean(CACHE_BLOCKS_ON_WRITE_KEY, DEFAULT_CACHE_DATA_ON_WRITE) ||
189            (family == null ? false : family.isCacheDataOnWrite());
190    this.cacheIndexesOnWrite =
191        conf.getBoolean(CACHE_INDEX_BLOCKS_ON_WRITE_KEY, DEFAULT_CACHE_INDEXES_ON_WRITE) ||
192            (family == null ? false : family.isCacheIndexesOnWrite());
193    this.cacheBloomsOnWrite =
194        conf.getBoolean(CACHE_BLOOM_BLOCKS_ON_WRITE_KEY, DEFAULT_CACHE_BLOOMS_ON_WRITE) ||
195            (family == null ? false : family.isCacheBloomsOnWrite());
196    this.evictOnClose = conf.getBoolean(EVICT_BLOCKS_ON_CLOSE_KEY, DEFAULT_EVICT_ON_CLOSE) ||
197        (family == null ? false : family.isEvictBlocksOnClose());
198    this.prefetchOnOpen = conf.getBoolean(PREFETCH_BLOCKS_ON_OPEN_KEY, DEFAULT_PREFETCH_ON_OPEN) ||
199        (family == null ? false : family.isPrefetchBlocksOnOpen());
200    this.cacheCompactedDataOnWrite = conf.getBoolean(CACHE_COMPACTED_BLOCKS_ON_WRITE_KEY,
201      DEFAULT_CACHE_COMPACTED_BLOCKS_ON_WRITE);
202    this.cacheCompactedDataOnWriteThreshold = getCacheCompactedBlocksOnWriteThreshold(conf);
203    this.blockCache = blockCache;
204    this.byteBuffAllocator = byteBuffAllocator;
205  }
206
207  /**
208   * Constructs a cache configuration copied from the specified configuration.
209   */
210  public CacheConfig(CacheConfig cacheConf) {
211    this.cacheDataOnRead = cacheConf.cacheDataOnRead;
212    this.inMemory = cacheConf.inMemory;
213    this.cacheDataOnWrite = cacheConf.cacheDataOnWrite;
214    this.cacheIndexesOnWrite = cacheConf.cacheIndexesOnWrite;
215    this.cacheBloomsOnWrite = cacheConf.cacheBloomsOnWrite;
216    this.evictOnClose = cacheConf.evictOnClose;
217    this.cacheDataCompressed = cacheConf.cacheDataCompressed;
218    this.prefetchOnOpen = cacheConf.prefetchOnOpen;
219    this.cacheCompactedDataOnWrite = cacheConf.cacheCompactedDataOnWrite;
220    this.cacheCompactedDataOnWriteThreshold = cacheConf.cacheCompactedDataOnWriteThreshold;
221    this.dropBehindCompaction = cacheConf.dropBehindCompaction;
222    this.blockCache = cacheConf.blockCache;
223    this.byteBuffAllocator = cacheConf.byteBuffAllocator;
224  }
225
226  private CacheConfig() {
227    this.cacheDataOnRead = false;
228    this.inMemory = false;
229    this.cacheDataOnWrite = false;
230    this.cacheIndexesOnWrite = false;
231    this.cacheBloomsOnWrite = false;
232    this.evictOnClose = false;
233    this.cacheDataCompressed = false;
234    this.prefetchOnOpen = false;
235    this.cacheCompactedDataOnWrite = false;
236    this.dropBehindCompaction = false;
237    this.blockCache = null;
238    this.byteBuffAllocator = ByteBuffAllocator.HEAP;
239  }
240
241  /**
242   * Returns whether the DATA blocks of this HFile should be cached on read or not (we always
243   * cache the meta blocks, the INDEX and BLOOM blocks).
244   * @return true if blocks should be cached on read, false if not
245   */
246  public boolean shouldCacheDataOnRead() {
247    return cacheDataOnRead;
248  }
249
250  public boolean shouldDropBehindCompaction() {
251    return dropBehindCompaction;
252  }
253
254  /**
255   * Should we cache a block of a particular category? We always cache
256   * important blocks such as index blocks, as long as the block cache is
257   * available.
258   */
259  public boolean shouldCacheBlockOnRead(BlockCategory category) {
260    return cacheDataOnRead || category == BlockCategory.INDEX || category == BlockCategory.BLOOM ||
261        (prefetchOnOpen && (category != BlockCategory.META && category != BlockCategory.UNKNOWN));
262  }
263
264  /**
265   * @return true if blocks in this file should be flagged as in-memory
266   */
267  public boolean isInMemory() {
268    return this.inMemory;
269  }
270
271  /**
272   * @return true if data blocks should be written to the cache when an HFile is
273   *         written, false if not
274   */
275  public boolean shouldCacheDataOnWrite() {
276    return this.cacheDataOnWrite;
277  }
278
279  /**
280   * @param cacheDataOnWrite whether data blocks should be written to the cache
281   *                         when an HFile is written
282   */
283  public void setCacheDataOnWrite(boolean cacheDataOnWrite) {
284    this.cacheDataOnWrite = cacheDataOnWrite;
285  }
286
287  /**
288   * Enable cache on write including:
289   * cacheDataOnWrite
290   * cacheIndexesOnWrite
291   * cacheBloomsOnWrite
292   */
293  public void enableCacheOnWrite() {
294    this.cacheDataOnWrite = true;
295    this.cacheIndexesOnWrite = true;
296    this.cacheBloomsOnWrite = true;
297  }
298
299  /**
300   * @return true if index blocks should be written to the cache when an HFile
301   *         is written, false if not
302   */
303  public boolean shouldCacheIndexesOnWrite() {
304    return this.cacheIndexesOnWrite;
305  }
306
307  /**
308   * @return true if bloom blocks should be written to the cache when an HFile
309   *         is written, false if not
310   */
311  public boolean shouldCacheBloomsOnWrite() {
312    return this.cacheBloomsOnWrite;
313  }
314
315  /**
316   * @return true if blocks should be evicted from the cache when an HFile
317   *         reader is closed, false if not
318   */
319  public boolean shouldEvictOnClose() {
320    return this.evictOnClose;
321  }
322
323  /**
324   * Only used for testing.
325   * @param evictOnClose whether blocks should be evicted from the cache when an
326   *                     HFile reader is closed
327   */
328  public void setEvictOnClose(boolean evictOnClose) {
329    this.evictOnClose = evictOnClose;
330  }
331
332  /**
333   * @return true if data blocks should be compressed in the cache, false if not
334   */
335  public boolean shouldCacheDataCompressed() {
336    return this.cacheDataOnRead && this.cacheDataCompressed;
337  }
338
339  /**
340   * @return true if this {@link BlockCategory} should be compressed in blockcache, false otherwise
341   */
342  public boolean shouldCacheCompressed(BlockCategory category) {
343    switch (category) {
344      case DATA:
345        return this.cacheDataOnRead && this.cacheDataCompressed;
346      default:
347        return false;
348    }
349  }
350
351  /**
352   * @return true if blocks should be prefetched into the cache on open, false if not
353   */
354  public boolean shouldPrefetchOnOpen() {
355    return this.prefetchOnOpen;
356  }
357
358  /**
359   * @return true if blocks should be cached while writing during compaction, false if not
360   */
361  public boolean shouldCacheCompactedBlocksOnWrite() {
362    return this.cacheCompactedDataOnWrite;
363  }
364
365  /**
366   * @return total file size in bytes threshold for caching while writing during compaction
367   */
368  public long getCacheCompactedBlocksOnWriteThreshold() {
369    return this.cacheCompactedDataOnWriteThreshold;
370  }
371  /**
372   * Return true if we may find this type of block in block cache.
373   * <p>
374   * TODO: today {@code family.isBlockCacheEnabled()} only means {@code cacheDataOnRead}, so here we
375   * consider lots of other configurations such as {@code cacheDataOnWrite}. We should fix this in
376   * the future, {@code cacheDataOnWrite} should honor the CF level {@code isBlockCacheEnabled}
377   * configuration.
378   */
379  public boolean shouldReadBlockFromCache(BlockType blockType) {
380    if (cacheDataOnRead) {
381      return true;
382    }
383    if (prefetchOnOpen) {
384      return true;
385    }
386    if (cacheDataOnWrite) {
387      return true;
388    }
389    if (blockType == null) {
390      return true;
391    }
392    if (blockType.getCategory() == BlockCategory.BLOOM ||
393        blockType.getCategory() == BlockCategory.INDEX) {
394      return true;
395    }
396    return false;
397  }
398
399  /**
400   * If we make sure the block could not be cached, we will not acquire the lock
401   * otherwise we will acquire lock
402   */
403  public boolean shouldLockOnCacheMiss(BlockType blockType) {
404    if (blockType == null) {
405      return true;
406    }
407    return shouldCacheBlockOnRead(blockType.getCategory());
408  }
409
410  /**
411   * Returns the block cache.
412   *
413   * @return the block cache, or null if caching is completely disabled
414   */
415  public Optional<BlockCache> getBlockCache() {
416    return Optional.ofNullable(this.blockCache);
417  }
418
419  public boolean isCombinedBlockCache() {
420    return blockCache instanceof CombinedBlockCache;
421  }
422
423  public ByteBuffAllocator getByteBuffAllocator() {
424    return this.byteBuffAllocator;
425  }
426
427  private long getCacheCompactedBlocksOnWriteThreshold(Configuration conf) {
428    long cacheCompactedBlocksOnWriteThreshold = conf
429      .getLong(CACHE_COMPACTED_BLOCKS_ON_WRITE_THRESHOLD_KEY,
430        DEFAULT_CACHE_COMPACTED_BLOCKS_ON_WRITE_THRESHOLD);
431
432    if (cacheCompactedBlocksOnWriteThreshold < 0) {
433      LOG.warn(
434        "cacheCompactedBlocksOnWriteThreshold value : {} is less than 0, resetting it to: {}",
435        cacheCompactedBlocksOnWriteThreshold, DEFAULT_CACHE_COMPACTED_BLOCKS_ON_WRITE_THRESHOLD);
436      cacheCompactedBlocksOnWriteThreshold = DEFAULT_CACHE_COMPACTED_BLOCKS_ON_WRITE_THRESHOLD;
437    }
438
439    return cacheCompactedBlocksOnWriteThreshold;
440  }
441
442  @Override
443  public String toString() {
444    return "cacheDataOnRead=" + shouldCacheDataOnRead() + ", cacheDataOnWrite="
445        + shouldCacheDataOnWrite() + ", cacheIndexesOnWrite=" + shouldCacheIndexesOnWrite()
446        + ", cacheBloomsOnWrite=" + shouldCacheBloomsOnWrite() + ", cacheEvictOnClose="
447        + shouldEvictOnClose() + ", cacheDataCompressed=" + shouldCacheDataCompressed()
448        + ", prefetchOnOpen=" + shouldPrefetchOnOpen();
449  }
450}