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 */
018package org.apache.hadoop.hbase.util;
019
020import java.io.File;
021import java.io.IOException;
022import java.net.MalformedURLException;
023import java.net.URL;
024import java.util.HashMap;
025
026import org.apache.hadoop.conf.Configuration;
027import org.apache.hadoop.fs.FileStatus;
028import org.apache.hadoop.fs.FileSystem;
029import org.apache.hadoop.fs.Path;
030import org.apache.yetus.audience.InterfaceAudience;
031import org.slf4j.Logger;
032import org.slf4j.LoggerFactory;
033
034/**
035 * This is a class loader that can load classes dynamically from new
036 * jar files under a configured folder. The paths to the jar files are
037 * converted to URLs, and URLClassLoader logic is actually used to load
038 * classes. This class loader always uses its parent class loader
039 * to load a class at first. Only if its parent class loader
040 * can not load a class, we will try to load it using the logic here.
041 * <p>
042 * The configured folder can be a HDFS path. In this case, the jar files
043 * under that folder will be copied to local at first under ${hbase.local.dir}/jars/.
044 * The local copy will be updated if the remote copy is updated, according to its
045 * last modified timestamp.
046 * <p>
047 * We can't unload a class already loaded. So we will use the existing
048 * jar files we already know to load any class which can't be loaded
049 * using the parent class loader. If we still can't load the class from
050 * the existing jar files, we will check if any new jar file is added,
051 * if so, we will load the new jar file and try to load the class again.
052 * If still failed, a class not found exception will be thrown.
053 * <p>
054 * Be careful in uploading new jar files and make sure all classes
055 * are consistent, otherwise, we may not be able to load your
056 * classes properly.
057 */
058@InterfaceAudience.Private
059public class DynamicClassLoader extends ClassLoaderBase {
060  private static final Logger LOG =
061      LoggerFactory.getLogger(DynamicClassLoader.class);
062
063  // Dynamic jars are put under ${hbase.local.dir}/jars/
064  private static final String DYNAMIC_JARS_DIR = File.separator
065    + "jars" + File.separator;
066
067  private static final String DYNAMIC_JARS_DIR_KEY = "hbase.dynamic.jars.dir";
068
069  private static final String DYNAMIC_JARS_OPTIONAL_CONF_KEY = "hbase.use.dynamic.jars";
070  private static final boolean DYNAMIC_JARS_OPTIONAL_DEFAULT = true;
071
072  // The user-provided value for using the DynamicClassLoader
073  private final boolean userConfigUseDynamicJars;
074  // The current state of whether to use the DynamicClassLoader
075  private final boolean useDynamicJars;
076
077  private File localDir;
078
079  // FileSystem of the remote path, set only if remoteDir != null
080  private FileSystem remoteDirFs;
081  private Path remoteDir;
082
083  // Last modified time of local jars
084  private HashMap<String, Long> jarModifiedTime;
085
086  /**
087   * Creates a DynamicClassLoader that can load classes dynamically
088   * from jar files under a specific folder.
089   *
090   * @param conf the configuration for the cluster.
091   * @param parent the parent ClassLoader to set.
092   */
093  public DynamicClassLoader(
094      final Configuration conf, final ClassLoader parent) {
095    super(parent);
096
097    // Save off the user's original configuration value for the DynamicClassLoader
098    userConfigUseDynamicJars = conf.getBoolean(
099        DYNAMIC_JARS_OPTIONAL_CONF_KEY, DYNAMIC_JARS_OPTIONAL_DEFAULT);
100
101    boolean dynamicJarsEnabled = userConfigUseDynamicJars;
102    if (dynamicJarsEnabled) {
103      try {
104        initTempDir(conf);
105        dynamicJarsEnabled = true;
106      } catch (Exception e) {
107        LOG.error("Disabling the DynamicClassLoader as it failed to initialize its temp directory."
108            + " Check your configuration and filesystem permissions. Custom coprocessor code may"
109            + " not be loaded as a result of this failure.", e);
110        dynamicJarsEnabled = false;
111      }
112    }
113    useDynamicJars = dynamicJarsEnabled;
114  }
115
116  // FindBugs: Making synchronized to avoid IS2_INCONSISTENT_SYNC complaints about
117  // remoteDirFs and jarModifiedTime being part synchronized protected.
118  private synchronized void initTempDir(final Configuration conf) {
119    jarModifiedTime = new HashMap<>();
120    String localDirPath = conf.get(
121      LOCAL_DIR_KEY, DEFAULT_LOCAL_DIR) + DYNAMIC_JARS_DIR;
122    localDir = new File(localDirPath);
123    if (!localDir.mkdirs() && !localDir.isDirectory()) {
124      throw new RuntimeException("Failed to create local dir " + localDir.getPath()
125        + ", DynamicClassLoader failed to init");
126    }
127
128    String remotePath = conf.get(DYNAMIC_JARS_DIR_KEY);
129    if (remotePath == null || remotePath.equals(localDirPath)) {
130      remoteDir = null;  // ignore if it is the same as the local path
131    } else {
132      remoteDir = new Path(remotePath);
133      try {
134        remoteDirFs = remoteDir.getFileSystem(conf);
135      } catch (IOException ioe) {
136        LOG.warn("Failed to identify the fs of dir "
137          + remoteDir + ", ignored", ioe);
138        remoteDir = null;
139      }
140    }
141  }
142
143  @Override
144  public Class<?> loadClass(String name)
145      throws ClassNotFoundException {
146    try {
147      return parent.loadClass(name);
148    } catch (ClassNotFoundException e) {
149      if (useDynamicJars) {
150        LOG.debug("Class {} not found - using dynamical class loader", name);
151        return tryRefreshClass(name);
152      } else if (userConfigUseDynamicJars) {
153        // If the user tried to enable the DCL, then warn again.
154        LOG.debug("Not checking DynamicClassLoader for missing class because it is disabled."
155            + " See the log for previous errors.");
156      }
157      throw e;
158    }
159  }
160
161
162  private Class<?> tryRefreshClass(String name)
163      throws ClassNotFoundException {
164    synchronized (getClassLoadingLock(name)) {
165        // Check whether the class has already been loaded:
166        Class<?> clasz = findLoadedClass(name);
167        if (clasz != null) {
168          if (LOG.isDebugEnabled()) {
169            LOG.debug("Class " + name + " already loaded");
170          }
171        }
172        else {
173          try {
174            if (LOG.isDebugEnabled()) {
175              LOG.debug("Finding class: " + name);
176            }
177            clasz = findClass(name);
178          } catch (ClassNotFoundException cnfe) {
179            // Load new jar files if any
180            if (LOG.isDebugEnabled()) {
181              LOG.debug("Loading new jar files, if any");
182            }
183            loadNewJars();
184
185            if (LOG.isDebugEnabled()) {
186              LOG.debug("Finding class again: " + name);
187            }
188            clasz = findClass(name);
189          }
190        }
191        return clasz;
192      }
193  }
194
195  private synchronized void loadNewJars() {
196    // Refresh local jar file lists
197    File[] files = localDir == null ? null : localDir.listFiles();
198    if (files != null) {
199      for (File file : files) {
200        String fileName = file.getName();
201        if (jarModifiedTime.containsKey(fileName)) {
202          continue;
203        }
204        if (file.isFile() && fileName.endsWith(".jar")) {
205          jarModifiedTime.put(fileName, Long.valueOf(file.lastModified()));
206          try {
207            URL url = file.toURI().toURL();
208            addURL(url);
209          } catch (MalformedURLException mue) {
210            // This should not happen, just log it
211            LOG.warn("Failed to load new jar " + fileName, mue);
212          }
213        }
214      }
215    }
216
217    // Check remote files
218    FileStatus[] statuses = null;
219    if (remoteDir != null) {
220      try {
221        statuses = remoteDirFs.listStatus(remoteDir);
222      } catch (IOException ioe) {
223        LOG.warn("Failed to check remote dir status " + remoteDir, ioe);
224      }
225    }
226    if (statuses == null || statuses.length == 0) {
227      return; // no remote files at all
228    }
229
230    for (FileStatus status: statuses) {
231      if (status.isDirectory()) continue; // No recursive lookup
232      Path path = status.getPath();
233      String fileName = path.getName();
234      if (!fileName.endsWith(".jar")) {
235        if (LOG.isDebugEnabled()) {
236          LOG.debug("Ignored non-jar file " + fileName);
237        }
238        continue; // Ignore non-jar files
239      }
240      Long cachedLastModificationTime = jarModifiedTime.get(fileName);
241      if (cachedLastModificationTime != null) {
242        long lastModified = status.getModificationTime();
243        if (lastModified < cachedLastModificationTime.longValue()) {
244          // There could be some race, for example, someone uploads
245          // a new one right in the middle the old one is copied to
246          // local. We can check the size as well. But it is still
247          // not guaranteed. This should be rare. Most likely,
248          // we already have the latest one.
249          // If you are unlucky to hit this race issue, you have
250          // to touch the remote jar to update its last modified time
251          continue;
252        }
253      }
254      try {
255        // Copy it to local
256        File dst = new File(localDir, fileName);
257        remoteDirFs.copyToLocalFile(path, new Path(dst.getPath()));
258        jarModifiedTime.put(fileName, Long.valueOf(dst.lastModified()));
259        URL url = dst.toURI().toURL();
260        addURL(url);
261      } catch (IOException ioe) {
262        LOG.warn("Failed to load new jar " + fileName, ioe);
263      }
264    }
265  }
266}