1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19 package org.apache.hadoop.hbase.replication;
20
21 import java.io.Closeable;
22 import java.io.IOException;
23 import java.util.ArrayList;
24 import java.util.List;
25 import java.util.concurrent.atomic.AtomicBoolean;
26
27 import org.apache.commons.logging.Log;
28 import org.apache.commons.logging.LogFactory;
29 import org.apache.hadoop.classification.InterfaceAudience;
30 import org.apache.hadoop.conf.Configuration;
31 import org.apache.hadoop.hbase.Abortable;
32 import org.apache.hadoop.hbase.ServerName;
33 import org.apache.hadoop.hbase.exceptions.DeserializationException;
34 import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
35 import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos;
36 import org.apache.hadoop.hbase.zookeeper.ZKUtil;
37 import org.apache.hadoop.hbase.zookeeper.ZooKeeperNodeTracker;
38 import org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher;
39 import org.apache.zookeeper.KeeperException;
40 import org.apache.zookeeper.KeeperException.NodeExistsException;
41
42 import com.google.protobuf.InvalidProtocolBufferException;
43
44
45
46
47
48
49 @InterfaceAudience.Private
50 public class ReplicationPeer implements Abortable, Closeable {
51 private static final Log LOG = LogFactory.getLog(ReplicationPeer.class);
52
53 private final String clusterKey;
54 private final String id;
55 private List<ServerName> regionServers = new ArrayList<ServerName>(0);
56 private final AtomicBoolean peerEnabled = new AtomicBoolean();
57
58 private ZooKeeperWatcher zkw;
59 private final Configuration conf;
60 private long lastRegionserverUpdate;
61
62 private PeerStateTracker peerStateTracker;
63
64
65
66
67
68
69
70
71
72 public ReplicationPeer(Configuration conf, String key, String id) throws ReplicationException {
73 this.conf = conf;
74 this.clusterKey = key;
75 this.id = id;
76 try {
77 this.reloadZkWatcher();
78 } catch (IOException e) {
79 throw new ReplicationException("Error connecting to peer cluster with peerId=" + id, e);
80 }
81 }
82
83
84
85
86
87
88
89
90 public void startStateTracker(ZooKeeperWatcher zookeeper, String peerStateNode)
91 throws KeeperException {
92 ensurePeerEnabled(zookeeper, peerStateNode);
93 this.peerStateTracker = new PeerStateTracker(peerStateNode, zookeeper, this);
94 this.peerStateTracker.start();
95 try {
96 this.readPeerStateZnode();
97 } catch (DeserializationException e) {
98 throw ZKUtil.convert(e);
99 }
100 }
101
102 private void readPeerStateZnode() throws DeserializationException {
103 this.peerEnabled.set(isStateEnabled(this.peerStateTracker.getData(false)));
104 }
105
106
107
108
109
110
111 public String getClusterKey() {
112 return clusterKey;
113 }
114
115
116
117
118
119 public AtomicBoolean getPeerEnabled() {
120 return peerEnabled;
121 }
122
123
124
125
126
127
128 public List<ServerName> getRegionServers() {
129 return regionServers;
130 }
131
132
133
134
135
136 public void setRegionServers(List<ServerName> regionServers) {
137 this.regionServers = regionServers;
138 lastRegionserverUpdate = System.currentTimeMillis();
139 }
140
141
142
143
144
145 public ZooKeeperWatcher getZkw() {
146 return zkw;
147 }
148
149
150
151
152
153
154 public long getLastRegionserverUpdate() {
155 return lastRegionserverUpdate;
156 }
157
158
159
160
161
162 public String getId() {
163 return id;
164 }
165
166
167
168
169
170 public Configuration getConfiguration() {
171 return conf;
172 }
173
174 @Override
175 public void abort(String why, Throwable e) {
176 LOG.fatal("The ReplicationPeer coresponding to peer " + clusterKey
177 + " was aborted for the following reason(s):" + why, e);
178 }
179
180
181
182
183
184 public void reloadZkWatcher() throws IOException {
185 if (zkw != null) zkw.close();
186 zkw = new ZooKeeperWatcher(conf,
187 "connection to cluster: " + id, this);
188 }
189
190 @Override
191 public boolean isAborted() {
192
193
194 return false;
195 }
196
197 @Override
198 public void close() throws IOException {
199 if (zkw != null){
200 zkw.close();
201 }
202 }
203
204
205
206
207
208
209
210 public static boolean isStateEnabled(final byte[] bytes) throws DeserializationException {
211 ZooKeeperProtos.ReplicationState.State state = parseStateFrom(bytes);
212 return ZooKeeperProtos.ReplicationState.State.ENABLED == state;
213 }
214
215
216
217
218
219
220 private static ZooKeeperProtos.ReplicationState.State parseStateFrom(final byte[] bytes)
221 throws DeserializationException {
222 ProtobufUtil.expectPBMagicPrefix(bytes);
223 int pblen = ProtobufUtil.lengthOfPBMagic();
224 ZooKeeperProtos.ReplicationState.Builder builder =
225 ZooKeeperProtos.ReplicationState.newBuilder();
226 ZooKeeperProtos.ReplicationState state;
227 try {
228 state = builder.mergeFrom(bytes, pblen, bytes.length - pblen).build();
229 return state.getState();
230 } catch (InvalidProtocolBufferException e) {
231 throw new DeserializationException(e);
232 }
233 }
234
235
236
237
238
239
240
241
242
243 private static boolean ensurePeerEnabled(final ZooKeeperWatcher zookeeper, final String path)
244 throws NodeExistsException, KeeperException {
245 if (ZKUtil.checkExists(zookeeper, path) == -1) {
246
247
248
249 ZKUtil.createNodeIfNotExistsAndWatch(zookeeper, path,
250 ReplicationStateZKBase.ENABLED_ZNODE_BYTES);
251 return true;
252 }
253 return false;
254 }
255
256
257
258
259 public class PeerStateTracker extends ZooKeeperNodeTracker {
260
261 public PeerStateTracker(String peerStateZNode, ZooKeeperWatcher watcher,
262 Abortable abortable) {
263 super(watcher, peerStateZNode, abortable);
264 }
265
266 @Override
267 public synchronized void nodeDataChanged(String path) {
268 if (path.equals(node)) {
269 super.nodeDataChanged(path);
270 try {
271 readPeerStateZnode();
272 } catch (DeserializationException e) {
273 LOG.warn("Failed deserializing the content of " + path, e);
274 }
275 }
276 }
277 }
278 }