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