You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@nifi.apache.org by ma...@apache.org on 2014/12/11 15:46:48 UTC

[1/2] incubator-nifi git commit: NIFI-145: Added run-nifi.bat and fixed bugs

Repository: incubator-nifi
Updated Branches:
  refs/heads/bootstrap e544191f1 -> 192d78227


NIFI-145: Added run-nifi.bat and fixed bugs


Project: http://git-wip-us.apache.org/repos/asf/incubator-nifi/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-nifi/commit/6d468297
Tree: http://git-wip-us.apache.org/repos/asf/incubator-nifi/tree/6d468297
Diff: http://git-wip-us.apache.org/repos/asf/incubator-nifi/diff/6d468297

Branch: refs/heads/bootstrap
Commit: 6d46829795196afc797448cc386783fb147c0a7d
Parents: e544191
Author: Mark Payne <ma...@hotmail.com>
Authored: Wed Dec 10 09:23:51 2014 -0500
Committer: Mark Payne <ma...@hotmail.com>
Committed: Wed Dec 10 09:23:51 2014 -0500

----------------------------------------------------------------------
 .../src/main/resources/bin/run-nifi.bat         |  15 +++
 .../src/main/resources/conf/bootstrap.conf      |   7 ++
 .../java/org/apache/nifi/BootstrapListener.java |  19 ++--
 .../src/main/java/org/apache/nifi/NiFi.java     |  15 ++-
 .../java/org/apache/nifi/bootstrap/RunNiFi.java | 109 ++++++++++++++++---
 .../org/apache/nifi/bootstrap/ShutdownHook.java |  12 +-
 6 files changed, 144 insertions(+), 33 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/6d468297/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat
new file mode 100644
index 0000000..ee00204
--- /dev/null
+++ b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat
@@ -0,0 +1,15 @@
+@echo off
+
+rem Use JAVA_HOME if it's set; otherwise, just use java
+IF "%JAVA_HOME%"=="" (SET JAVA_EXE=java) ELSE (SET JAVA_EXE=%JAVA_HOME%\bin\java.exe)
+
+SET LIB_DIR=%~dp0..\lib
+SET CONF_DIR=%~dp0..\conf
+
+SET BOOTSTRAP_CONF_FILE=%CONF_DIR%\bootstrap.conf
+SET JAVA_ARGS=-Dorg.apache.nifi.boostrap.config.file=%BOOTSTRAP_CONF_FILE%
+
+SET JAVA_PARAMS=-cp %LIB_DIR%\nifi-bootstrap*.jar -Xms12m -Xmx24m %JAVA_ARGS% org.apache.nifi.bootstrap.RunNiFi
+SET BOOTSTRAP_ACTION=run
+
+cmd.exe /C "%JAVA_EXE%" %JAVA_PARAMS% %BOOTSTRAP_ACTION%

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/6d468297/nar-bundles/framework-bundle/framework/resources/src/main/resources/conf/bootstrap.conf
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/resources/src/main/resources/conf/bootstrap.conf b/nar-bundles/framework-bundle/framework/resources/src/main/resources/conf/bootstrap.conf
index 97d48f8..c45d8f8 100644
--- a/nar-bundles/framework-bundle/framework/resources/src/main/resources/conf/bootstrap.conf
+++ b/nar-bundles/framework-bundle/framework/resources/src/main/resources/conf/bootstrap.conf
@@ -1,7 +1,14 @@
+# Configure where NiFi's lib and conf directories live
 lib.dir=./lib
 conf.dir=./conf
+
+# How long to wait after telling NiFi to shutdown before explicitly killing the Process
+graceful.shutdown.seconds=20
+
+# Disable JSR 199 so that we can use JSP's without running a JDK
 java.arg.1=-Dorg.apache.jasper.compiler.disablejsr199=true
 
+# JVM memory settings
 java.arg.2=-Xms256m
 java.arg.3=-Xmx512m
 

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/6d468297/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/BootstrapListener.java
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/BootstrapListener.java b/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/BootstrapListener.java
index 3bcbeb3..31f336c 100644
--- a/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/BootstrapListener.java
+++ b/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/BootstrapListener.java
@@ -24,6 +24,7 @@ import java.io.OutputStream;
 import java.net.InetSocketAddress;
 import java.net.ServerSocket;
 import java.net.Socket;
+import java.net.SocketTimeoutException;
 import java.nio.charset.StandardCharsets;
 import java.util.Arrays;
 import java.util.concurrent.ExecutorService;
@@ -38,8 +39,8 @@ public class BootstrapListener {
 	private final NiFi nifi;
 	private final int bootstrapPort;
 
-	private Listener listener;
-	private ServerSocket serverSocket;
+	private volatile Listener listener;
+	private volatile ServerSocket serverSocket;
 	
 	
 	public BootstrapListener(final NiFi nifi, final int port) {
@@ -52,12 +53,14 @@ public class BootstrapListener {
 		
 		serverSocket = new ServerSocket();
 		serverSocket.bind(new InetSocketAddress("localhost", 0));
+		serverSocket.setSoTimeout(2000);
 		
 		final int localPort = serverSocket.getLocalPort();
 		logger.info("Started Bootstrap Listener, Listening for incoming requests on port {}", localPort);
 		
 		listener = new Listener(serverSocket);
 		final Thread listenThread = new Thread(listener);
+		listenThread.setDaemon(true);
 		listenThread.setName("Listen to Bootstrap");
 		listenThread.start();
 		
@@ -114,15 +117,17 @@ public class BootstrapListener {
 		
 		@Override
 		public void run() {
-			while (!serverSocket.isClosed()) {
+			while (!stopped) {
 				try {
-					if ( stopped ) {
-						return;
-					}
-					
 					final Socket socket;
 					try {
 						socket = serverSocket.accept();
+					} catch (final SocketTimeoutException ste) {
+						if ( stopped ) {
+							return;
+						}
+						
+						continue;
 					} catch (final IOException ioe) {
 						if ( stopped ) {
 							return;

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/6d468297/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/NiFi.java
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/NiFi.java b/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/NiFi.java
index bf50a21..13cd4d6 100644
--- a/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/NiFi.java
+++ b/nar-bundles/framework-bundle/framework/runtime/src/main/java/org/apache/nifi/NiFi.java
@@ -48,6 +48,7 @@ public class NiFi {
     private final BootstrapListener bootstrapListener;
     
     public static final String BOOTSTRAP_PORT_PROPERTY = "nifi.bootstrap.listen.port";
+    private volatile boolean shutdown = false;
 
     public NiFi(final NiFiProperties properties) throws ClassNotFoundException, IOException, NoSuchMethodException, InstantiationException, IllegalAccessException, IllegalArgumentException, InvocationTargetException {
         Thread.setDefaultUncaughtExceptionHandler(new UncaughtExceptionHandler() {
@@ -126,13 +127,21 @@ public class NiFi {
         final long startTime = System.nanoTime();
         nifiServer = (NiFiServer) jettyConstructor.newInstance(properties);
         nifiServer.setExtensionMapping(extensionMapping);
-        nifiServer.start();
-        final long endTime = System.nanoTime();
-        logger.info("Controller initialization took " + (endTime - startTime) + " nanoseconds.");
+        
+        if ( shutdown ) {
+        	logger.info("NiFi has been shutdown via NiFi Bootstrap. Will not start Controller");
+        } else {
+	        nifiServer.start();
+	        
+	        final long endTime = System.nanoTime();
+	        logger.info("Controller initialization took " + (endTime - startTime) + " nanoseconds.");
+        }
     }
 
     protected void shutdownHook() {
         try {
+        	this.shutdown = true;
+        	
             logger.info("Initiating shutdown of Jetty web server...");
             if (nifiServer != null) {
                 nifiServer.stop();

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/6d468297/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java
----------------------------------------------------------------------
diff --git a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java
index 54932c8..f93500f 100644
--- a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java
+++ b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java
@@ -57,6 +57,9 @@ public class RunNiFi {
 	public static final String DEFAULT_CONFIG_FILE = "./conf/boostrap.conf";
 	public static final String DEFAULT_NIFI_PROPS_FILE = "./conf/nifi.properties";
 
+	public static final String GRACEFUL_SHUTDOWN_PROP = "graceful.shutdown.seconds";
+	public static final String DEFAULT_GRACEFUL_SHUTDOWN_VALUE = "20";
+	
 	public static final int MAX_RESTART_ATTEMPTS = 5;
 	public static final int STARTUP_WAIT_SECONDS = 60;
 	
@@ -85,6 +88,7 @@ public class RunNiFi {
 		System.out.println("Start : Start a new instance of Apache NiFi");
 		System.out.println("Stop : Stop a running instance of Apache NiFi");
 		System.out.println("Status : Determine if there is a running instance of Apache NiFi");
+		System.out.println("Run : Start a new instance of Apache NiFi and monitor the Process, restarting if the instance dies");
 		System.out.println();
 	}
 	
@@ -96,6 +100,7 @@ public class RunNiFi {
 		
 		switch (args[0].toLowerCase()) {
 			case "start":
+			case "run":
 			case "stop":
 			case "status":
 				break;
@@ -127,7 +132,10 @@ public class RunNiFi {
 		
 		switch (args[0].toLowerCase()) {
 			case "start":
-				runNiFi.start();
+				runNiFi.start(false);
+				break;
+			case "run":
+				runNiFi.start(true);
 				break;
 			case "stop":
 				runNiFi.stop();
@@ -140,8 +148,10 @@ public class RunNiFi {
 	
 	
 	public File getStatusFile() {
-		final File rootDir = bootstrapConfigFile.getParentFile();
-		final File statusFile = new File(rootDir, "nifi.port");
+		final File confDir = bootstrapConfigFile.getParentFile();
+		final File nifiHome = confDir.getParentFile();
+		final File bin = new File(nifiHome, "bin");
+		final File statusFile = new File(bin, "nifi.port");
 		return statusFile;
 	}
 
@@ -165,11 +175,7 @@ public class RunNiFi {
 					return port;
 				}
 			} catch (final IOException ioe) {
-				System.out.println("Found NiFi instance info at " + statusFile + " but information appears to be stale. Removing file.");
-				if ( !statusFile.delete() ) {
-					System.err.println("Unable to remove status file");
-				}
-				
+				System.out.println("Found NiFi instance info at " + statusFile + " indicating that NiFi is running and listening to port " + port + " but unable to communicate with NiFi on that port. The process may have died or may be hung.");
 				throw ioe;
 			}
 		} catch (final Exception e) {
@@ -212,6 +218,11 @@ public class RunNiFi {
 			final String response = reader.readLine();
 			if ( SHUTDOWN_CMD.equals(response) ) {
 				System.out.println("Apache NiFi has accepted the Shutdown Command and is shutting down now");
+				
+				final File statusFile = getStatusFile();
+				if ( !statusFile.delete() ) {
+					System.err.println("Failed to delete status file " + statusFile + "; this file should be cleaned up manually");
+				}
 			} else {
 				System.err.println("When sending SHUTDOWN command to NiFi, got unexpected response " + response);
 			}
@@ -222,8 +233,17 @@ public class RunNiFi {
 	}
 	
 	
+	private boolean isAlive(final Process process) {
+		try {
+			process.exitValue();
+			return false;
+		} catch (final IllegalThreadStateException itse) {
+			return true;
+		}
+	}
+	
 	@SuppressWarnings({ "rawtypes", "unchecked" })
-	public void start() throws IOException, InterruptedException {
+	public void start(final boolean monitor) throws IOException, InterruptedException {
 		final Integer port = getCurrentPort();
 		if ( port != null ) {
 			System.out.println("Apache NiFi is already running, listening on port " + port);
@@ -344,16 +364,71 @@ public class RunNiFi {
 		System.out.println("Working Directory: " + workingDir.getAbsolutePath());
 		System.out.println("Command: " + cmdBuilder.toString());
 		
-		builder.start();
-		boolean started = waitForStart();
-		
-		if ( started ) {
-			System.out.println("Successfully started Apache NiFi");
+		if ( monitor ) {
+			String gracefulShutdown = props.get(GRACEFUL_SHUTDOWN_PROP);
+			if ( gracefulShutdown == null ) {
+				gracefulShutdown = DEFAULT_GRACEFUL_SHUTDOWN_VALUE;
+			}
+
+			final int gracefulShutdownSeconds;
+			try {
+				gracefulShutdownSeconds = Integer.parseInt(gracefulShutdown);
+			} catch (final NumberFormatException nfe) {
+				throw new NumberFormatException("The '" + GRACEFUL_SHUTDOWN_PROP + "' property in Boostrap Config File " + bootstrapConfigAbsoluteFile.getAbsolutePath() + " has an invalid value. Must be a non-negative integer");
+			}
+			
+			if ( gracefulShutdownSeconds < 0 ) {
+				throw new NumberFormatException("The '" + GRACEFUL_SHUTDOWN_PROP + "' property in Boostrap Config File " + bootstrapConfigAbsoluteFile.getAbsolutePath() + " has an invalid value. Must be a non-negative integer");
+			}
+			
+			Process process = builder.start();
+			
+			ShutdownHook shutdownHook = new ShutdownHook(process, this, gracefulShutdownSeconds);
+			final Runtime runtime = Runtime.getRuntime();
+			runtime.addShutdownHook(shutdownHook);
+			
+			while (true) {
+				final boolean alive = isAlive(process);
+				
+				if ( alive ) {
+					try {
+						Thread.sleep(1000L);
+					} catch (final InterruptedException ie) {
+					}
+				} else {
+					runtime.removeShutdownHook(shutdownHook);
+					
+					if (autoRestartNiFi) {
+						System.out.println("Apache NiFi appears to have died. Restarting...");
+						process = builder.start();
+						
+						shutdownHook = new ShutdownHook(process, this, gracefulShutdownSeconds);
+						runtime.addShutdownHook(shutdownHook);
+						
+						final boolean started = waitForStart();
+						
+						if ( started ) {
+							System.out.println("Successfully started Apache NiFi");
+						} else {
+							System.err.println("Apache NiFi does not appear to have started");
+						}
+					} else {
+						return;
+					}
+				}
+			}
 		} else {
-			System.err.println("Apache NiFi does not appear to have started");
+			builder.start();
+			boolean started = waitForStart();
+			
+			if ( started ) {
+				System.out.println("Successfully started Apache NiFi");
+			} else {
+				System.err.println("Apache NiFi does not appear to have started");
+			}
+			
+			listener.stop();
 		}
-		
-		listener.stop();
 	}
 	
 	

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/6d468297/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java
----------------------------------------------------------------------
diff --git a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java
index f804c7c..781b690 100644
--- a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java
+++ b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java
@@ -26,12 +26,12 @@ import java.util.concurrent.TimeUnit;
 public class ShutdownHook extends Thread {
 	private final Process nifiProcess;
 	private final RunNiFi runner;
+	private final int gracefulShutdownSeconds;
 	
-	public static final int WAIT_SECONDS = 10;
-	
-	public ShutdownHook(final Process nifiProcess, final RunNiFi runner) {
+	public ShutdownHook(final Process nifiProcess, final RunNiFi runner, final int gracefulShutdownSeconds) {
 		this.nifiProcess = nifiProcess;
 		this.runner = runner;
+		this.gracefulShutdownSeconds = gracefulShutdownSeconds;
 	}
 	
 	@Override
@@ -58,9 +58,9 @@ public class ShutdownHook extends Thread {
 		while ( isAlive(nifiProcess) ) {
 			final long waitNanos = System.nanoTime() - startWait;
 			final long waitSeconds = TimeUnit.NANOSECONDS.toSeconds(waitNanos);
-			if ( waitSeconds >= WAIT_SECONDS ) {
+			if ( waitSeconds >= gracefulShutdownSeconds && gracefulShutdownSeconds > 0 ) {
 				if ( isAlive(nifiProcess) ) {
-					System.out.println("NiFi has not finished shutting down after " + WAIT_SECONDS + " seconds. Killing process.");
+					System.out.println("NiFi has not finished shutting down after " + gracefulShutdownSeconds + " seconds. Killing process.");
 					nifiProcess.destroy();
 				}
 				break;
@@ -73,7 +73,7 @@ public class ShutdownHook extends Thread {
 		
 		final File statusFile = runner.getStatusFile();
 		if ( !statusFile.delete() ) {
-			System.err.println("Failed to delete status file " + statusFile.getAbsolutePath());
+			System.err.println("Failed to delete status file " + statusFile.getAbsolutePath() + "; this file should be cleaned up manually");
 		}
 	}
 	


[2/2] incubator-nifi git commit: NIFI-145: Bug Fixes and updated nifi.sh to use bootstrap code

Posted by ma...@apache.org.
NIFI-145: Bug Fixes and updated nifi.sh to use bootstrap code


Project: http://git-wip-us.apache.org/repos/asf/incubator-nifi/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-nifi/commit/192d7822
Tree: http://git-wip-us.apache.org/repos/asf/incubator-nifi/tree/192d7822
Diff: http://git-wip-us.apache.org/repos/asf/incubator-nifi/diff/192d7822

Branch: refs/heads/bootstrap
Commit: 192d7822772752cb939921af1610476c8e7dcfda
Parents: 6d46829
Author: Mark Payne <ma...@hotmail.com>
Authored: Wed Dec 10 11:03:21 2014 -0500
Committer: Mark Payne <ma...@hotmail.com>
Committed: Wed Dec 10 11:03:21 2014 -0500

----------------------------------------------------------------------
 .../apache/nifi/controller/FlowController.java  |  15 ++
 .../repository/FileSystemRepository.java        |   4 +-
 .../WriteAheadFlowFileRepository.java           |   7 +-
 .../scheduling/StandardProcessScheduler.java    |   3 +
 .../java/org/apache/nifi/engine/FlowEngine.java |  15 ++
 .../src/main/resources/bin/nifi-status.bat      |  19 +-
 .../resources/src/main/resources/bin/nifi.sh    | 193 +------------------
 .../src/main/resources/bin/run-nifi.bat         |  19 +-
 .../src/main/resources/bin/start-nifi.bat       |  19 +-
 .../src/main/resources/bin/stop-nifi.bat        |  19 +-
 .../java/org/apache/nifi/bootstrap/RunNiFi.java |  15 +-
 .../org/apache/nifi/bootstrap/ShutdownHook.java |  13 +-
 12 files changed, 130 insertions(+), 211 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/FlowController.java
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/FlowController.java b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/FlowController.java
index a0a07f2..20c50b5 100644
--- a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/FlowController.java
+++ b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/FlowController.java
@@ -1045,6 +1045,21 @@ public class FlowController implements EventAccess, ControllerServiceProvider, H
             if (flowFileSwapManager != null) {
                 flowFileSwapManager.shutdown();
             }
+            
+            if ( processScheduler != null ) {
+            	processScheduler.shutdown();
+            }
+            
+            if ( provenanceEventRepository != null ) {
+            	try {
+            		provenanceEventRepository.close();
+            	} catch (final IOException ioe) {
+            		LOG.warn("There was a problem shutting down the Provenance Repository: " + ioe.toString());
+            		if ( LOG.isDebugEnabled() ) {
+            			LOG.warn("", ioe);
+            		}
+            	}
+            }
         } finally {
             writeLock.unlock();
         }

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/FileSystemRepository.java
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/FileSystemRepository.java b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/FileSystemRepository.java
index 8be9c62..ba74295 100644
--- a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/FileSystemRepository.java
+++ b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/FileSystemRepository.java
@@ -92,7 +92,7 @@ public class FileSystemRepository implements ContentRepository {
     private final List<String> containerNames;
     private final AtomicLong index;
 
-    private final ScheduledExecutorService executor = new FlowEngine(4, "FileSystemRepository Workers");
+    private final ScheduledExecutorService executor = new FlowEngine(4, "FileSystemRepository Workers", true);
     private final ConcurrentMap<String, BlockingQueue<ContentClaim>> reclaimable = new ConcurrentHashMap<>();
     private final Map<String, ContainerState> containerStateMap = new HashMap<>();
 
@@ -209,7 +209,7 @@ public class FileSystemRepository implements ContentRepository {
             }
         }
 
-        containerCleanupExecutor = new FlowEngine(containers.size(), "Cleanup FileSystemRepository Container");
+        containerCleanupExecutor = new FlowEngine(containers.size(), "Cleanup FileSystemRepository Container", true);
         for (final Map.Entry<String, Path> containerEntry : containers.entrySet()) {
             final String containerName = containerEntry.getKey();
             final Path containerPath = containerEntry.getValue();

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/WriteAheadFlowFileRepository.java
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/WriteAheadFlowFileRepository.java b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/WriteAheadFlowFileRepository.java
index ab3a6a1..292c258 100644
--- a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/WriteAheadFlowFileRepository.java
+++ b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/repository/WriteAheadFlowFileRepository.java
@@ -90,6 +90,7 @@ public class WriteAheadFlowFileRepository implements FlowFileRepository, SyncLis
     private final long checkpointDelayMillis;
     private final Path flowFileRepositoryPath;
     private final int numPartitions;
+    private final ScheduledExecutorService checkpointExecutor;
 
     // effectively final
     private WriteAheadRepository<RepositoryRecord> wal;
@@ -128,6 +129,8 @@ public class WriteAheadFlowFileRepository implements FlowFileRepository, SyncLis
         flowFileRepositoryPath = properties.getFlowFileRepositoryPath();
         numPartitions = properties.getFlowFileRepositoryPartitions();
         checkpointDelayMillis = FormatUtils.getTimeDuration(properties.getFlowFileRepositoryCheckpointInterval(), TimeUnit.MILLISECONDS);
+        
+        checkpointExecutor = Executors.newSingleThreadScheduledExecutor();
     }
 
     @Override
@@ -150,6 +153,7 @@ public class WriteAheadFlowFileRepository implements FlowFileRepository, SyncLis
             checkpointFuture.cancel(false);
         }
 
+        checkpointExecutor.shutdown();
         wal.shutdown();
     }
 
@@ -363,8 +367,7 @@ public class WriteAheadFlowFileRepository implements FlowFileRepository, SyncLis
             }
         };
 
-        final ScheduledExecutorService executorService = Executors.newSingleThreadScheduledExecutor();
-        checkpointFuture = executorService.scheduleWithFixedDelay(checkpointRunnable, checkpointDelayMillis, checkpointDelayMillis, TimeUnit.MILLISECONDS);
+        checkpointFuture = checkpointExecutor.scheduleWithFixedDelay(checkpointRunnable, checkpointDelayMillis, checkpointDelayMillis, TimeUnit.MILLISECONDS);
 
         return maxId;
     }

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/scheduling/StandardProcessScheduler.java
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/scheduling/StandardProcessScheduler.java b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/scheduling/StandardProcessScheduler.java
index 0d5055a..7fc65f9 100644
--- a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/scheduling/StandardProcessScheduler.java
+++ b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/controller/scheduling/StandardProcessScheduler.java
@@ -138,6 +138,9 @@ public final class StandardProcessScheduler implements ProcessScheduler {
                 LOG.error("", t);
             }
         }
+        
+        frameworkTaskExecutor.shutdown();
+        componentLifeCycleThreadPool.shutdown();
     }
 
     public void schedule(final ReportingTaskNode taskNode) {

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/engine/FlowEngine.java
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/engine/FlowEngine.java b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/engine/FlowEngine.java
index 2430f56..76e8e3e 100644
--- a/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/engine/FlowEngine.java
+++ b/nar-bundles/framework-bundle/framework/core/src/main/java/org/apache/nifi/engine/FlowEngine.java
@@ -42,6 +42,18 @@ public final class FlowEngine extends ScheduledThreadPoolExecutor {
      * @param threadNamePrefix
      */
     public FlowEngine(int corePoolSize, final String threadNamePrefix) {
+    	this(corePoolSize, threadNamePrefix, false);
+    }
+    	
+    /**
+     * Creates a new instance of FlowEngine
+     *
+     * @param corePoolSize the maximum number of threads available to tasks
+     * running in the engine.
+     * @param threadNamePrefix
+     * @param deamon if true, the thread pool will be populated with daemon threads, otherwise the threads will not be marked as daemon.
+     */
+    public FlowEngine(int corePoolSize, final String threadNamePrefix, final boolean daemon) {
         super(corePoolSize);
 
         final AtomicInteger threadIndex = new AtomicInteger(0);
@@ -50,6 +62,9 @@ public final class FlowEngine extends ScheduledThreadPoolExecutor {
             @Override
             public Thread newThread(final Runnable r) {
                 final Thread t = defaultThreadFactory.newThread(r);
+                if ( daemon ) {
+                	t.setDaemon(true);
+                }
                 t.setName(threadNamePrefix + " Thread-" + threadIndex.incrementAndGet());
                 return t;
             }

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi-status.bat
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi-status.bat b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi-status.bat
index 9b88349..ed9c516 100644
--- a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi-status.bat
+++ b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi-status.bat
@@ -1,3 +1,20 @@
+rem
+rem    Licensed to the Apache Software Foundation (ASF) under one or more
+rem    contributor license agreements.  See the NOTICE file distributed with
+rem    this work for additional information regarding copyright ownership.
+rem    The ASF licenses this file to You under the Apache License, Version 2.0
+rem    (the "License"); you may not use this file except in compliance with
+rem    the License.  You may obtain a copy of the License at
+rem
+rem       http://www.apache.org/licenses/LICENSE-2.0
+rem
+rem    Unless required by applicable law or agreed to in writing, software
+rem    distributed under the License is distributed on an "AS IS" BASIS,
+rem    WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+rem    See the License for the specific language governing permissions and
+rem    limitations under the License.
+rem
+
 @echo off
 
 rem Use JAVA_HOME if it's set; otherwise, just use java
@@ -7,7 +24,7 @@ SET LIB_DIR=%~dp0..\lib
 SET CONF_DIR=%~dp0..\conf
 
 SET BOOTSTRAP_CONF_FILE=%CONF_DIR%\bootstrap.conf
-SET JAVA_ARGS=-Dorg.apache.nifi.boostrap.config.file=%BOOTSTRAP_CONF_FILE%
+SET JAVA_ARGS=-Dorg.apache.nifi.bootstrap.config.file=%BOOTSTRAP_CONF_FILE%
 
 SET JAVA_PARAMS=-cp %LIB_DIR%\nifi-bootstrap*.jar -Xms12m -Xmx24m %JAVA_ARGS% org.apache.nifi.bootstrap.RunNiFi
 SET BOOTSTRAP_ACTION=status

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi.sh
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi.sh b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi.sh
index d069fc9..81ae384 100644
--- a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi.sh
+++ b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/nifi.sh
@@ -21,34 +21,6 @@
 DIRNAME=`dirname "$0"`
 PROGNAME=`basename "$0"`
 
-#
-# Sourcing environment settings for NIFI similar to tomcats setenv
-#
-NIFI_SCRIPT="nifi.sh"
-export NIFI_SCRIPT
-if [ -f "$DIRNAME/setenv.sh" ]; then
-  . "$DIRNAME/setenv.sh"
-fi
-
-#
-# Check/Set up some easily accessible MIN/MAX params for JVM mem usage
-#
-if [ "x$JAVA_MIN_MEM" = "x" ]; then
-    JAVA_MIN_MEM=512M
-    export JAVA_MIN_MEM
-fi
-if [ "x$JAVA_MAX_MEM" = "x" ]; then
-    JAVA_MAX_MEM=512M
-    export JAVA_MAX_MEM
-fi
-if [ "x$JAVA_PERMSIZE" = "x" ]; then
-    JAVA_PERMSIZE=128M
-    export JAVA_PERMSIZE
-fi
-if [ "x$JAVA_MAX_PERMSIZE" = "x" ]; then
-    JAVA_MAX_PERMSIZE=128M
-    export JAVA_MAX_PERMSIZE
-fi
 
 
 warn() {
@@ -128,58 +100,6 @@ locateHome() {
 
 }
 
-locateBase() {
-    if [ "x$NIFI_BASE" != "x" ]; then
-        if [ ! -d "$NIFI_BASE" ]; then
-            die "NIFI_BASE is not valid: $NIFI_BASE"
-        fi
-    else
-        NIFI_BASE=$NIFI_HOME
-    fi
-}
-
-
-locateConf() {
-    if [ "x$NIFI_CONF" != "x" ]; then
-        if [ ! -d "$NIFI_CONF" ]; then
-            die "NIFI_CONF is not valid: $NIFI_CONF"
-        fi
-    else
-        NIFI_CONF=$NIFI_BASE/conf
-    fi
-}
-
-setupNativePath() {
-    # Support for loading native libraries
-    LD_LIBRARY_PATH="${LD_LIBRARY_PATH}:$NIFI_BASE/lib:$NIFI_HOME/lib"
-
-    # For Cygwin, set PATH from LD_LIBRARY_PATH
-    if $cygwin; then
-        LD_LIBRARY_PATH=`cygpath --path --windows "$LD_LIBRARY_PATH"`
-        PATH="$PATH;$LD_LIBRARY_PATH"
-        export PATH
-    fi
-    export LD_LIBRARY_PATH
-}
-
-pathCanonical() {
-    local dst="${1}"
-    while [ -h "${dst}" ] ; do
-        ls=`ls -ld "${dst}"`
-        link=`expr "$ls" : '.*-> \(.*\)$'`
-        if expr "$link" : '/.*' > /dev/null; then
-            dst="$link"
-        else
-            dst="`dirname "${dst}"`/$link"
-        fi
-    done
-    local bas=`basename "${dst}"`
-    local dir=`dirname "${dst}"`
-    if [ "$bas" != "$dir" ]; then
-        dst="`pathCanonical "$dir"`/$bas"
-    fi
-    echo "${dst}" | sed -e 's#//#/#g' -e 's#/./#/#g' -e 's#/[^/]*/../#/#g'
-}
 
 locateJava() {
     # Setup the Java Virtual Machine
@@ -211,82 +131,6 @@ locateJava() {
     fi
 }
 
-detectJVM() {
-   #echo "`$JAVA -version`"
-   # This service should call `java -version`,
-   # read stdout, and look for hints
-   if $JAVA -version 2>&1 | grep "^IBM" ; then
-       JVM_VENDOR="IBM"
-   # on OS/400, java -version does not contain IBM explicitly
-   elif $os400; then
-       JVM_VENDOR="IBM"
-   else
-       JVM_VENDOR="SUN"
-   fi
-   # echo "JVM vendor is $JVM_VENDOR"
-}
-
-setupDebugOptions() {
-    if [ "x$JAVA_OPTS" = "x" ]; then
-        JAVA_OPTS="$DEFAULT_JAVA_OPTS"
-    fi
-    export JAVA_OPTS
-
-    if [ "x$EXTRA_JAVA_OPTS" != "x" ]; then
-        JAVA_OPTS="$JAVA_OPTS $EXTRA_JAVA_OPTS"
-    fi
-
-    # Set Debug options if enabled
-    if [ "x$NIFI_DEBUG" != "x" ]; then
-        # Use the defaults if JAVA_DEBUG_OPTS was not set
-        if [ "x$JAVA_DEBUG_OPTS" = "x" ]; then
-            JAVA_DEBUG_OPTS="$DEFAULT_JAVA_DEBUG_OPTS"
-        fi
-
-        JAVA_OPTS="$JAVA_DEBUG_OPTS $JAVA_OPTS"
-        warn "Enabling Java debug options: $JAVA_DEBUG_OPTS"
-    fi
-}
-
-setupDefaults() {
-    DEFAULT_JAVA_OPTS="-Xms$JAVA_MIN_MEM -Xmx$JAVA_MAX_MEM -XX:PermSize=$JAVA_PERMSIZE -XX:MaxPermSize=$JAVA_MAX_PERMSIZE"
-
-    #Set the JVM_VENDOR specific JVM flags
-    if [ "$JVM_VENDOR" = "SUN" ]; then
-        #
-        # Check some easily accessible MIN/MAX params for JVM mem usage
-        #
-        if [ "x$JAVA_PERM_MEM" != "x" ]; then
-            DEFAULT_JAVA_OPTS="$DEFAULT_JAVA_OPTS -XX:PermSize=$JAVA_PERM_MEM"
-        fi
-        if [ "x$JAVA_MAX_PERM_MEM" != "x" ]; then
-            DEFAULT_JAVA_OPTS="$DEFAULT_JAVA_OPTS -XX:MaxPermSize=$JAVA_MAX_PERM_MEM"
-        fi
-        DEFAULT_JAVA_OPTS="-server $DEFAULT_JAVA_OPTS -Dcom.sun.management.jmxremote"
-    elif [ "$JVM_VENDOR" = "IBM" ]; then
-        if $os400; then
-            DEFAULT_JAVA_OPTS="$DEFAULT_JAVA_OPTS"
-        elif $aix; then
-            DEFAULT_JAVA_OPTS="-Xverify:none -Xdump:heap -Xlp $DEFAULT_JAVA_OPTS"
-        else
-            DEFAULT_JAVA_OPTS="-Xverify:none $DEFAULT_JAVA_OPTS"
-        fi
-    fi
-
-    DEFAULT_JAVA_OPTS="$DEFAULT_JAVA_OPTS  -Djava.net.preferIPv4Stack=true -Dsun.net.http.allowRestrictedHeaders=true -Djava.protocol.handler.pkgs=sun.net.www.protocol -Dorg.apache.jasper.compiler.disablejsr199=true -XX:ReservedCodeCacheSize=128m -XX:+UseCodeCacheFlushing"
-    
-    # Setup classpath
-    CLASSPATH="$NIFI_HOME"/conf
-    for f in "$NIFI_HOME"/lib/*
-    do
-      CLASSPATH="${CLASSPATH}":"${f}"
-    done
-    
-    
-    DEFAULT_JAVA_DEBUG_OPTS="-Xdebug -Xnoagent -Djava.compiler=NONE -Xrunjdwp:transport=dt_socket,server=y,suspend=n,address=5005"
-
-}
-
 init() {
     # Determine if there is special OS handling we must perform
     detectOS
@@ -297,49 +141,28 @@ init() {
     # Locate the NiFi home directory
     locateHome
 
-    # Locate the NiFi base directory
-    locateBase
-
-    # Locate the NiFi conf directory
-    locateConf
-
-    # Setup the native library path
-    setupNativePath
-
     # Locate the Java VM to execute
     locateJava
-
-    # Determine the JVM vendor
-    detectJVM
-
-    # Setup default options
-    setupDefaults
-
-    # Install debug options
-    setupDebugOptions
-
 }
 
 run() {
-
+	BOOTSTRAP_CONF="$NIFI_HOME/conf/bootstrap.conf";
+	
     if $cygwin; then
         NIFI_HOME=`cygpath --path --windows "$NIFI_HOME"`
-        NIFI_BASE=`cygpath --path --windows "$NIFI_BASE"`
-        NIFI_CONF=`cygpath --path --windows "$NIFI_CONF"`
-        CLASSPATH=`cygpath --path --windows "$CLASSPATH"`
+        BOOTSTRAP_CONF=`cygpath --path --windows "$BOOTSTRAP_CONF"`
     fi
-    # export CLASSPATH to the java process. Could also pass in via -cp
-    export CLASSPATH
+    
     echo 
     echo "Classpath: $CLASSPATH"
     echo
     echo "Java home: $JAVA_HOME"
     echo "NiFi home: $NIFI_HOME"
-    echo "Java Options: $JAVA_OPTS"
     echo
-    echo "Launching NiFi.  See logs..."
-    exec "$JAVA" -Dapp=nifi $JAVA_OPTS -Dnifi.properties.file.path="$NIFI_HOME"/conf/nifi.properties org.apache.nifi.NiFi
-
+    echo "Bootstrap Config File: $BOOTSTRAP_CONF"
+    echo
+    
+    exec "$JAVA" -cp "$NIFI_HOME"/lib/nifi-bootstrap*.jar -Xms12m -Xmx24m -Dorg.apache.nifi.bootstrap.config.file="$BOOTSTRAP_CONF" org.apache.nifi.bootstrap.RunNiFi $1
 }
 
 main() {

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat
index ee00204..fdff815 100644
--- a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat
+++ b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/run-nifi.bat
@@ -1,3 +1,20 @@
+rem
+rem    Licensed to the Apache Software Foundation (ASF) under one or more
+rem    contributor license agreements.  See the NOTICE file distributed with
+rem    this work for additional information regarding copyright ownership.
+rem    The ASF licenses this file to You under the Apache License, Version 2.0
+rem    (the "License"); you may not use this file except in compliance with
+rem    the License.  You may obtain a copy of the License at
+rem
+rem       http://www.apache.org/licenses/LICENSE-2.0
+rem
+rem    Unless required by applicable law or agreed to in writing, software
+rem    distributed under the License is distributed on an "AS IS" BASIS,
+rem    WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+rem    See the License for the specific language governing permissions and
+rem    limitations under the License.
+rem
+
 @echo off
 
 rem Use JAVA_HOME if it's set; otherwise, just use java
@@ -7,7 +24,7 @@ SET LIB_DIR=%~dp0..\lib
 SET CONF_DIR=%~dp0..\conf
 
 SET BOOTSTRAP_CONF_FILE=%CONF_DIR%\bootstrap.conf
-SET JAVA_ARGS=-Dorg.apache.nifi.boostrap.config.file=%BOOTSTRAP_CONF_FILE%
+SET JAVA_ARGS=-Dorg.apache.nifi.bootstrap.config.file=%BOOTSTRAP_CONF_FILE%
 
 SET JAVA_PARAMS=-cp %LIB_DIR%\nifi-bootstrap*.jar -Xms12m -Xmx24m %JAVA_ARGS% org.apache.nifi.bootstrap.RunNiFi
 SET BOOTSTRAP_ACTION=run

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/start-nifi.bat
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/start-nifi.bat b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/start-nifi.bat
index c088672..ba4739a 100644
--- a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/start-nifi.bat
+++ b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/start-nifi.bat
@@ -1,3 +1,20 @@
+rem
+rem    Licensed to the Apache Software Foundation (ASF) under one or more
+rem    contributor license agreements.  See the NOTICE file distributed with
+rem    this work for additional information regarding copyright ownership.
+rem    The ASF licenses this file to You under the Apache License, Version 2.0
+rem    (the "License"); you may not use this file except in compliance with
+rem    the License.  You may obtain a copy of the License at
+rem
+rem       http://www.apache.org/licenses/LICENSE-2.0
+rem
+rem    Unless required by applicable law or agreed to in writing, software
+rem    distributed under the License is distributed on an "AS IS" BASIS,
+rem    WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+rem    See the License for the specific language governing permissions and
+rem    limitations under the License.
+rem
+
 @echo off
 
 rem Use JAVA_HOME if it's set; otherwise, just use java
@@ -7,7 +24,7 @@ SET LIB_DIR=%~dp0..\lib
 SET CONF_DIR=%~dp0..\conf
 
 SET BOOTSTRAP_CONF_FILE=%CONF_DIR%\bootstrap.conf
-SET JAVA_ARGS=-Dorg.apache.nifi.boostrap.config.file=%BOOTSTRAP_CONF_FILE%
+SET JAVA_ARGS=-Dorg.apache.nifi.bootstrap.config.file=%BOOTSTRAP_CONF_FILE%
 
 SET JAVA_PARAMS=-cp %LIB_DIR%\nifi-bootstrap*.jar -Xms12m -Xmx24m %JAVA_ARGS% org.apache.nifi.bootstrap.RunNiFi
 SET BOOTSTRAP_ACTION=start

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/stop-nifi.bat
----------------------------------------------------------------------
diff --git a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/stop-nifi.bat b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/stop-nifi.bat
index 753b09f..828be6e 100644
--- a/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/stop-nifi.bat
+++ b/nar-bundles/framework-bundle/framework/resources/src/main/resources/bin/stop-nifi.bat
@@ -1,3 +1,20 @@
+rem
+rem    Licensed to the Apache Software Foundation (ASF) under one or more
+rem    contributor license agreements.  See the NOTICE file distributed with
+rem    this work for additional information regarding copyright ownership.
+rem    The ASF licenses this file to You under the Apache License, Version 2.0
+rem    (the "License"); you may not use this file except in compliance with
+rem    the License.  You may obtain a copy of the License at
+rem
+rem       http://www.apache.org/licenses/LICENSE-2.0
+rem
+rem    Unless required by applicable law or agreed to in writing, software
+rem    distributed under the License is distributed on an "AS IS" BASIS,
+rem    WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+rem    See the License for the specific language governing permissions and
+rem    limitations under the License.
+rem
+
 @echo off
 
 rem Use JAVA_HOME if it's set; otherwise, just use java
@@ -7,7 +24,7 @@ SET LIB_DIR=%~dp0..\lib
 SET CONF_DIR=%~dp0..\conf
 
 SET BOOTSTRAP_CONF_FILE=%CONF_DIR%\bootstrap.conf
-SET JAVA_ARGS=-Dorg.apache.nifi.boostrap.config.file=%BOOTSTRAP_CONF_FILE%
+SET JAVA_ARGS=-Dorg.apache.nifi.bootstrap.config.file=%BOOTSTRAP_CONF_FILE%
 
 SET JAVA_PARAMS=-cp %LIB_DIR%\nifi-bootstrap*.jar -Xms12m -Xmx24m %JAVA_ARGS% org.apache.nifi.bootstrap.RunNiFi
 SET BOOTSTRAP_ACTION=stop

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java
----------------------------------------------------------------------
diff --git a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java
index f93500f..af783a1 100644
--- a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java
+++ b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/RunNiFi.java
@@ -54,7 +54,7 @@ import java.util.concurrent.locks.ReentrantLock;
  * If the {@code bootstrap.conf} file cannot be found, throws a {@code FileNotFoundException].
  */
 public class RunNiFi {
-	public static final String DEFAULT_CONFIG_FILE = "./conf/boostrap.conf";
+	public static final String DEFAULT_CONFIG_FILE = "./conf/bootstrap.conf";
 	public static final String DEFAULT_NIFI_PROPS_FILE = "./conf/nifi.properties";
 
 	public static final String GRACEFUL_SHUTDOWN_PROP = "graceful.shutdown.seconds";
@@ -111,7 +111,7 @@ public class RunNiFi {
 				return;
 		}
 		
-		String configFilename = System.getProperty("org.apache.nifi.boostrap.config.file");
+		String configFilename = System.getProperty("org.apache.nifi.bootstrap.config.file");
 		
 		if ( configFilename == null ) {
 			final String nifiHome = System.getenv("NIFI_HOME");
@@ -233,11 +233,11 @@ public class RunNiFi {
 	}
 	
 	
-	private boolean isAlive(final Process process) {
+	public static boolean isAlive(final Process process) {
 		try {
 			process.exitValue();
 			return false;
-		} catch (final IllegalThreadStateException itse) {
+		} catch (final IllegalStateException | IllegalThreadStateException itse) {
 			return true;
 		}
 	}
@@ -253,7 +253,7 @@ public class RunNiFi {
 		final ProcessBuilder builder = new ProcessBuilder();
 
 		if ( !bootstrapConfigFile.exists() ) {
-			throw new FileNotFoundException(DEFAULT_CONFIG_FILE);
+			throw new FileNotFoundException(bootstrapConfigFile.getAbsolutePath());
 		}
 		
 		final Properties properties = new Properties();
@@ -351,6 +351,7 @@ public class RunNiFi {
 		cmd.addAll(javaAdditionalArgs);
 		cmd.add("-Dnifi.properties.file.path=" + nifiPropsFilename);
 		cmd.add("-Dnifi.bootstrap.listen.port=" + listenPort);
+		cmd.add("-Dapp=NiFi");
 		cmd.add("org.apache.nifi.NiFi");
 		
 		builder.command(cmd);
@@ -374,11 +375,11 @@ public class RunNiFi {
 			try {
 				gracefulShutdownSeconds = Integer.parseInt(gracefulShutdown);
 			} catch (final NumberFormatException nfe) {
-				throw new NumberFormatException("The '" + GRACEFUL_SHUTDOWN_PROP + "' property in Boostrap Config File " + bootstrapConfigAbsoluteFile.getAbsolutePath() + " has an invalid value. Must be a non-negative integer");
+				throw new NumberFormatException("The '" + GRACEFUL_SHUTDOWN_PROP + "' property in Bootstrap Config File " + bootstrapConfigAbsoluteFile.getAbsolutePath() + " has an invalid value. Must be a non-negative integer");
 			}
 			
 			if ( gracefulShutdownSeconds < 0 ) {
-				throw new NumberFormatException("The '" + GRACEFUL_SHUTDOWN_PROP + "' property in Boostrap Config File " + bootstrapConfigAbsoluteFile.getAbsolutePath() + " has an invalid value. Must be a non-negative integer");
+				throw new NumberFormatException("The '" + GRACEFUL_SHUTDOWN_PROP + "' property in Bootstrap Config File " + bootstrapConfigAbsoluteFile.getAbsolutePath() + " has an invalid value. Must be a non-negative integer");
 			}
 			
 			Process process = builder.start();

http://git-wip-us.apache.org/repos/asf/incubator-nifi/blob/192d7822/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java
----------------------------------------------------------------------
diff --git a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java
index 781b690..3c5ed1f 100644
--- a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java
+++ b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/ShutdownHook.java
@@ -55,11 +55,11 @@ public class ShutdownHook extends Thread {
 		
 		System.out.println("Waiting for Apache NiFi to finish shutting down...");
 		final long startWait = System.nanoTime();
-		while ( isAlive(nifiProcess) ) {
+		while ( RunNiFi.isAlive(nifiProcess) ) {
 			final long waitNanos = System.nanoTime() - startWait;
 			final long waitSeconds = TimeUnit.NANOSECONDS.toSeconds(waitNanos);
 			if ( waitSeconds >= gracefulShutdownSeconds && gracefulShutdownSeconds > 0 ) {
-				if ( isAlive(nifiProcess) ) {
+				if ( RunNiFi.isAlive(nifiProcess) ) {
 					System.out.println("NiFi has not finished shutting down after " + gracefulShutdownSeconds + " seconds. Killing process.");
 					nifiProcess.destroy();
 				}
@@ -76,13 +76,4 @@ public class ShutdownHook extends Thread {
 			System.err.println("Failed to delete status file " + statusFile.getAbsolutePath() + "; this file should be cleaned up manually");
 		}
 	}
-	
-	private boolean isAlive(final Process process) {
-		try {
-			process.exitValue();
-			return false;
-		} catch (final IllegalThreadStateException itse) {
-			return true;
-		}
-	}
 }