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.util; 020 021import java.io.IOException; 022import org.apache.hadoop.conf.Configuration; 023import org.apache.hadoop.fs.FileSystem; 024import org.apache.hadoop.fs.Path; 025import org.apache.hadoop.hbase.ReplicationPeerNotFoundException; 026import org.apache.hadoop.hbase.client.Admin; 027import org.apache.hadoop.hbase.client.ConnectionFactory; 028import org.apache.hadoop.hbase.client.RegionInfo; 029import org.apache.hadoop.hbase.client.RegionReplicaUtil; 030import org.apache.hadoop.hbase.io.HFileLink; 031import org.apache.hadoop.hbase.io.Reference; 032import org.apache.hadoop.hbase.regionserver.HRegion; 033import org.apache.hadoop.hbase.regionserver.StoreFileInfo; 034import org.apache.hadoop.hbase.replication.ReplicationPeerConfig; 035import org.apache.hadoop.hbase.replication.regionserver.RegionReplicaReplicationEndpoint; 036import org.apache.hadoop.hbase.zookeeper.ZKConfig; 037import org.apache.yetus.audience.InterfaceAudience; 038import org.slf4j.Logger; 039import org.slf4j.LoggerFactory; 040 041/** 042 * Similar to {@link RegionReplicaUtil} but for the server side 043 */ 044@InterfaceAudience.Private 045public class ServerRegionReplicaUtil extends RegionReplicaUtil { 046 047 private static final Logger LOG = LoggerFactory.getLogger(ServerRegionReplicaUtil.class); 048 049 /** 050 * Whether asynchronous WAL replication to the secondary region replicas is enabled or not. 051 * If this is enabled, a replication peer named "region_replica_replication" will be created 052 * which will tail the logs and replicate the mutatations to region replicas for tables that 053 * have region replication > 1. If this is enabled once, disabling this replication also 054 * requires disabling the replication peer using shell or {@link Admin} java class. 055 * Replication to secondary region replicas works over standard inter-cluster replication.ยท 056 */ 057 public static final String REGION_REPLICA_REPLICATION_CONF_KEY 058 = "hbase.region.replica.replication.enabled"; 059 private static final boolean DEFAULT_REGION_REPLICA_REPLICATION = false; 060 private static final String REGION_REPLICA_REPLICATION_PEER = "region_replica_replication"; 061 062 /** 063 * Enables or disables refreshing store files of secondary region replicas when the memory is 064 * above the global memstore lower limit. Refreshing the store files means that we will do a file 065 * list of the primary regions store files, and pick up new files. Also depending on the store 066 * files, we can drop some memstore contents which will free up memory. 067 */ 068 public static final String REGION_REPLICA_STORE_FILE_REFRESH 069 = "hbase.region.replica.storefile.refresh"; 070 private static final boolean DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH = true; 071 072 /** 073 * The multiplier to use when we want to refresh a secondary region instead of flushing a primary 074 * region. Default value assumes that for doing the file refresh, the biggest secondary should be 075 * 4 times bigger than the biggest primary. 076 */ 077 public static final String REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER 078 = "hbase.region.replica.storefile.refresh.memstore.multiplier"; 079 private static final double DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER = 4; 080 081 /** 082 * Returns the regionInfo object to use for interacting with the file system. 083 * @return An RegionInfo object to interact with the filesystem 084 */ 085 public static RegionInfo getRegionInfoForFs(RegionInfo regionInfo) { 086 if (regionInfo == null) { 087 return null; 088 } 089 return RegionReplicaUtil.getRegionInfoForDefaultReplica(regionInfo); 090 } 091 092 /** 093 * Returns whether this region replica can accept writes. 094 * @param region the HRegion object 095 * @return whether the replica is read only 096 */ 097 public static boolean isReadOnly(HRegion region) { 098 return region.getTableDescriptor().isReadOnly() 099 || !isDefaultReplica(region.getRegionInfo()); 100 } 101 102 /** 103 * Returns whether to replay the recovered edits to flush the results. 104 * Currently secondary region replicas do not replay the edits, since it would 105 * cause flushes which might affect the primary region. Primary regions even opened 106 * in read only mode should replay the edits. 107 * @param region the HRegion object 108 * @return whether recovered edits should be replayed. 109 */ 110 public static boolean shouldReplayRecoveredEdits(HRegion region) { 111 return isDefaultReplica(region.getRegionInfo()); 112 } 113 114 /** 115 * Returns a StoreFileInfo from the given FileStatus. Secondary replicas refer to the 116 * files of the primary region, so an HFileLink is used to construct the StoreFileInfo. This 117 * way ensures that the secondary will be able to continue reading the store files even if 118 * they are moved to archive after compaction 119 * @throws IOException 120 */ 121 public static StoreFileInfo getStoreFileInfo(Configuration conf, FileSystem fs, 122 RegionInfo regionInfo, RegionInfo regionInfoForFs, String familyName, Path path) 123 throws IOException { 124 125 // if this is a primary region, just return the StoreFileInfo constructed from path 126 if (RegionInfo.COMPARATOR.compare(regionInfo, regionInfoForFs) == 0) { 127 return new StoreFileInfo(conf, fs, path); 128 } 129 130 // else create a store file link. The link file does not exists on filesystem though. 131 HFileLink link = HFileLink.build(conf, regionInfoForFs.getTable(), 132 regionInfoForFs.getEncodedName(), familyName, path.getName()); 133 134 if (StoreFileInfo.isReference(path)) { 135 Reference reference = Reference.read(fs, path); 136 return new StoreFileInfo(conf, fs, link.getFileStatus(fs), reference); 137 } 138 139 return new StoreFileInfo(conf, fs, link.getFileStatus(fs), link); 140 } 141 142 /** 143 * Create replication peer for replicating to region replicas if needed. 144 * @param conf configuration to use 145 * @throws IOException 146 */ 147 public static void setupRegionReplicaReplication(Configuration conf) throws IOException { 148 if (!isRegionReplicaReplicationEnabled(conf)) { 149 return; 150 } 151 Admin admin = ConnectionFactory.createConnection(conf).getAdmin(); 152 ReplicationPeerConfig peerConfig = null; 153 try { 154 peerConfig = admin.getReplicationPeerConfig(REGION_REPLICA_REPLICATION_PEER); 155 } catch (ReplicationPeerNotFoundException e) { 156 LOG.warn("Region replica replication peer id=" + REGION_REPLICA_REPLICATION_PEER 157 + " not exist", e); 158 } 159 try { 160 if (peerConfig == null) { 161 LOG.info("Region replica replication peer id=" + REGION_REPLICA_REPLICATION_PEER 162 + " not exist. Creating..."); 163 peerConfig = new ReplicationPeerConfig(); 164 peerConfig.setClusterKey(ZKConfig.getZooKeeperClusterKey(conf)); 165 peerConfig.setReplicationEndpointImpl(RegionReplicaReplicationEndpoint.class.getName()); 166 admin.addReplicationPeer(REGION_REPLICA_REPLICATION_PEER, peerConfig); 167 } 168 } finally { 169 admin.close(); 170 } 171 } 172 173 public static boolean isRegionReplicaReplicationEnabled(Configuration conf) { 174 return conf.getBoolean(REGION_REPLICA_REPLICATION_CONF_KEY, 175 DEFAULT_REGION_REPLICA_REPLICATION); 176 } 177 178 public static boolean isRegionReplicaWaitForPrimaryFlushEnabled(Configuration conf) { 179 return conf.getBoolean(REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH_CONF_KEY, 180 DEFAULT_REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH); 181 } 182 183 public static boolean isRegionReplicaStoreFileRefreshEnabled(Configuration conf) { 184 return conf.getBoolean(REGION_REPLICA_STORE_FILE_REFRESH, 185 DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH); 186 } 187 188 public static double getRegionReplicaStoreFileRefreshMultiplier(Configuration conf) { 189 return conf.getDouble(REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER, 190 DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER); 191 } 192 193 /** 194 * Return the peer id used for replicating to secondary region replicas 195 */ 196 public static String getReplicationPeerId() { 197 return REGION_REPLICA_REPLICATION_PEER; 198 } 199 200}