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