You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by li...@apache.org on 2013/05/18 20:18:33 UTC

svn commit: r1484158 - in /hbase/branches/0.89-fb/src: main/java/org/apache/hadoop/hbase/conf/ main/java/org/apache/hadoop/hbase/regionserver/ test/java/org/apache/hadoop/hbase/conf/

Author: liyin
Date: Sat May 18 18:18:32 2013
New Revision: 1484158

URL: http://svn.apache.org/r1484158
Log:
[HBASE-8576] Allow individual classes to get notified when Configuration is reloaded

Author: gauravm

Summary: As a part of online configuration update, we want to allow individual classes to listen for changes in certain configuration properties.

Test Plan: Unit Tests

Reviewers: liyintang, aaiyer, shaneh, rshroff, manukranthk, adela

Reviewed By: liyintang

CC: hbase-eng@

Differential Revision: https://phabricator.fb.com/D812768

Task ID: 2258346

Added:
    hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/
    hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationManager.java
    hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationObserver.java
    hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/conf/
    hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/conf/TestConfigurationManager.java
Modified:
    hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/HRegionServer.java

Added: hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationManager.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationManager.java?rev=1484158&view=auto
==============================================================================
--- hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationManager.java (added)
+++ hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationManager.java Sat May 18 18:18:32 2013
@@ -0,0 +1,97 @@
+/**
+ * Copyright 2013 The Apache Software Foundation
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.hbase.conf;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.conf.Configuration;
+
+import java.util.Collections;
+import java.util.Set;
+import java.util.WeakHashMap;
+
+/**
+ * Maintains a set of all the classes which are would like to get notified
+ * when the Configuration is reloaded from disk.
+ */
+public class ConfigurationManager {
+  public static final Log LOG = LogFactory.getLog(ConfigurationManager.class);
+
+  // The set of Configuration Observers. These classes would like to get
+  // notified when the configuration is reloaded from disk. This is a set
+  // constructed from a WeakHashMap, whose entries would be removed if the
+  // observer classes go out of scope.
+  private Set<ConfigurationObserver> configurationObservers =
+    Collections.newSetFromMap(new WeakHashMap<ConfigurationObserver,
+                                              Boolean>());
+
+  /**
+   * Register an observer class
+   * @param observer
+   */
+  public void registerObserver(ConfigurationObserver observer) {
+    synchronized (configurationObservers) {
+      configurationObservers.add(observer);
+    }
+  }
+
+  /**
+   * Deregister an observer class
+   * @param observer
+   */
+  public void deregisterObserver(ConfigurationObserver observer) {
+    synchronized (configurationObservers) {
+      configurationObservers.remove(observer);
+    }
+  }
+
+  /**
+   * The conf object has been repopulated from disk, and we have to notify
+   * all the observers that are expressed interest to do that.
+   */
+  public void notifyAllObservers(Configuration conf) {
+    synchronized (configurationObservers) {
+      for (ConfigurationObserver observer : configurationObservers) {
+        try {
+          observer.notifyOnChange(conf);
+        } catch (NullPointerException e) {
+          // Though not likely, but a null pointer exception might happen
+          // if the GC happens after this iteration started and before
+          // we call the notifyOnChange() method. A
+          // ConcurrentModificationException will not happen since GC doesn't
+          // change the structure of the map.
+          LOG.error("Encountered a NPE while notifying observers.");
+        } catch (Throwable t) {
+          LOG.error("Encountered a throwable while notifying observers: " +
+                    t.getMessage());
+        }
+      }
+    }
+  }
+
+  /**
+   * Return the number of observers.
+   * @return
+   */
+  public int getNumObservers() {
+    return configurationObservers.size();
+  }
+}

Added: hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationObserver.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationObserver.java?rev=1484158&view=auto
==============================================================================
--- hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationObserver.java (added)
+++ hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/conf/ConfigurationObserver.java Sat May 18 18:18:32 2013
@@ -0,0 +1,36 @@
+/**
+ * Copyright 2013 The Apache Software Foundation
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hbase.conf;
+
+import org.apache.hadoop.conf.Configuration;
+
+/**
+ * Every class that wants to observe changes in Configuration properties,
+ * must implement interface (and also, register itself with the
+ * <code>ConfigurationManager</code> object.
+ */
+public interface ConfigurationObserver {
+
+  /**
+   * This method would be called by the <code>ConfigurationManager</code>
+   * object when the <code>Configuration</code> object is reloaded from disk.
+   */
+  void notifyOnChange(Configuration conf);
+}

Modified: hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/HRegionServer.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/HRegionServer.java?rev=1484158&r1=1484157&r2=1484158&view=diff
==============================================================================
--- hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/HRegionServer.java (original)
+++ hbase/branches/0.89-fb/src/main/java/org/apache/hadoop/hbase/regionserver/HRegionServer.java Sat May 18 18:18:32 2013
@@ -108,6 +108,7 @@ import org.apache.hadoop.hbase.client.Ro
 import org.apache.hadoop.hbase.client.Scan;
 import org.apache.hadoop.hbase.client.ServerConnection;
 import org.apache.hadoop.hbase.client.ServerConnectionManager;
+import org.apache.hadoop.hbase.conf.ConfigurationManager;
 import org.apache.hadoop.hbase.io.hfile.CacheConfig;
 import org.apache.hadoop.hbase.io.hfile.LruBlockCache;
 import org.apache.hadoop.hbase.io.hfile.LruBlockCache.CacheStats;
@@ -373,6 +374,10 @@ public class HRegionServer implements HR
 
   public static boolean runMetrics = true;
 
+  // This object lets classes register themselves to get notified on
+  // Configuration changes.
+  public ConfigurationManager configurationManager;
+
   public static long getResponseSizeLimit() {
     return responseSizeLimit;
   }
@@ -405,6 +410,7 @@ public class HRegionServer implements HR
     this.abortRequested = false;
     this.fsOk = true;
     this.conf = conf;
+    this.configurationManager = new ConfigurationManager();
     this.connection = ServerConnectionManager.getConnection(conf);
 
     this.isOnline = false;
@@ -1453,6 +1459,14 @@ public class HRegionServer implements HR
     return isOnline;
   }
 
+  /**
+   * Return the ConfigurationManager object.
+   * @return
+   */
+  public ConfigurationManager getConfigurationManager() {
+    return this.configurationManager;
+  }
+
   private void setupHLog(Path logDir, Path oldLogDir, int totalHLogCnt) throws IOException {
     hlogs = new HLog[totalHLogCnt];
     for (int i = 0; i < totalHLogCnt; i++) { 
@@ -3788,9 +3802,13 @@ public class HRegionServer implements HR
   public void updateConfiguration() {
     LOG.info("Reloading the configuration from disk.");
     conf.reloadConfiguration();
+    // TODO @gauravm
+    // Move this to the notifyOnChange() method in HRegionServer
     for (HRegion r : onlineRegions.values()) {
       r.updateConfiguration();
     }
+    // Notify all the observers that the configuration has changed.
+    configurationManager.notifyAllObservers(conf);
   }
 
   public long getResponseQueueSize(){

Added: hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/conf/TestConfigurationManager.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/conf/TestConfigurationManager.java?rev=1484158&view=auto
==============================================================================
--- hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/conf/TestConfigurationManager.java (added)
+++ hbase/branches/0.89-fb/src/test/java/org/apache/hadoop/hbase/conf/TestConfigurationManager.java Sat May 18 18:18:32 2013
@@ -0,0 +1,133 @@
+/**
+ * Copyright 2013 The Apache Software Foundation
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.hbase.conf;
+
+import junit.framework.TestCase;
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.conf.Configuration;
+
+import static junit.framework.Assert.assertEquals;
+import static junit.framework.Assert.assertTrue;
+
+public class TestConfigurationManager extends TestCase {
+  public static final Log LOG = LogFactory.getLog(TestConfigurationManager.class);
+
+  class DummyConfigurationObserver implements ConfigurationObserver {
+    private boolean notifiedOnChange = false;
+    private ConfigurationManager cm;
+
+    public DummyConfigurationObserver(ConfigurationManager cm) {
+      this.cm = cm;
+      register();
+    }
+
+    public void notifyOnChange(Configuration conf) {
+      notifiedOnChange = true;
+    }
+
+    // Was the observer notified on Configuration change?
+    public boolean wasNotifiedOnChange() {
+      return notifiedOnChange;
+    }
+
+    public void resetNotifiedOnChange() {
+      notifiedOnChange = false;
+    }
+
+    public void register() {
+      this.cm.registerObserver(this);
+    }
+
+    public void deregister() {
+      this.cm.deregisterObserver(this);
+    }
+  }
+
+  /**
+   * Test if observers get notified by the <code>ConfigurationManager</code>
+   * when the Configuration is reloaded.
+   */
+  public void testCheckIfObserversNotified() {
+    Configuration conf = new Configuration();
+    ConfigurationManager cm = new ConfigurationManager();
+    DummyConfigurationObserver d1 = new DummyConfigurationObserver(cm);
+
+    // Check if we get notified.
+    cm.notifyAllObservers(conf);
+    assertTrue(d1.wasNotifiedOnChange());
+    d1.resetNotifiedOnChange();
+
+    // Now check if we get notified on change with more than one observers.
+    DummyConfigurationObserver d2 = new DummyConfigurationObserver(cm);
+    cm.notifyAllObservers(conf);
+    assertTrue(d1.wasNotifiedOnChange());
+    d1.resetNotifiedOnChange();
+    assertTrue(d2.wasNotifiedOnChange());
+    d2.resetNotifiedOnChange();
+
+    // Now try deregistering an observer and verify that it was not notified
+    d2.deregister();
+    cm.notifyAllObservers(conf);
+    assertTrue(d1.wasNotifiedOnChange());
+    d1.resetNotifiedOnChange();
+    assertFalse(d2.wasNotifiedOnChange());
+  }
+
+  // Register an observer that will go out of scope immediately, allowing
+  // us to test that out of scope observers are deregistered.
+  private void registerLocalObserver(ConfigurationManager cm) {
+    new DummyConfigurationObserver(cm);
+  }
+
+  /**
+   * Test if out-of-scope observers are deregistered on GC.
+   */
+  public void testDeregisterOnOutOfScope() {
+    Configuration conf = new Configuration();
+    ConfigurationManager cm = new ConfigurationManager();
+
+    boolean outOfScopeObserversDeregistered = false;
+
+    // On my machine, I was able to cause a GC after around 5 iterations.
+    // If we do not cause a GC in 100k iterations, which is very unlikely,
+    // there might be something wrong with the GC.
+    for (int i = 0; i < 100000; i++) {
+      registerLocalObserver(cm);
+      cm.notifyAllObservers(conf);
+
+      // 'Suggest' the system to do a GC. We should be able to cause GC
+      // atleast once in the 2000 iterations.
+      System.gc();
+
+      // If GC indeed happened, all the observers (which are all out of scope),
+      // should have been deregistered.
+      if (cm.getNumObservers() <= i) {
+        outOfScopeObserversDeregistered = true;
+        break;
+      }
+    }
+    if (!outOfScopeObserversDeregistered) {
+      LOG.warn("Observers were not GC-ed! Something seems to be wrong.");
+    }
+    assertTrue(outOfScopeObserversDeregistered);
+  }
+}