View Javadoc

1   /*
2    * Copyright The Apache Software Foundation
3    *
4    * Licensed to the Apache Software Foundation (ASF) under one or more
5    * contributor license agreements. See the NOTICE file distributed with this
6    * work for additional information regarding copyright ownership. The ASF
7    * licenses this file to you under the Apache License, Version 2.0 (the
8    * "License"); you may not use this file except in compliance with the License.
9    * You may obtain a copy of the License at
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
13   * Unless required by applicable law or agreed to in writing, software
14   * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
15   * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
16   * License for the specific language governing permissions and limitations
17   * under the License.
18   */
19  package org.apache.hadoop.hbase.coprocessor.example;
20  
21  import java.io.IOException;
22  import java.util.Collections;
23  import java.util.List;
24  import java.util.NavigableSet;
25  
26  import org.apache.commons.logging.Log;
27  import org.apache.commons.logging.LogFactory;
28  import org.apache.hadoop.hbase.CoprocessorEnvironment;
29  import org.apache.hadoop.hbase.HConstants;
30  import org.apache.hadoop.hbase.client.IsolationLevel;
31  import org.apache.hadoop.hbase.client.Scan;
32  import org.apache.hadoop.hbase.coprocessor.BaseRegionObserver;
33  import org.apache.hadoop.hbase.coprocessor.ObserverContext;
34  import org.apache.hadoop.hbase.coprocessor.RegionCoprocessorEnvironment;
35  import org.apache.hadoop.hbase.regionserver.HStore;
36  import org.apache.hadoop.hbase.regionserver.InternalScanner;
37  import org.apache.hadoop.hbase.regionserver.KeyValueScanner;
38  import org.apache.hadoop.hbase.regionserver.ScanInfo;
39  import org.apache.hadoop.hbase.regionserver.ScanType;
40  import org.apache.hadoop.hbase.regionserver.Store;
41  import org.apache.hadoop.hbase.regionserver.StoreScanner;
42  import org.apache.hadoop.hbase.util.Bytes;
43  import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
44  import org.apache.zookeeper.KeeperException;
45  import org.apache.zookeeper.WatchedEvent;
46  import org.apache.zookeeper.Watcher;
47  import org.apache.zookeeper.ZooKeeper;
48  
49  /**
50   * This is an example showing how a RegionObserver could configured
51   * via ZooKeeper in order to control a Region compaction, flush, and scan policy.
52   *
53   * This also demonstrated the use of shared 
54   * {@link org.apache.hadoop.hbase.coprocessor.RegionObserver} state.
55   * See {@link RegionCoprocessorEnvironment#getSharedData()}.
56   *
57   * This would be useful for an incremental backup tool, which would indicate the last
58   * time of a successful backup via ZK and instruct HBase to not delete data that was
59   * inserted since (based on wall clock time). 
60   *
61   * This implements org.apache.zookeeper.Watcher directly instead of using
62   * {@link org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher}, 
63   * because RegionObservers come and go and currently
64   * listeners registered with ZooKeeperWatcher cannot be removed.
65   */
66  public class ZooKeeperScanPolicyObserver extends BaseRegionObserver {
67    public static final String node = "/backup/example/lastbackup";
68    public static final String zkkey = "ZK";
69    private static final Log LOG = LogFactory.getLog(ZooKeeperScanPolicyObserver.class);
70  
71    /**
72     * Internal watcher that keep "data" up to date asynchronously.
73     */
74    private static class ZKWatcher implements Watcher {
75      private byte[] data = null;
76      private ZooKeeper zk;
77      private volatile boolean needSetup = true;
78      private volatile long lastSetupTry = 0;
79  
80      public ZKWatcher(ZooKeeper zk) {
81        this.zk = zk;
82        // trigger the listening
83        getData();
84      }
85  
86      /**
87       * Get the maintained data. In case of any ZK exceptions this will retry
88       * establishing the connection (but not more than twice/minute).
89       *
90       * getData is on the critical path, so make sure it is fast unless there is
91       * a problem (network partion, ZK ensemble down, etc)
92       * Make sure at most one (unlucky) thread retries and other threads don't pile up
93       * while that threads tries to recreate the connection.
94       *
95       * @return the last know version of the data
96       */
97      @edu.umd.cs.findbugs.annotations.SuppressWarnings(value="REC_CATCH_EXCEPTION")
98      public byte[] getData() {
99        // try at most twice/minute
100       if (needSetup && EnvironmentEdgeManager.currentTime() > lastSetupTry + 30000) {
101         synchronized (this) {
102           // make sure only one thread tries to reconnect
103           if (needSetup) {
104             needSetup = false;
105           } else {
106             return data;
107           }
108         }
109         // do this without the lock held to avoid threads piling up on this lock,
110         // as it can take a while
111         try {
112           LOG.debug("Connecting to ZK");
113           // record this attempt
114           lastSetupTry = EnvironmentEdgeManager.currentTime();
115           if (zk.exists(node, false) != null) {
116             data = zk.getData(node, this, null);
117             LOG.debug("Read synchronously: "+(data == null ? "null" : Bytes.toLong(data)));
118           } else {
119             zk.exists(node, this);
120           }
121         } catch (Exception x) {
122           // try again if this fails
123           needSetup = true;
124         }
125       }
126       return data;
127     }
128 
129     @Override
130     public void process(WatchedEvent event) {
131       switch(event.getType()) {
132       case NodeDataChanged:
133       case NodeCreated:
134       try {
135         // get data and re-watch
136         data = zk.getData(node, this, null);
137         LOG.debug("Read asynchronously: "+(data == null ? "null" : Bytes.toLong(data)));
138       } catch (InterruptedException ix) {
139       } catch (KeeperException kx) {
140         needSetup = true;
141       }
142       break;
143 
144       case NodeDeleted:
145       try {
146         // just re-watch
147         zk.exists(node, this);
148         data = null;
149       } catch (InterruptedException ix) {
150       } catch (KeeperException kx) {
151         needSetup = true;
152       }
153       break;
154 
155       default:
156         // ignore
157       }
158     }
159   }
160 
161   @Override
162   public void start(CoprocessorEnvironment e) throws IOException {
163     RegionCoprocessorEnvironment re = (RegionCoprocessorEnvironment) e;
164     if (!re.getSharedData().containsKey(zkkey)) {
165       // there is a short race here
166       // in the worst case we create a watcher that will be notified once
167       re.getSharedData().putIfAbsent(
168           zkkey,
169           new ZKWatcher(re.getRegionServerServices().getZooKeeper()
170               .getRecoverableZooKeeper().getZooKeeper()));
171     }
172   }
173 
174   @Override
175   public void stop(CoprocessorEnvironment e) throws IOException {
176     // nothing to do here
177   }
178 
179   protected ScanInfo getScanInfo(Store store, RegionCoprocessorEnvironment e) {
180     byte[] data = ((ZKWatcher)e.getSharedData().get(zkkey)).getData();
181     if (data == null) {
182       return null;
183     }
184     ScanInfo oldSI = store.getScanInfo();
185     if (oldSI.getTtl() == Long.MAX_VALUE) {
186       return null;
187     }
188     long ttl = Math.max(EnvironmentEdgeManager.currentTime() -
189         Bytes.toLong(data), oldSI.getTtl());
190     return new ScanInfo(oldSI.getConfiguration(), store.getFamily(), ttl,
191         oldSI.getTimeToPurgeDeletes(), oldSI.getComparator());
192   }
193 
194   @Override
195   public InternalScanner preFlushScannerOpen(final ObserverContext<RegionCoprocessorEnvironment> c,
196       Store store, KeyValueScanner memstoreScanner, InternalScanner s) throws IOException {
197     ScanInfo scanInfo = getScanInfo(store, c.getEnvironment());
198     if (scanInfo == null) {
199       // take default action
200       return null;
201     }
202     Scan scan = new Scan();
203     scan.setMaxVersions(scanInfo.getMaxVersions());
204     return new StoreScanner(store, scanInfo, scan, Collections.singletonList(memstoreScanner),
205         ScanType.COMPACT_RETAIN_DELETES, store.getSmallestReadPoint(), HConstants.OLDEST_TIMESTAMP);
206   }
207 
208   @Override
209   public InternalScanner preCompactScannerOpen(
210       final ObserverContext<RegionCoprocessorEnvironment> c,
211       Store store, List<? extends KeyValueScanner> scanners, ScanType scanType, long earliestPutTs,
212       InternalScanner s) throws IOException {
213     ScanInfo scanInfo = getScanInfo(store, c.getEnvironment());
214     if (scanInfo == null) {
215       // take default action
216       return null;
217     }
218     Scan scan = new Scan();
219     scan.setMaxVersions(scanInfo.getMaxVersions());
220     return new StoreScanner(store, scanInfo, scan, scanners, scanType,
221         store.getSmallestReadPoint(), earliestPutTs);
222   }
223 
224   @Override
225   public KeyValueScanner preStoreScannerOpen(final ObserverContext<RegionCoprocessorEnvironment> c,
226       final Store store, final Scan scan, final NavigableSet<byte[]> targetCols,
227       final KeyValueScanner s) throws IOException {
228     ScanInfo scanInfo = getScanInfo(store, c.getEnvironment());
229     if (scanInfo == null) {
230       // take default action
231       return null;
232     }
233     return new StoreScanner(store, scanInfo, scan, targetCols,
234       ((HStore)store).getHRegion().getReadPoint(IsolationLevel.READ_COMMITTED));
235   }
236 }