View Javadoc

1   /**
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  
19  package org.apache.hadoop.hbase.util;
20  
21  import java.io.IOException;
22  
23  import org.apache.hadoop.conf.Configuration;
24  import org.apache.hadoop.fs.FileSystem;
25  import org.apache.hadoop.fs.Path;
26  import org.apache.hadoop.hbase.HRegionInfo;
27  import org.apache.hadoop.hbase.client.RegionReplicaUtil;
28  import org.apache.hadoop.hbase.client.replication.ReplicationAdmin;
29  import org.apache.hadoop.hbase.io.HFileLink;
30  import org.apache.hadoop.hbase.io.Reference;
31  import org.apache.hadoop.hbase.regionserver.HRegion;
32  import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
33  import org.apache.hadoop.hbase.replication.ReplicationException;
34  import org.apache.hadoop.hbase.replication.ReplicationPeerConfig;
35  import org.apache.hadoop.hbase.replication.regionserver.RegionReplicaReplicationEndpoint;
36  import org.apache.hadoop.hbase.zookeeper.ZKConfig;
37  
38  /**
39   * Similar to {@link RegionReplicaUtil} but for the server side
40   */
41  public class ServerRegionReplicaUtil extends RegionReplicaUtil {
42  
43    /**
44     * Whether asynchronous WAL replication to the secondary region replicas is enabled or not.
45     * If this is enabled, a replication peer named "region_replica_replication" will be created
46     * which will tail the logs and replicate the mutatations to region replicas for tables that
47     * have region replication > 1. If this is enabled once, disabling this replication also
48     * requires disabling the replication peer using shell or ReplicationAdmin java class.
49     * Replication to secondary region replicas works over standard inter-cluster replication.·
50     * So replication, if disabled explicitly, also has to be enabled by setting "hbase.replication"·
51     * to true for this feature to work.
52     */
53    public static final String REGION_REPLICA_REPLICATION_CONF_KEY
54      = "hbase.region.replica.replication.enabled";
55    private static final boolean DEFAULT_REGION_REPLICA_REPLICATION = false;
56    private static final String REGION_REPLICA_REPLICATION_PEER = "region_replica_replication";
57  
58    /**
59     * Enables or disables refreshing store files of secondary region replicas when the memory is
60     * above the global memstore lower limit. Refreshing the store files means that we will do a file
61     * list of the primary regions store files, and pick up new files. Also depending on the store
62     * files, we can drop some memstore contents which will free up memory.
63     */
64    public static final String REGION_REPLICA_STORE_FILE_REFRESH
65      = "hbase.region.replica.storefile.refresh";
66    private static final boolean DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH = true;
67  
68    /**
69     * The multiplier to use when we want to refresh a secondary region instead of flushing a primary
70     * region. Default value assumes that for doing the file refresh, the biggest secondary should be
71     * 4 times bigger than the biggest primary.
72     */
73    public static final String REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER
74      = "hbase.region.replica.storefile.refresh.memstore.multiplier";
75    private static final double DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER = 4;
76  
77    /**
78     * Returns the regionInfo object to use for interacting with the file system.
79     * @return An HRegionInfo object to interact with the filesystem
80     */
81    public static HRegionInfo getRegionInfoForFs(HRegionInfo regionInfo) {
82      if (regionInfo == null) {
83        return null;
84      }
85      return RegionReplicaUtil.getRegionInfoForDefaultReplica(regionInfo);
86    }
87  
88    /**
89     * Returns whether this region replica can accept writes.
90     * @param region the HRegion object
91     * @return whether the replica is read only
92     */
93    public static boolean isReadOnly(HRegion region) {
94      return region.getTableDesc().isReadOnly()
95        || !isDefaultReplica(region.getRegionInfo());
96    }
97  
98    /**
99     * Returns whether to replay the recovered edits to flush the results.
100    * Currently secondary region replicas do not replay the edits, since it would
101    * cause flushes which might affect the primary region. Primary regions even opened
102    * in read only mode should replay the edits.
103    * @param region the HRegion object
104    * @return whether recovered edits should be replayed.
105    */
106   public static boolean shouldReplayRecoveredEdits(HRegion region) {
107     return isDefaultReplica(region.getRegionInfo());
108   }
109 
110   /**
111    * Returns a StoreFileInfo from the given FileStatus. Secondary replicas refer to the
112    * files of the primary region, so an HFileLink is used to construct the StoreFileInfo. This
113    * way ensures that the secondary will be able to continue reading the store files even if
114    * they are moved to archive after compaction
115    * @throws IOException
116    */
117   public static StoreFileInfo getStoreFileInfo(Configuration conf, FileSystem fs,
118       HRegionInfo regionInfo, HRegionInfo regionInfoForFs, String familyName, Path path)
119       throws IOException {
120 
121     // if this is a primary region, just return the StoreFileInfo constructed from path
122     if (regionInfo.equals(regionInfoForFs)) {
123       return new StoreFileInfo(conf, fs, path);
124     }
125 
126     // else create a store file link. The link file does not exists on filesystem though.
127     HFileLink link = HFileLink.build(conf, regionInfoForFs.getTable(),
128             regionInfoForFs.getEncodedName(), familyName, path.getName());
129 
130     if (StoreFileInfo.isReference(path)) {
131       Reference reference = Reference.read(fs, path);
132       return new StoreFileInfo(conf, fs, link.getFileStatus(fs), reference);
133     }
134 
135     return new StoreFileInfo(conf, fs, link.getFileStatus(fs), link);
136   }
137 
138   /**
139    * Create replication peer for replicating to region replicas if needed.
140    * @param conf configuration to use
141    * @throws IOException
142    */
143   public static void setupRegionReplicaReplication(Configuration conf) throws IOException {
144     if (!isRegionReplicaReplicationEnabled(conf)) {
145       return;
146     }
147     ReplicationAdmin repAdmin = new ReplicationAdmin(conf);
148     try {
149       if (repAdmin.getPeerConfig(REGION_REPLICA_REPLICATION_PEER) == null) {
150         ReplicationPeerConfig peerConfig = new ReplicationPeerConfig();
151         peerConfig.setClusterKey(ZKConfig.getZooKeeperClusterKey(conf));
152         peerConfig.setReplicationEndpointImpl(RegionReplicaReplicationEndpoint.class.getName());
153         repAdmin.addPeer(REGION_REPLICA_REPLICATION_PEER, peerConfig, null);
154       }
155     } catch (ReplicationException ex) {
156       throw new IOException(ex);
157     } finally {
158       repAdmin.close();
159     }
160   }
161 
162   public static boolean isRegionReplicaReplicationEnabled(Configuration conf) {
163     return conf.getBoolean(REGION_REPLICA_REPLICATION_CONF_KEY,
164       DEFAULT_REGION_REPLICA_REPLICATION);
165   }
166 
167   public static boolean isRegionReplicaWaitForPrimaryFlushEnabled(Configuration conf) {
168     return conf.getBoolean(REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH_CONF_KEY,
169       DEFAULT_REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH);
170   }
171 
172   public static boolean isRegionReplicaStoreFileRefreshEnabled(Configuration conf) {
173     return conf.getBoolean(REGION_REPLICA_STORE_FILE_REFRESH,
174       DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH);
175   }
176 
177   public static double getRegionReplicaStoreFileRefreshMultiplier(Configuration conf) {
178     return conf.getDouble(REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER,
179       DEFAULT_REGION_REPLICA_STORE_FILE_REFRESH_MEMSTORE_MULTIPLIER);
180   }
181 
182   /**
183    * Return the peer id used for replicating to secondary region replicas
184    */
185   public static String getReplicationPeerId() {
186     return REGION_REPLICA_REPLICATION_PEER;
187   }
188 
189 }