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 = new ConcurrentHashMap<>(); 047 048 public MetricsTableWrapperAggregateImpl(final HRegionServer regionServer) { 049 this.regionServer = regionServer; 050 this.period = regionServer.conf.getLong(HConstants.REGIONSERVER_METRICS_PERIOD, 051 HConstants.DEFAULT_REGIONSERVER_METRICS_PERIOD) + 1000; 052 this.executor = CompatibilitySingletonFactory.getInstance(MetricsExecutor.class).getExecutor(); 053 this.runnable = new TableMetricsWrapperRunnable(); 054 this.tableMetricsUpdateTask = this.executor.scheduleWithFixedDelay(this.runnable, period, this.period, 055 TimeUnit.MILLISECONDS); 056 } 057 058 public class TableMetricsWrapperRunnable implements Runnable { 059 060 @Override 061 public void run() { 062 Map<TableName, MetricsTableValues> localMetricsTableMap = new HashMap<>(); 063 064 for (Region r : regionServer.getOnlineRegionsLocalContext()) { 065 TableName tbl= r.getTableDescriptor().getTableName(); 066 MetricsTableValues metricsTable = localMetricsTableMap.get(tbl); 067 if (metricsTable == null) { 068 metricsTable = new MetricsTableValues(); 069 localMetricsTableMap.put(tbl, metricsTable); 070 } 071 long tempStorefilesSize = 0; 072 for (Store store : r.getStores()) { 073 tempStorefilesSize += store.getStorefilesSize(); 074 } 075 metricsTable.setMemStoresSize(metricsTable.getMemStoresSize() + r.getMemStoreDataSize()); 076 metricsTable.setStoreFilesSize(metricsTable.getStoreFilesSize() + tempStorefilesSize); 077 metricsTable.setTableSize(metricsTable.getMemStoresSize() + metricsTable.getStoreFilesSize()); 078 metricsTable.setReadRequestsCount(metricsTable.getReadRequestsCount() + r.getReadRequestsCount()); 079 metricsTable.setWriteRequestsCount(metricsTable.getWriteRequestsCount() + r.getWriteRequestsCount()); 080 metricsTable.setTotalRequestsCount(metricsTable.getReadRequestsCount() + metricsTable.getWriteRequestsCount()); 081 } 082 083 for(Map.Entry<TableName, MetricsTableValues> entry : localMetricsTableMap.entrySet()) { 084 TableName tbl = entry.getKey(); 085 if (metricsTableMap.get(tbl) == null) { 086 MetricsTableSource tableSource = CompatibilitySingletonFactory 087 .getInstance(MetricsRegionServerSourceFactory.class).createTable(tbl.getNameAsString(), 088 MetricsTableWrapperAggregateImpl.this); 089 CompatibilitySingletonFactory 090 .getInstance(MetricsRegionServerSourceFactory.class).getTableAggregate() 091 .register(tbl.getNameAsString(), tableSource); 092 } 093 metricsTableMap.put(entry.getKey(), entry.getValue()); 094 } 095 Set<TableName> existingTableNames = Sets.newHashSet(metricsTableMap.keySet()); 096 existingTableNames.removeAll(localMetricsTableMap.keySet()); 097 MetricsTableAggregateSource agg = CompatibilitySingletonFactory 098 .getInstance(MetricsRegionServerSourceFactory.class).getTableAggregate(); 099 for (TableName table : existingTableNames) { 100 agg.deregister(table.getNameAsString()); 101 if (metricsTableMap.get(table) != null) { 102 metricsTableMap.remove(table); 103 } 104 } 105 } 106 } 107 108 @Override 109 public long getReadRequestsCount(String table) { 110 MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table)); 111 if (metricsTable == null) 112 return 0; 113 else 114 return metricsTable.getReadRequestsCount(); 115 } 116 117 @Override 118 public long getWriteRequestsCount(String table) { 119 MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table)); 120 if (metricsTable == null) 121 return 0; 122 else 123 return metricsTable.getWriteRequestsCount(); 124 } 125 126 @Override 127 public long getTotalRequestsCount(String table) { 128 MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table)); 129 if (metricsTable == null) 130 return 0; 131 else 132 return metricsTable.getTotalRequestsCount(); 133 } 134 135 @Override 136 public long getMemStoresSize(String table) { 137 MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table)); 138 if (metricsTable == null) 139 return 0; 140 else 141 return metricsTable.getMemStoresSize(); 142 } 143 144 @Override 145 public long getStoreFilesSize(String table) { 146 MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table)); 147 if (metricsTable == null) 148 return 0; 149 else 150 return metricsTable.getStoreFilesSize(); 151 } 152 153 @Override 154 public long getTableSize(String table) { 155 MetricsTableValues metricsTable = metricsTableMap.get(TableName.valueOf(table)); 156 if (metricsTable == null) 157 return 0; 158 else 159 return metricsTable.getTableSize(); 160 } 161 162 @Override 163 public void close() throws IOException { 164 tableMetricsUpdateTask.cancel(true); 165 } 166 167 private static class MetricsTableValues { 168 169 private long totalRequestsCount; 170 private long readRequestsCount; 171 private long writeRequestsCount; 172 private long memstoresSize; 173 private long storeFilesSize; 174 private long tableSize; 175 176 public long getTotalRequestsCount() { 177 return totalRequestsCount; 178 } 179 180 public void setTotalRequestsCount(long totalRequestsCount) { 181 this.totalRequestsCount = totalRequestsCount; 182 } 183 184 public long getReadRequestsCount() { 185 return readRequestsCount; 186 } 187 188 public void setReadRequestsCount(long readRequestsCount) { 189 this.readRequestsCount = readRequestsCount; 190 } 191 192 public long getWriteRequestsCount() { 193 return writeRequestsCount; 194 } 195 196 public void setWriteRequestsCount(long writeRequestsCount) { 197 this.writeRequestsCount = writeRequestsCount; 198 } 199 200 public long getMemStoresSize() { 201 return memstoresSize; 202 } 203 204 public void setMemStoresSize(long memstoresSize) { 205 this.memstoresSize = memstoresSize; 206 } 207 208 public long getStoreFilesSize() { 209 return storeFilesSize; 210 } 211 212 public void setStoreFilesSize(long storeFilesSize) { 213 this.storeFilesSize = storeFilesSize; 214 } 215 216 public long getTableSize() { 217 return tableSize; 218 } 219 220 public void setTableSize(long tableSize) { 221 this.tableSize = tableSize; 222 } 223 } 224 225}