View Javadoc

1   /**
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  package org.apache.hadoop.hbase.io.hfile;
19  
20  import java.io.IOException;
21  import java.util.NavigableMap;
22  import java.util.NavigableSet;
23  import java.util.concurrent.ConcurrentSkipListMap;
24  import java.util.concurrent.ConcurrentSkipListSet;
25  
26  import org.apache.hadoop.hbase.classification.InterfaceAudience;
27  import org.apache.hadoop.conf.Configuration;
28  import org.codehaus.jackson.JsonGenerationException;
29  import org.codehaus.jackson.annotate.JsonIgnoreProperties;
30  import org.codehaus.jackson.map.JsonMappingException;
31  import org.codehaus.jackson.map.ObjectMapper;
32  import org.codehaus.jackson.map.SerializationConfig;
33  
34  import com.yammer.metrics.core.Histogram;
35  import com.yammer.metrics.core.MetricsRegistry;
36  import com.yammer.metrics.stats.Snapshot;
37  
38  /**
39   * Utilty for aggregating counts in CachedBlocks and toString/toJSON CachedBlocks and BlockCaches.
40   * No attempt has been made at making this thread safe.
41   */
42  @InterfaceAudience.Private
43  public class BlockCacheUtil {
44  
45    public static final long NANOS_PER_SECOND = 1000000000;
46    /**
47     * Needed making histograms.
48     */
49    private static final MetricsRegistry METRICS = new MetricsRegistry();
50  
51    /**
52     * Needed generating JSON.
53     */
54    private static final ObjectMapper MAPPER = new ObjectMapper();
55    static {
56      MAPPER.configure(SerializationConfig.Feature.FAIL_ON_EMPTY_BEANS, false);
57      MAPPER.configure(SerializationConfig.Feature.FLUSH_AFTER_WRITE_VALUE, true);
58      MAPPER.configure(SerializationConfig.Feature.INDENT_OUTPUT, true);
59    }
60  
61    /**
62     * @param cb
63     * @return The block content as String.
64     */
65    public static String toString(final CachedBlock cb, final long now) {
66      return "filename=" + cb.getFilename() + ", " + toStringMinusFileName(cb, now);
67    }
68  
69    /**
70     * Little data structure to hold counts for a file.
71     * Used doing a toJSON.
72     */
73    static class CachedBlockCountsPerFile {
74      private int count = 0;
75      private long size = 0;
76      private int countData = 0;
77      private long sizeData = 0;
78      private final String filename;
79  
80      CachedBlockCountsPerFile(final String filename) {
81        this.filename = filename;
82      }
83  
84      public int getCount() {
85        return count;
86      }
87  
88      public long getSize() {
89        return size;
90      }
91  
92      public int getCountData() {
93        return countData;
94      }
95  
96      public long getSizeData() {
97        return sizeData;
98      }
99  
100     public String getFilename() {
101       return filename;
102     }
103   }
104 
105   /**
106    * @param filename
107    * @param blocks
108    * @return A JSON String of <code>filename</code> and counts of <code>blocks</code>
109    * @throws JsonGenerationException
110    * @throws JsonMappingException
111    * @throws IOException
112    */
113   public static String toJSON(final String filename, final NavigableSet<CachedBlock> blocks)
114   throws JsonGenerationException, JsonMappingException, IOException {
115     CachedBlockCountsPerFile counts = new CachedBlockCountsPerFile(filename);
116     for (CachedBlock cb: blocks) {
117       counts.count++;
118       counts.size += cb.getSize();
119       BlockType bt = cb.getBlockType();
120       if (bt != null && bt.isData()) {
121         counts.countData++;
122         counts.sizeData += cb.getSize();
123       }
124     }
125     return MAPPER.writeValueAsString(counts);
126   }
127 
128   /**
129    * @param cbsbf
130    * @return JSON string of <code>cbsf</code> aggregated
131    * @throws JsonGenerationException
132    * @throws JsonMappingException
133    * @throws IOException
134    */
135   public static String toJSON(final CachedBlocksByFile cbsbf)
136   throws JsonGenerationException, JsonMappingException, IOException {
137     return MAPPER.writeValueAsString(cbsbf);
138   }
139 
140   /**
141    * @param bc
142    * @return JSON string of <code>bc</code> content.
143    * @throws JsonGenerationException
144    * @throws JsonMappingException
145    * @throws IOException
146    */
147   public static String toJSON(final BlockCache bc)
148   throws JsonGenerationException, JsonMappingException, IOException {
149     return MAPPER.writeValueAsString(bc);
150   }
151 
152   /**
153    * @param cb
154    * @return The block content of <code>bc</code> as a String minus the filename.
155    */
156   public static String toStringMinusFileName(final CachedBlock cb, final long now) {
157     return "offset=" + cb.getOffset() +
158       ", size=" + cb.getSize() +
159       ", age=" + (now - cb.getCachedTime()) +
160       ", type=" + cb.getBlockType() +
161       ", priority=" + cb.getBlockPriority();
162   }
163 
164   /**
165    * Get a {@link CachedBlocksByFile} instance and load it up by iterating content in
166    * {@link BlockCache}.
167    * @param conf Used to read configurations
168    * @param bc Block Cache to iterate.
169    * @return Laoded up instance of CachedBlocksByFile
170    */
171   public static CachedBlocksByFile getLoadedCachedBlocksByFile(final Configuration conf,
172       final BlockCache bc) {
173     CachedBlocksByFile cbsbf = new CachedBlocksByFile(conf);
174     for (CachedBlock cb: bc) {
175       if (cbsbf.update(cb)) break;
176     }
177     return cbsbf;
178   }
179 
180   /**
181    * Use one of these to keep a running account of cached blocks by file.  Throw it away when done.
182    * This is different than metrics in that it is stats on current state of a cache.
183    * See getLoadedCachedBlocksByFile
184    */
185   @JsonIgnoreProperties({"cachedBlockStatsByFile"})
186   public static class CachedBlocksByFile {
187     private int count;
188     private int dataBlockCount;
189     private long size;
190     private long dataSize;
191     private final long now = System.nanoTime();
192     private final int max;
193     public static final int DEFAULT_MAX = 100000;
194  
195     CachedBlocksByFile() {
196       this(null);
197     }
198 
199     CachedBlocksByFile(final Configuration c) {
200       this.max = c == null? DEFAULT_MAX:
201         c.getInt("hbase.ui.blockcache.by.file.max", DEFAULT_MAX);
202     }
203 
204     /**
205      * Map by filename. use concurent utils because we want our Map and contained blocks sorted.
206      */
207     private NavigableMap<String, NavigableSet<CachedBlock>> cachedBlockByFile =
208       new ConcurrentSkipListMap<String, NavigableSet<CachedBlock>>();
209     Histogram age = METRICS.newHistogram(CachedBlocksByFile.class, "age");
210 
211     /**
212      * @param cb
213      * @return True if full.... if we won't be adding any more.
214      */
215     public boolean update(final CachedBlock cb) {
216       if (isFull()) return true;
217       NavigableSet<CachedBlock> set = this.cachedBlockByFile.get(cb.getFilename());
218       if (set == null) {
219         set = new ConcurrentSkipListSet<CachedBlock>();
220         this.cachedBlockByFile.put(cb.getFilename(), set);
221       }
222       set.add(cb);
223       this.size += cb.getSize();
224       this.count++;
225       BlockType bt = cb.getBlockType();
226       if (bt != null && bt.isData()) {
227         this.dataBlockCount++;
228         this.dataSize += cb.getSize();
229       }
230       long age = (this.now - cb.getCachedTime())/NANOS_PER_SECOND;
231       this.age.update(age);
232       return false;
233     }
234 
235     /**
236      * @return True if full; i.e. there are more items in the cache but we only loaded up
237      * the maximum set in configuration <code>hbase.ui.blockcache.by.file.max</code>
238      * (Default: DEFAULT_MAX).
239      */
240     public boolean isFull() {
241       return this.count >= this.max;
242     }
243  
244     public NavigableMap<String, NavigableSet<CachedBlock>> getCachedBlockStatsByFile() {
245       return this.cachedBlockByFile;
246     }
247 
248     /**
249      * @return count of blocks in the cache
250      */
251     public int getCount() {
252       return count;
253     }
254 
255     public int getDataCount() {
256       return dataBlockCount;
257     }
258 
259     /**
260      * @return size of blocks in the cache
261      */
262     public long getSize() {
263       return size;
264     }
265 
266     /**
267      * @return Size of data.
268      */
269     public long getDataSize() {
270       return dataSize;
271     }
272 
273     public AgeSnapshot getAgeInCacheSnapshot() {
274       return new AgeSnapshot(this.age);
275     }
276 
277     @Override
278     public String toString() {
279       Snapshot snapshot = this.age.getSnapshot();
280       return "count=" + count + ", dataBlockCount=" + this.dataBlockCount + ", size=" + size +
281           ", dataSize=" + getDataSize() +
282           ", mean age=" + this.age.mean() + ", stddev age=" + this.age.stdDev() +
283           ", min age=" + this.age.min() + ", max age=" + this.age.max() +
284           ", 95th percentile age=" + snapshot.get95thPercentile() +
285           ", 99th percentile age=" + snapshot.get99thPercentile();
286     }
287   }
288 }