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 */
018
019package org.apache.hadoop.hbase.regionserver;
020
021import java.io.Closeable;
022import java.io.IOException;
023import java.util.HashMap;
024import java.util.Map;
025import java.util.Set;
026import java.util.concurrent.ConcurrentHashMap;
027import java.util.concurrent.ScheduledExecutorService;
028import java.util.concurrent.ScheduledFuture;
029import java.util.concurrent.TimeUnit;
030
031import org.apache.hadoop.hbase.CompatibilitySingletonFactory;
032import org.apache.hadoop.hbase.HConstants;
033import org.apache.hadoop.hbase.TableName;
034import org.apache.yetus.audience.InterfaceAudience;
035import org.apache.hadoop.metrics2.MetricsExecutor;
036
037import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
038
039@InterfaceAudience.Private
040public class MetricsTableWrapperAggregateImpl implements MetricsTableWrapperAggregate, Closeable {
041  private final HRegionServer regionServer;
042  private ScheduledExecutorService executor;
043  private Runnable runnable;
044  private long period;
045  private ScheduledFuture<?> tableMetricsUpdateTask;
046  private ConcurrentHashMap<TableName, MetricsTableValues> metricsTableMap
047    = new ConcurrentHashMap<>();
048
049  public MetricsTableWrapperAggregateImpl(final HRegionServer regionServer) {
050    this.regionServer = regionServer;
051    this.period = regionServer.getConfiguration().getLong(HConstants.REGIONSERVER_METRICS_PERIOD,
052      HConstants.DEFAULT_REGIONSERVER_METRICS_PERIOD) + 1000;
053    this.executor = CompatibilitySingletonFactory.getInstance(MetricsExecutor.class).getExecutor();
054    this.runnable = new TableMetricsWrapperRunnable();
055    this.tableMetricsUpdateTask = this.executor.scheduleWithFixedDelay(this.runnable, period,
056      period, TimeUnit.MILLISECONDS);
057  }
058
059  public class TableMetricsWrapperRunnable implements Runnable {
060
061    @Override
062    public void run() {
063      Map<TableName, MetricsTableValues> localMetricsTableMap = new HashMap<>();
064      for (Region r : regionServer.getOnlineRegionsLocalContext()) {
065        TableName tbl = r.getTableDescriptor().getTableName();
066        MetricsTableValues mt = localMetricsTableMap.get(tbl);
067        if (mt == null) {
068          mt = new MetricsTableValues();
069          localMetricsTableMap.put(tbl, mt);
070        }
071        long memstoreReadCount = 0L;
072        long mixedReadCount = 0L;
073        String tempKey = null;
074        if (r.getStores() != null) {
075          String familyName = null;
076          for (Store store : r.getStores()) {
077            familyName = store.getColumnFamilyName();
078
079            mt.storeFileCount += store.getStorefilesCount();
080            mt.memstoreSize += (store.getMemStoreSize().getDataSize()
081                + store.getMemStoreSize().getHeapSize() + store.getMemStoreSize().getOffHeapSize());
082            mt.storeFileSize += store.getStorefilesSize();
083            mt.referenceFileCount += store.getNumReferenceFiles();
084            if (store.getMaxStoreFileAge().isPresent()) {
085              mt.maxStoreFileAge =
086                  Math.max(mt.maxStoreFileAge, store.getMaxStoreFileAge().getAsLong());
087            }
088            if (store.getMinStoreFileAge().isPresent()) {
089              mt.minStoreFileAge =
090                  Math.min(mt.minStoreFileAge, store.getMinStoreFileAge().getAsLong());
091            }
092            if (store.getAvgStoreFileAge().isPresent()) {
093              mt.totalStoreFileAge =
094                  (long) store.getAvgStoreFileAge().getAsDouble() * store.getStorefilesCount();
095            }
096            mt.storeCount += 1;
097            tempKey = tbl.getNameAsString() + UNDERSCORE + familyName;
098            Long tempVal = mt.perStoreMemstoreOnlyReadCount.get(tempKey);
099            if (tempVal == null) {
100              tempVal = 0L;
101            }
102            memstoreReadCount = store.getMemstoreOnlyRowReadsCount() + tempVal;
103            tempVal = mt.perStoreMixedReadCount.get(tempKey);
104            if (tempVal == null) {
105              tempVal = 0L;
106            }
107            mixedReadCount = store.getMixedRowReadsCount() + tempVal;
108            // accumulate the count
109            mt.perStoreMemstoreOnlyReadCount.put(tempKey, memstoreReadCount);
110            mt.perStoreMixedReadCount.put(tempKey, mixedReadCount);
111          }
112
113          mt.regionCount += 1;
114
115          mt.readRequestCount += r.getReadRequestsCount();
116          mt.filteredReadRequestCount += r.getFilteredReadRequestsCount();
117          mt.writeRequestCount += r.getWriteRequestsCount();
118        }
119      }
120
121      for (Map.Entry<TableName, MetricsTableValues> entry : localMetricsTableMap.entrySet()) {
122        TableName tbl = entry.getKey();
123        if (metricsTableMap.get(tbl) == null) {
124          // this will add the Wrapper to the list of TableMetrics
125          CompatibilitySingletonFactory
126              .getInstance(MetricsRegionServerSourceFactory.class)
127              .getTableAggregate()
128              .getOrCreateTableSource(tbl.getNameAsString(), MetricsTableWrapperAggregateImpl.this);
129        }
130        metricsTableMap.put(entry.getKey(), entry.getValue());
131      }
132      Set<TableName> existingTableNames = Sets.newHashSet(metricsTableMap.keySet());
133      existingTableNames.removeAll(localMetricsTableMap.keySet());
134      MetricsTableAggregateSource agg = CompatibilitySingletonFactory
135          .getInstance(MetricsRegionServerSourceFactory.class).getTableAggregate();
136      for (TableName table : existingTableNames) {
137        agg.deleteTableSource(table.getNameAsString());
138        if (metricsTableMap.get(table) != null) {
139          metricsTableMap.remove(table);
140        }
141      }
142    }
143  }
144
145  @Override
146  public long getReadRequestCount(String table) {
147    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
148    if (metricsTable == null) {
149      return 0;
150    } else {
151      return metricsTable.readRequestCount;
152    }
153  }
154
155  @Override
156  public Map<String, Long> getMemstoreOnlyRowReadsCount(String table) {
157    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
158    if (metricsTable == null) {
159      return null;
160    } else {
161      return metricsTable.perStoreMemstoreOnlyReadCount;
162    }
163  }
164
165  @Override
166  public Map<String, Long> getMixedRowReadsCount(String table) {
167    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
168    if (metricsTable == null) {
169      return null;
170    } else {
171      return metricsTable.perStoreMixedReadCount;
172    }
173  }
174
175  public long getCpRequestsCount(String table) {
176    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
177    if (metricsTable == null) {
178      return 0;
179    } else {
180      return metricsTable.cpRequestCount;
181    }
182  }
183
184  public long getFilteredReadRequestCount(String table) {
185    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
186    if (metricsTable == null) {
187      return 0;
188    }
189    return metricsTable.filteredReadRequestCount;
190  }
191
192  @Override
193  public long getWriteRequestCount(String table) {
194    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
195    if (metricsTable == null) {
196      return 0;
197    } else {
198      return metricsTable.writeRequestCount;
199    }
200  }
201
202  @Override
203  public long getTotalRequestsCount(String table) {
204    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
205    if (metricsTable == null) {
206      return 0;
207    } else {
208      return metricsTable.readRequestCount + metricsTable.writeRequestCount;
209    }
210  }
211
212  @Override
213  public long getMemStoreSize(String table) {
214    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
215    if (metricsTable == null) {
216      return 0;
217    } else {
218      return metricsTable.memstoreSize;
219    }
220  }
221
222  @Override
223  public long getStoreFileSize(String table) {
224    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
225    if (metricsTable == null) {
226      return 0;
227    } else {
228      return metricsTable.storeFileSize;
229    }
230  }
231
232  @Override
233  public long getTableSize(String table) {
234    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
235    if (metricsTable == null) {
236      return 0;
237    } else {
238      return metricsTable.memstoreSize + metricsTable.storeFileSize;
239    }
240  }
241
242  public long getNumRegions(String table) {
243    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
244    if (metricsTable == null) {
245      return 0;
246    }
247    return metricsTable.regionCount;
248  }
249
250  @Override
251  public long getNumStores(String table) {
252    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
253    if (metricsTable == null) {
254      return 0;
255    }
256    return metricsTable.storeCount;
257  }
258
259  @Override
260  public long getNumStoreFiles(String table) {
261    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
262    if (metricsTable == null) {
263      return 0;
264    }
265    return metricsTable.storeFileCount;
266  }
267
268  @Override
269  public long getMaxStoreFileAge(String table) {
270    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
271    if (metricsTable == null) {
272      return 0;
273    }
274    return metricsTable.maxStoreFileAge;
275  }
276
277  @Override
278  public long getMinStoreFileAge(String table) {
279    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
280    if (metricsTable == null) {
281      return 0;
282    }
283    return metricsTable.minStoreFileAge == Long.MAX_VALUE ? 0 : metricsTable.minStoreFileAge;
284  }
285
286  @Override
287  public long getAvgStoreFileAge(String table) {
288    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
289    if (metricsTable == null) {
290      return 0;
291    }
292
293    return metricsTable.storeFileCount == 0
294        ? 0
295        : (metricsTable.totalStoreFileAge / metricsTable.storeFileCount);
296  }
297
298  @Override
299  public long getNumReferenceFiles(String table) {
300    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
301    if (metricsTable == null) {
302      return 0;
303    }
304    return metricsTable.referenceFileCount;
305  }
306
307  @Override
308  public long getAvgRegionSize(String table) {
309    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
310    if (metricsTable == null) {
311      return 0;
312    }
313    return metricsTable.regionCount == 0
314        ? 0
315        : (metricsTable.memstoreSize + metricsTable.storeFileSize) / metricsTable.regionCount;
316  }
317
318  public long getCpRequestCount(String table) {
319    MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table));
320    if (metricsTable == null) {
321      return 0;
322    }
323    return metricsTable.cpRequestCount;
324  }
325
326  @Override
327  public void close() throws IOException {
328    tableMetricsUpdateTask.cancel(true);
329  }
330
331  private static class MetricsTableValues {
332    long readRequestCount;
333    long filteredReadRequestCount;
334    long writeRequestCount;
335    long memstoreSize;
336    long regionCount;
337    long storeCount;
338    long storeFileCount;
339    long storeFileSize;
340    long maxStoreFileAge;
341    long minStoreFileAge = Long.MAX_VALUE;
342    long totalStoreFileAge;
343    long referenceFileCount;
344    long cpRequestCount;
345    Map<String, Long> perStoreMemstoreOnlyReadCount = new HashMap<>();
346    Map<String, Long> perStoreMixedReadCount = new HashMap<>();
347  }
348
349}