001/** 002 * Licensed to the Apache Software Foundation (ASF) under one or more contributor license 003 * agreements. See the NOTICE file distributed with this work for additional information regarding 004 * copyright ownership. The ASF licenses this file to you under the Apache License, Version 2.0 (the 005 * "License"); you may not use this file except in compliance with the License. You may obtain a 006 * copy of the License at http://www.apache.org/licenses/LICENSE-2.0 Unless required by applicable 007 * law or agreed to in writing, software distributed under the License is distributed on an "AS IS" 008 * BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the License 009 * for the specific language governing permissions and limitations under the License. 010 */ 011 012package org.apache.hadoop.hbase.coprocessor; 013 014import java.io.IOException; 015import java.nio.charset.StandardCharsets; 016import java.util.List; 017import java.util.Map; 018import java.util.Optional; 019import java.util.Set; 020import java.util.concurrent.ConcurrentHashMap; 021import org.apache.hadoop.hbase.Cell; 022import org.apache.hadoop.hbase.CoprocessorEnvironment; 023import org.apache.hadoop.hbase.TableName; 024import org.apache.hadoop.hbase.client.Delete; 025import org.apache.hadoop.hbase.client.Durability; 026import org.apache.hadoop.hbase.client.Get; 027import org.apache.hadoop.hbase.client.Put; 028import org.apache.hadoop.hbase.client.Row; 029import org.apache.hadoop.hbase.ipc.RpcServer; 030import org.apache.hadoop.hbase.metrics.Meter; 031import org.apache.hadoop.hbase.metrics.Metric; 032import org.apache.hadoop.hbase.metrics.MetricRegistry; 033import org.apache.hadoop.hbase.util.LossyCounting; 034import org.apache.hadoop.hbase.wal.WALEdit; 035import org.apache.yetus.audience.InterfaceAudience; 036import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableMap; 037 038 039/** 040 * A coprocessor that collects metrics from meta table. 041 * <p> 042 * These metrics will be available through the regular Hadoop metrics2 sinks (ganglia, opentsdb, 043 * etc) as well as JMX output. 044 * </p> 045 * @see MetaTableMetrics 046 */ 047 048@InterfaceAudience.Private 049public class MetaTableMetrics implements RegionCoprocessor { 050 051 private ExampleRegionObserverMeta observer; 052 private Map<String, Optional<Metric>> requestsMap; 053 private RegionCoprocessorEnvironment regionCoprocessorEnv; 054 private LossyCounting clientMetricsLossyCounting; 055 private boolean active = false; 056 057 enum MetaTableOps { 058 GET, PUT, DELETE; 059 } 060 061 private ImmutableMap<Class, MetaTableOps> opsNameMap = 062 ImmutableMap.<Class, MetaTableOps>builder() 063 .put(Put.class, MetaTableOps.PUT) 064 .put(Get.class, MetaTableOps.GET) 065 .put(Delete.class, MetaTableOps.DELETE) 066 .build(); 067 068 class ExampleRegionObserverMeta implements RegionCoprocessor, RegionObserver { 069 070 @Override 071 public Optional<RegionObserver> getRegionObserver() { 072 return Optional.of(this); 073 } 074 075 @Override 076 public void preGetOp(ObserverContext<RegionCoprocessorEnvironment> e, Get get, 077 List<Cell> results) throws IOException { 078 if (!active || !isMetaTableOp(e)) { 079 return; 080 } 081 tableMetricRegisterAndMark(e, get); 082 clientMetricRegisterAndMark(e); 083 regionMetricRegisterAndMark(e, get); 084 opMetricRegisterAndMark(e, get); 085 opWithClientMetricRegisterAndMark(e, get); 086 } 087 088 @Override 089 public void prePut(ObserverContext<RegionCoprocessorEnvironment> e, Put put, WALEdit edit, 090 Durability durability) throws IOException { 091 if (!active || !isMetaTableOp(e)) { 092 return; 093 } 094 tableMetricRegisterAndMark(e, put); 095 clientMetricRegisterAndMark(e); 096 regionMetricRegisterAndMark(e, put); 097 opMetricRegisterAndMark(e, put); 098 opWithClientMetricRegisterAndMark(e, put); 099 } 100 101 @Override 102 public void preDelete(ObserverContext<RegionCoprocessorEnvironment> e, Delete delete, 103 WALEdit edit, Durability durability) throws IOException { 104 if (!active || !isMetaTableOp(e)) { 105 return; 106 } 107 tableMetricRegisterAndMark(e, delete); 108 clientMetricRegisterAndMark(e); 109 regionMetricRegisterAndMark(e, delete); 110 opMetricRegisterAndMark(e, delete); 111 opWithClientMetricRegisterAndMark(e, delete); 112 } 113 114 private void markMeterIfPresent(String requestMeter) { 115 if (requestMeter.isEmpty()) { 116 return; 117 } 118 Metric metric = 119 requestsMap.get(requestMeter).isPresent() ? requestsMap.get(requestMeter).get() : null; 120 if (metric != null) { 121 ((Meter) metric).mark(); 122 } 123 } 124 125 private void registerMeterIfNotPresent(ObserverContext<RegionCoprocessorEnvironment> e, 126 String requestMeter) { 127 if (requestMeter.isEmpty()) { 128 return; 129 } 130 if (!requestsMap.containsKey(requestMeter)) { 131 MetricRegistry registry = regionCoprocessorEnv.getMetricRegistryForRegionServer(); 132 registry.meter(requestMeter); 133 requestsMap.put(requestMeter, registry.get(requestMeter)); 134 } 135 } 136 137 /** 138 * Registers and counts lossyCount for Meters that kept by lossy counting. 139 * By using lossy count to maintain meters, at most 7 / e meters will be kept (e is error rate) 140 * e.g. when e is 0.02 by default, at most 50 Clients request metrics will be kept 141 * also, all kept elements have frequency higher than e * N. (N is total count) 142 * @param e Region coprocessor environment 143 * @param requestMeter meter to be registered 144 * @param lossyCounting lossyCounting object for one type of meters. 145 */ 146 private void registerLossyCountingMeterIfNotPresent( 147 ObserverContext<RegionCoprocessorEnvironment> e, 148 String requestMeter, LossyCounting lossyCounting) { 149 if (requestMeter.isEmpty()) { 150 return; 151 } 152 Set<String> metersToBeRemoved = lossyCounting.addByOne(requestMeter); 153 if(!requestsMap.containsKey(requestMeter) && metersToBeRemoved.contains(requestMeter)){ 154 for(String meter: metersToBeRemoved) { 155 //cleanup requestsMap according swept data from lossy count; 156 requestsMap.remove(meter); 157 MetricRegistry registry = regionCoprocessorEnv.getMetricRegistryForRegionServer(); 158 registry.remove(meter); 159 } 160 // newly added meter is swept by lossy counting cleanup. No need to put it into requestsMap. 161 return; 162 } 163 164 if (!requestsMap.containsKey(requestMeter)) { 165 MetricRegistry registry = regionCoprocessorEnv.getMetricRegistryForRegionServer(); 166 registry.meter(requestMeter); 167 requestsMap.put(requestMeter, registry.get(requestMeter)); 168 } 169 } 170 171 /** 172 * Get table name from Ops such as: get, put, delete. 173 * @param op such as get, put or delete. 174 */ 175 private String getTableNameFromOp(Row op) { 176 String tableName = null; 177 String tableRowKey = new String(((Row) op).getRow(), StandardCharsets.UTF_8); 178 if (tableRowKey.isEmpty()) { 179 return null; 180 } 181 tableName = tableRowKey.split(",").length > 0 ? tableRowKey.split(",")[0] : null; 182 return tableName; 183 } 184 185 /** 186 * Get regionId from Ops such as: get, put, delete. 187 * @param op such as get, put or delete. 188 */ 189 private String getRegionIdFromOp(Row op) { 190 String regionId = null; 191 String tableRowKey = new String(((Row) op).getRow(), StandardCharsets.UTF_8); 192 if (tableRowKey.isEmpty()) { 193 return null; 194 } 195 regionId = tableRowKey.split(",").length > 2 ? tableRowKey.split(",")[2] : null; 196 return regionId; 197 } 198 199 private boolean isMetaTableOp(ObserverContext<RegionCoprocessorEnvironment> e) { 200 return TableName.META_TABLE_NAME 201 .equals(e.getEnvironment().getRegionInfo().getTable()); 202 } 203 204 private void clientMetricRegisterAndMark(ObserverContext<RegionCoprocessorEnvironment> e) { 205 String clientIP = RpcServer.getRemoteIp() != null ? RpcServer.getRemoteIp().toString() : ""; 206 207 String clientRequestMeter = clientRequestMeterName(clientIP); 208 registerLossyCountingMeterIfNotPresent(e, clientRequestMeter, clientMetricsLossyCounting); 209 markMeterIfPresent(clientRequestMeter); 210 } 211 212 private void tableMetricRegisterAndMark(ObserverContext<RegionCoprocessorEnvironment> e, 213 Row op) { 214 // Mark the meta table meter whenever the coprocessor is called 215 String tableName = getTableNameFromOp(op); 216 String tableRequestMeter = tableMeterName(tableName); 217 registerMeterIfNotPresent(e, tableRequestMeter); 218 markMeterIfPresent(tableRequestMeter); 219 } 220 221 private void regionMetricRegisterAndMark(ObserverContext<RegionCoprocessorEnvironment> e, 222 Row op) { 223 // Mark the meta table meter whenever the coprocessor is called 224 String regionId = getRegionIdFromOp(op); 225 String regionRequestMeter = regionMeterName(regionId); 226 registerMeterIfNotPresent(e, regionRequestMeter); 227 markMeterIfPresent(regionRequestMeter); 228 } 229 230 private void opMetricRegisterAndMark(ObserverContext<RegionCoprocessorEnvironment> e, 231 Row op) { 232 String opMeterName = opMeterName(op); 233 registerMeterIfNotPresent(e, opMeterName); 234 markMeterIfPresent(opMeterName); 235 } 236 237 private void opWithClientMetricRegisterAndMark(ObserverContext<RegionCoprocessorEnvironment> e, 238 Object op) { 239 String opWithClientMeterName = opWithClientMeterName(op); 240 registerMeterIfNotPresent(e, opWithClientMeterName); 241 markMeterIfPresent(opWithClientMeterName); 242 } 243 244 private String opWithClientMeterName(Object op) { 245 String clientIP = RpcServer.getRemoteIp() != null ? RpcServer.getRemoteIp().toString() : ""; 246 if (clientIP.isEmpty()) { 247 return ""; 248 } 249 MetaTableOps ops = opsNameMap.get(op.getClass()); 250 String opWithClientMeterName = ""; 251 switch (ops) { 252 case GET: 253 opWithClientMeterName = String.format("MetaTable_client_%s_get_request", clientIP); 254 break; 255 case PUT: 256 opWithClientMeterName = String.format("MetaTable_client_%s_put_request", clientIP); 257 break; 258 case DELETE: 259 opWithClientMeterName = String.format("MetaTable_client_%s_delete_request", clientIP); 260 break; 261 default: 262 break; 263 } 264 return opWithClientMeterName; 265 } 266 267 private String opMeterName(Object op) { 268 MetaTableOps ops = opsNameMap.get(op.getClass()); 269 String opMeterName = ""; 270 switch (ops) { 271 case GET: 272 opMeterName = "MetaTable_get_request"; 273 break; 274 case PUT: 275 opMeterName = "MetaTable_put_request"; 276 break; 277 case DELETE: 278 opMeterName = "MetaTable_delete_request"; 279 break; 280 default: 281 break; 282 } 283 return opMeterName; 284 } 285 286 private String tableMeterName(String tableName) { 287 return String.format("MetaTable_table_%s_request", tableName); 288 } 289 290 private String clientRequestMeterName(String clientIP) { 291 if (clientIP.isEmpty()) { 292 return ""; 293 } 294 return String.format("MetaTable_client_%s_request", clientIP); 295 } 296 297 private String regionMeterName(String regionId) { 298 return String.format("MetaTable_region_%s_request", regionId); 299 } 300 } 301 302 @Override 303 public Optional<RegionObserver> getRegionObserver() { 304 return Optional.of(observer); 305 } 306 307 @Override 308 public void start(CoprocessorEnvironment env) throws IOException { 309 if (env instanceof RegionCoprocessorEnvironment 310 && ((RegionCoprocessorEnvironment) env).getRegionInfo().getTable() != null 311 && ((RegionCoprocessorEnvironment) env).getRegionInfo().getTable() 312 .equals(TableName.META_TABLE_NAME)) { 313 regionCoprocessorEnv = (RegionCoprocessorEnvironment) env; 314 observer = new ExampleRegionObserverMeta(); 315 requestsMap = new ConcurrentHashMap<>(); 316 clientMetricsLossyCounting = new LossyCounting(); 317 // only be active mode when this region holds meta table. 318 active = true; 319 } else { 320 observer = new ExampleRegionObserverMeta(); 321 } 322 } 323 324 @Override 325 public void stop(CoprocessorEnvironment env) throws IOException { 326 // since meta region can move around, clear stale metrics when stop. 327 if (requestsMap != null) { 328 for (String meterName : requestsMap.keySet()) { 329 MetricRegistry registry = regionCoprocessorEnv.getMetricRegistryForRegionServer(); 330 registry.remove(meterName); 331 } 332 } 333 } 334 335}