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 if (HFileLink.isHFileLink(path) || StoreFileInfo.isHFile(path)) { 132 HFileLink link = HFileLink 133 .build(conf, regionInfoForFs.getTable(), regionInfoForFs.getEncodedName(), familyName, 134 path.getName()); 135 return new StoreFileInfo(conf, fs, link.getFileStatus(fs), link); 136 } else if (StoreFileInfo.isReference(path)) { 137 Reference reference = Reference.read(fs, path); 138 Path referencePath = StoreFileInfo.getReferredToFile(path); 139 if (HFileLink.isHFileLink(referencePath)) { 140 // HFileLink Reference 141 HFileLink link = HFileLink.buildFromHFileLinkPattern(conf, referencePath); 142 return new StoreFileInfo(conf, fs, link.getFileStatus(fs), reference, link); 143 } else { 144 // Reference 145 HFileLink link = HFileLink 146 .build(conf, regionInfoForFs.getTable(), regionInfoForFs.getEncodedName(), familyName, 147 path.getName()); 148 return new StoreFileInfo(conf, fs, link.getFileStatus(fs), reference); 149 } 150 } else { 151 throw new IOException("path=" + path + " doesn't look like a valid StoreFile"); 152 } 153 } 154 155 /** 156 * Create replication peer for replicating to region replicas if needed. 157 * @param conf configuration to use 158 * @throws IOException 159 */ 160 public static void setupRegionReplicaReplication(Configuration conf) throws IOException { 161 if (!isRegionReplicaReplicationEnabled(conf)) { 162 return; 163 } 164 Admin admin = ConnectionFactory.createConnection(conf).getAdmin(); 165 ReplicationPeerConfig peerConfig = null; 166 try { 167 peerConfig = admin.getReplicationPeerConfig(REGION_REPLICA_REPLICATION_PEER); 168 } catch (ReplicationPeerNotFoundException e) { 169 LOG.warn("Region replica replication peer id=" + REGION_REPLICA_REPLICATION_PEER 170 + " not exist", e); 171 } 172 try { 173 if (peerConfig == null) { 174 LOG.info("Region replica replication peer id=" + REGION_REPLICA_REPLICATION_PEER 175 + " not exist. Creating..."); 176 peerConfig = new ReplicationPeerConfig(); 177 peerConfig.setClusterKey(ZKConfig.getZooKeeperClusterKey(conf)); 178 peerConfig.setReplicationEndpointImpl(RegionReplicaReplicationEndpoint.class.getName()); 179 admin.addReplicationPeer(REGION_REPLICA_REPLICATION_PEER, peerConfig); 180 } 181 } finally { 182 admin.close(); 183 } 184 } 185 186 public static boolean isRegionReplicaReplicationEnabled(Configuration conf) { 187 return conf.getBoolean(REGION_REPLICA_REPLICATION_CONF_KEY, 188 DEFAULT_REGION_REPLICA_REPLICATION); 189 } 190 191 public static boolean isRegionReplicaWaitForPrimaryFlushEnabled(Configuration conf) { 192 return conf.getBoolean(REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH_CONF_KEY, 193 DEFAULT_REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH); 194 } 195 196 public static boolean isRegionReplicaStoreFileRefreshEnabled(Configuration conf) { 197 return conf.getBoolean(REGION_REPLICA_STORE_FILE_REFRESH, 198 DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH); 199 } 200 201 public static double getRegionReplicaStoreFileRefreshMultiplier(Configuration conf) { 202 return conf.getDouble(REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER, 203 DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER); 204 } 205 206 /** 207 * Return the peer id used for replicating to secondary region replicas 208 */ 209 public static String getReplicationPeerId() { 210 return REGION_REPLICA_REPLICATION_PEER; 211 } 212 213}