|
@@ -17,25 +17,15 @@
|
|
*/
|
|
*/
|
|
package org.apache.hadoop.yarn.server.resourcemanager.scheduler.fair;
|
|
package org.apache.hadoop.yarn.server.resourcemanager.scheduler.fair;
|
|
|
|
|
|
-import java.io.File;
|
|
|
|
-import java.io.IOException;
|
|
|
|
-import java.net.URL;
|
|
|
|
-import java.util.ArrayList;
|
|
|
|
-import java.util.HashMap;
|
|
|
|
-import java.util.HashSet;
|
|
|
|
-import java.util.List;
|
|
|
|
-import java.util.Map;
|
|
|
|
-import java.util.Set;
|
|
|
|
-
|
|
|
|
-import javax.xml.parsers.DocumentBuilder;
|
|
|
|
-import javax.xml.parsers.DocumentBuilderFactory;
|
|
|
|
-import javax.xml.parsers.ParserConfigurationException;
|
|
|
|
-
|
|
|
|
|
|
+import com.google.common.annotations.VisibleForTesting;
|
|
import org.apache.commons.logging.Log;
|
|
import org.apache.commons.logging.Log;
|
|
import org.apache.commons.logging.LogFactory;
|
|
import org.apache.commons.logging.LogFactory;
|
|
import org.apache.hadoop.classification.InterfaceAudience.Public;
|
|
import org.apache.hadoop.classification.InterfaceAudience.Public;
|
|
import org.apache.hadoop.classification.InterfaceStability.Unstable;
|
|
import org.apache.hadoop.classification.InterfaceStability.Unstable;
|
|
import org.apache.hadoop.conf.Configuration;
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
|
+import org.apache.hadoop.fs.FileSystem;
|
|
|
|
+import org.apache.hadoop.fs.Path;
|
|
|
|
+import org.apache.hadoop.fs.UnsupportedFileSystemException;
|
|
import org.apache.hadoop.security.authorize.AccessControlList;
|
|
import org.apache.hadoop.security.authorize.AccessControlList;
|
|
import org.apache.hadoop.service.AbstractService;
|
|
import org.apache.hadoop.service.AbstractService;
|
|
import org.apache.hadoop.yarn.api.records.QueueACL;
|
|
import org.apache.hadoop.yarn.api.records.QueueACL;
|
|
@@ -45,8 +35,8 @@ import org.apache.hadoop.yarn.security.AccessType;
|
|
import org.apache.hadoop.yarn.security.Permission;
|
|
import org.apache.hadoop.yarn.security.Permission;
|
|
import org.apache.hadoop.yarn.security.PrivilegedEntity;
|
|
import org.apache.hadoop.yarn.security.PrivilegedEntity;
|
|
import org.apache.hadoop.yarn.security.PrivilegedEntity.EntityType;
|
|
import org.apache.hadoop.yarn.security.PrivilegedEntity.EntityType;
|
|
-import org.apache.hadoop.yarn.server.resourcemanager.scheduler.fair.policies.FifoPolicy;
|
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.SchedulerUtils;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.SchedulerUtils;
|
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.fair.policies.FifoPolicy;
|
|
import org.apache.hadoop.yarn.util.Clock;
|
|
import org.apache.hadoop.yarn.util.Clock;
|
|
import org.apache.hadoop.yarn.util.SystemClock;
|
|
import org.apache.hadoop.yarn.util.SystemClock;
|
|
import org.apache.hadoop.yarn.util.resource.Resources;
|
|
import org.apache.hadoop.yarn.util.resource.Resources;
|
|
@@ -57,7 +47,17 @@ import org.w3c.dom.NodeList;
|
|
import org.w3c.dom.Text;
|
|
import org.w3c.dom.Text;
|
|
import org.xml.sax.SAXException;
|
|
import org.xml.sax.SAXException;
|
|
|
|
|
|
-import com.google.common.annotations.VisibleForTesting;
|
|
|
|
|
|
+import javax.xml.parsers.DocumentBuilder;
|
|
|
|
+import javax.xml.parsers.DocumentBuilderFactory;
|
|
|
|
+import javax.xml.parsers.ParserConfigurationException;
|
|
|
|
+import java.io.IOException;
|
|
|
|
+import java.net.URL;
|
|
|
|
+import java.util.ArrayList;
|
|
|
|
+import java.util.HashMap;
|
|
|
|
+import java.util.HashSet;
|
|
|
|
+import java.util.List;
|
|
|
|
+import java.util.Map;
|
|
|
|
+import java.util.Set;
|
|
|
|
|
|
@Public
|
|
@Public
|
|
@Unstable
|
|
@Unstable
|
|
@@ -77,6 +77,9 @@ public class AllocationFileLoaderService extends AbstractService {
|
|
|
|
|
|
public static final long THREAD_JOIN_TIMEOUT_MS = 1000;
|
|
public static final long THREAD_JOIN_TIMEOUT_MS = 1000;
|
|
|
|
|
|
|
|
+ //Permitted allocation file filesystems (case insensitive)
|
|
|
|
+ private static final String SUPPORTED_FS_REGEX =
|
|
|
|
+ "(?i)(hdfs)|(file)|(s3a)|(viewfs)";
|
|
private static final String ROOT = "root";
|
|
private static final String ROOT = "root";
|
|
private static final AccessControlList EVERYBODY_ACL =
|
|
private static final AccessControlList EVERYBODY_ACL =
|
|
new AccessControlList("*");
|
|
new AccessControlList("*");
|
|
@@ -85,12 +88,14 @@ public class AllocationFileLoaderService extends AbstractService {
|
|
|
|
|
|
private final Clock clock;
|
|
private final Clock clock;
|
|
|
|
|
|
- private long lastSuccessfulReload; // Last time we successfully reloaded queues
|
|
|
|
- private boolean lastReloadAttemptFailed = false;
|
|
|
|
-
|
|
|
|
- // Path to XML file containing allocations.
|
|
|
|
- private File allocFile;
|
|
|
|
-
|
|
|
|
|
|
+ // Last time we successfully reloaded queues
|
|
|
|
+ private volatile long lastSuccessfulReload;
|
|
|
|
+ private volatile boolean lastReloadAttemptFailed = false;
|
|
|
|
+
|
|
|
|
+ // Path to XML file containing allocations.
|
|
|
|
+ private Path allocFile;
|
|
|
|
+ private FileSystem fs;
|
|
|
|
+
|
|
private Listener reloadListener;
|
|
private Listener reloadListener;
|
|
|
|
|
|
@VisibleForTesting
|
|
@VisibleForTesting
|
|
@@ -108,19 +113,19 @@ public class AllocationFileLoaderService extends AbstractService {
|
|
public AllocationFileLoaderService(Clock clock) {
|
|
public AllocationFileLoaderService(Clock clock) {
|
|
super(AllocationFileLoaderService.class.getName());
|
|
super(AllocationFileLoaderService.class.getName());
|
|
this.clock = clock;
|
|
this.clock = clock;
|
|
-
|
|
|
|
}
|
|
}
|
|
|
|
|
|
@Override
|
|
@Override
|
|
public void serviceInit(Configuration conf) throws Exception {
|
|
public void serviceInit(Configuration conf) throws Exception {
|
|
this.allocFile = getAllocationFile(conf);
|
|
this.allocFile = getAllocationFile(conf);
|
|
- if (allocFile != null) {
|
|
|
|
- reloadThread = new Thread() {
|
|
|
|
- @Override
|
|
|
|
- public void run() {
|
|
|
|
- while (running) {
|
|
|
|
|
|
+ if(this.allocFile != null) {
|
|
|
|
+ this.fs = allocFile.getFileSystem(conf);
|
|
|
|
+ reloadThread = new Thread(() -> {
|
|
|
|
+ while (running) {
|
|
|
|
+ try {
|
|
long time = clock.getTime();
|
|
long time = clock.getTime();
|
|
- long lastModified = allocFile.lastModified();
|
|
|
|
|
|
+ long lastModified =
|
|
|
|
+ fs.getFileStatus(allocFile).getModificationTime();
|
|
if (lastModified > lastSuccessfulReload &&
|
|
if (lastModified > lastSuccessfulReload &&
|
|
time > lastModified + ALLOC_RELOAD_WAIT_MS) {
|
|
time > lastModified + ALLOC_RELOAD_WAIT_MS) {
|
|
try {
|
|
try {
|
|
@@ -136,19 +141,21 @@ public class AllocationFileLoaderService extends AbstractService {
|
|
if (!lastReloadAttemptFailed) {
|
|
if (!lastReloadAttemptFailed) {
|
|
LOG.warn("Failed to reload fair scheduler config file because" +
|
|
LOG.warn("Failed to reload fair scheduler config file because" +
|
|
" last modified returned 0. File exists: "
|
|
" last modified returned 0. File exists: "
|
|
- + allocFile.exists());
|
|
|
|
|
|
+ + fs.exists(allocFile));
|
|
}
|
|
}
|
|
lastReloadAttemptFailed = true;
|
|
lastReloadAttemptFailed = true;
|
|
}
|
|
}
|
|
- try {
|
|
|
|
- Thread.sleep(reloadIntervalMs);
|
|
|
|
- } catch (InterruptedException ex) {
|
|
|
|
- LOG.info(
|
|
|
|
- "Interrupted while waiting to reload alloc configuration");
|
|
|
|
- }
|
|
|
|
|
|
+ } catch (IOException e) {
|
|
|
|
+ LOG.info("Exception while loading allocation file: " + e);
|
|
|
|
+ }
|
|
|
|
+ try {
|
|
|
|
+ Thread.sleep(reloadIntervalMs);
|
|
|
|
+ } catch (InterruptedException ex) {
|
|
|
|
+ LOG.info(
|
|
|
|
+ "Interrupted while waiting to reload alloc configuration");
|
|
}
|
|
}
|
|
}
|
|
}
|
|
- };
|
|
|
|
|
|
+ });
|
|
reloadThread.setName("AllocationFileReloader");
|
|
reloadThread.setName("AllocationFileReloader");
|
|
reloadThread.setDaemon(true);
|
|
reloadThread.setDaemon(true);
|
|
}
|
|
}
|
|
@@ -182,24 +189,31 @@ public class AllocationFileLoaderService extends AbstractService {
|
|
* path is relative, it is searched for in the
|
|
* path is relative, it is searched for in the
|
|
* classpath, but loaded like a regular File.
|
|
* classpath, but loaded like a regular File.
|
|
*/
|
|
*/
|
|
- public File getAllocationFile(Configuration conf) {
|
|
|
|
|
|
+ public Path getAllocationFile(Configuration conf)
|
|
|
|
+ throws UnsupportedFileSystemException {
|
|
String allocFilePath = conf.get(FairSchedulerConfiguration.ALLOCATION_FILE,
|
|
String allocFilePath = conf.get(FairSchedulerConfiguration.ALLOCATION_FILE,
|
|
FairSchedulerConfiguration.DEFAULT_ALLOCATION_FILE);
|
|
FairSchedulerConfiguration.DEFAULT_ALLOCATION_FILE);
|
|
- File allocFile = new File(allocFilePath);
|
|
|
|
- if (!allocFile.isAbsolute()) {
|
|
|
|
|
|
+ Path allocPath = new Path(allocFilePath);
|
|
|
|
+ String allocPathScheme = allocPath.toUri().getScheme();
|
|
|
|
+ if(allocPathScheme != null && !allocPathScheme.matches(SUPPORTED_FS_REGEX)){
|
|
|
|
+ throw new UnsupportedFileSystemException("Allocation file "
|
|
|
|
+ + allocFilePath + " uses an unsupported filesystem");
|
|
|
|
+ } else if (!allocPath.isAbsolute()) {
|
|
URL url = Thread.currentThread().getContextClassLoader()
|
|
URL url = Thread.currentThread().getContextClassLoader()
|
|
.getResource(allocFilePath);
|
|
.getResource(allocFilePath);
|
|
if (url == null) {
|
|
if (url == null) {
|
|
LOG.warn(allocFilePath + " not found on the classpath.");
|
|
LOG.warn(allocFilePath + " not found on the classpath.");
|
|
- allocFile = null;
|
|
|
|
|
|
+ allocPath = null;
|
|
} else if (!url.getProtocol().equalsIgnoreCase("file")) {
|
|
} else if (!url.getProtocol().equalsIgnoreCase("file")) {
|
|
throw new RuntimeException("Allocation file " + url
|
|
throw new RuntimeException("Allocation file " + url
|
|
+ " found on the classpath is not on the local filesystem.");
|
|
+ " found on the classpath is not on the local filesystem.");
|
|
} else {
|
|
} else {
|
|
- allocFile = new File(url.getPath());
|
|
|
|
|
|
+ allocPath = new Path(url.getProtocol(), null, url.getPath());
|
|
}
|
|
}
|
|
|
|
+ } else if (allocPath.isAbsoluteAndSchemeAuthorityNull()){
|
|
|
|
+ allocPath = new Path("file", null, allocFilePath);
|
|
}
|
|
}
|
|
- return allocFile;
|
|
|
|
|
|
+ return allocPath;
|
|
}
|
|
}
|
|
|
|
|
|
public synchronized void setReloadListener(Listener reloadListener) {
|
|
public synchronized void setReloadListener(Listener reloadListener) {
|
|
@@ -274,7 +288,7 @@ public class AllocationFileLoaderService extends AbstractService {
|
|
DocumentBuilderFactory.newInstance();
|
|
DocumentBuilderFactory.newInstance();
|
|
docBuilderFactory.setIgnoringComments(true);
|
|
docBuilderFactory.setIgnoringComments(true);
|
|
DocumentBuilder builder = docBuilderFactory.newDocumentBuilder();
|
|
DocumentBuilder builder = docBuilderFactory.newDocumentBuilder();
|
|
- Document doc = builder.parse(allocFile);
|
|
|
|
|
|
+ Document doc = builder.parse(fs.open(allocFile));
|
|
Element root = doc.getDocumentElement();
|
|
Element root = doc.getDocumentElement();
|
|
if (!"allocations".equals(root.getTagName()))
|
|
if (!"allocations".equals(root.getTagName()))
|
|
throw new AllocationConfigurationException("Bad fair scheduler config " +
|
|
throw new AllocationConfigurationException("Bad fair scheduler config " +
|
|
@@ -437,7 +451,7 @@ public class AllocationFileLoaderService extends AbstractService {
|
|
fairSharePreemptionTimeouts, fairSharePreemptionThresholds, queueAcls,
|
|
fairSharePreemptionTimeouts, fairSharePreemptionThresholds, queueAcls,
|
|
reservationAcls, newPlacementPolicy, configuredQueues,
|
|
reservationAcls, newPlacementPolicy, configuredQueues,
|
|
globalReservationQueueConfig, reservableQueues, nonPreemptableQueues);
|
|
globalReservationQueueConfig, reservableQueues, nonPreemptableQueues);
|
|
-
|
|
|
|
|
|
+
|
|
lastSuccessfulReload = clock.getTime();
|
|
lastSuccessfulReload = clock.getTime();
|
|
lastReloadAttemptFailed = false;
|
|
lastReloadAttemptFailed = false;
|
|
|
|
|