You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by tr...@apache.org on 2014/09/26 11:35:18 UTC

git commit: Cleanup LOG.debug to make sure no extra toString and unnecessary String concatenation

Repository: incubator-flink
Updated Branches:
  refs/heads/master e601a0d92 -> ec2a646a1


Cleanup LOG.debug to make sure no extra toString and unnecessary String concatenation

Remove unnecessary type check and cast for FileInputSPlit.

Update LOG.debug to user SLF4J parameterized logging or protected by LOG.isDebugEnabled to save toString and String concat.

Remove unnecessary semicolons.

This closes #127


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

Branch: refs/heads/master
Commit: ec2a646a1f62afeeea05ee414ca0437a96be6d19
Parents: e601a0d
Author: Henry Saputra <he...@gmail.com>
Authored: Wed Sep 24 10:40:19 2014 -0700
Committer: Till Rohrmann <tr...@apache.org>
Committed: Fri Sep 26 11:34:50 2014 +0200

----------------------------------------------------------------------
 .../src/main/java/org/apache/flink/yarn/Utils.java    |  7 +++++--
 .../java/org/apache/flink/client/program/Client.java  |  2 +-
 .../java/org/apache/flink/compiler/PactCompiler.java  |  2 +-
 .../api/common/io/DefaultInputSplitAssigner.java      |  6 ++----
 .../apache/flink/api/common/io/FileInputFormat.java   |  8 +-------
 .../flink/configuration/GlobalConfiguration.java      | 11 ++++++-----
 .../io/network/netty/NettyConnectionManager.java      |  2 +-
 .../java/org/apache/flink/runtime/ipc/Server.java     | 14 +++++++-------
 .../apache/flink/runtime/jobmanager/JobManager.java   |  6 ++++--
 .../runtime/jobmanager/web/JobmanagerInfoServlet.java |  2 +-
 .../resettable/AbstractBlockResettableIterator.java   |  2 +-
 .../profiling/impl/JobManagerProfilerImpl.java        |  4 +++-
 .../profiling/impl/TaskManagerProfilerImpl.java       |  4 ++--
 .../apache/flink/runtime/taskmanager/TaskManager.java | 14 ++++++++------
 14 files changed, 43 insertions(+), 41 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-addons/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java
----------------------------------------------------------------------
diff --git a/flink-addons/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java b/flink-addons/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java
index d75e12a..3e30096 100644
--- a/flink-addons/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java
+++ b/flink-addons/flink-yarn/src/main/java/org/apache/flink/yarn/Utils.java
@@ -230,8 +230,11 @@ public class Utils {
 		}
 		DataOutputBuffer dob = new DataOutputBuffer();
 		credentials.writeTokenStorageToStream(dob);
-		LOG.debug("Wrote tokens. Credentials buffer length: "+dob.getLength());
-		
+
+		if(LOG.isDebugEnabled()) {
+			LOG.debug("Wrote tokens. Credentials buffer length: " + dob.getLength());
+		}
+
 		ByteBuffer securityTokens = ByteBuffer.wrap(dob.getData(), 0, dob.getLength());
 		amContainer.setTokens(securityTokens);
 	}

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-clients/src/main/java/org/apache/flink/client/program/Client.java
----------------------------------------------------------------------
diff --git a/flink-clients/src/main/java/org/apache/flink/client/program/Client.java b/flink-clients/src/main/java/org/apache/flink/client/program/Client.java
index 1d6adde..dcb54ac 100644
--- a/flink-clients/src/main/java/org/apache/flink/client/program/Client.java
+++ b/flink-clients/src/main/java/org/apache/flink/client/program/Client.java
@@ -253,7 +253,7 @@ public class Client {
 						catch (Throwable t) {
 							LOG.error("The program execution failed.", t);
 						}
-					};
+					}
 				};
 				backGroundRunner.start();
 			}

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-compiler/src/main/java/org/apache/flink/compiler/PactCompiler.java
----------------------------------------------------------------------
diff --git a/flink-compiler/src/main/java/org/apache/flink/compiler/PactCompiler.java b/flink-compiler/src/main/java/org/apache/flink/compiler/PactCompiler.java
index 174f8f3..7445759 100644
--- a/flink-compiler/src/main/java/org/apache/flink/compiler/PactCompiler.java
+++ b/flink-compiler/src/main/java/org/apache/flink/compiler/PactCompiler.java
@@ -1042,7 +1042,7 @@ public class PactCompiler {
 
 			node.computeUnclosedBranchStack();
 		}
-	};
+	}
 	
 	/**
 	 * Utility class that traverses a plan to collect all nodes and add them to the OptimizedPlan.

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-core/src/main/java/org/apache/flink/api/common/io/DefaultInputSplitAssigner.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/common/io/DefaultInputSplitAssigner.java b/flink-core/src/main/java/org/apache/flink/api/common/io/DefaultInputSplitAssigner.java
index 54d1667..379fc4e 100644
--- a/flink-core/src/main/java/org/apache/flink/api/common/io/DefaultInputSplitAssigner.java
+++ b/flink-core/src/main/java/org/apache/flink/api/common/io/DefaultInputSplitAssigner.java
@@ -63,11 +63,9 @@ public class DefaultInputSplitAssigner implements InputSplitAssigner {
 		
 		if (LOG.isDebugEnabled()) {
 			if (next == null) {
-				LOG.debug("Assigning split " + next + " to " + host);
+				LOG.debug("No more input splits available");
 			} else {
-				if (LOG.isDebugEnabled()) {
-					LOG.debug("No more input splits available");
-				}
+				LOG.debug("Assigning split " + next + " to " + host);
 			}
 		}
 		return next;

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-core/src/main/java/org/apache/flink/api/common/io/FileInputFormat.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/api/common/io/FileInputFormat.java b/flink-core/src/main/java/org/apache/flink/api/common/io/FileInputFormat.java
index 59888c7..25f1299 100644
--- a/flink-core/src/main/java/org/apache/flink/api/common/io/FileInputFormat.java
+++ b/flink-core/src/main/java/org/apache/flink/api/common/io/FileInputFormat.java
@@ -565,13 +565,7 @@ public abstract class FileInputFormat<OT> implements InputFormat<OT, FileInputSp
 	 * working on the input format do not reach the file system.
 	 */
 	@Override
-	public void open(FileInputSplit split) throws IOException {
-		
-		if (!(split instanceof FileInputSplit)) {
-			throw new IllegalArgumentException("File Input Formats can only be used with FileInputSplits.");
-		}
-		
-		final FileInputSplit fileSplit = (FileInputSplit) split;
+	public void open(FileInputSplit fileSplit) throws IOException {
 		
 		this.splitStart = fileSplit.getStart();
 		this.splitLength = fileSplit.getLength();

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-core/src/main/java/org/apache/flink/configuration/GlobalConfiguration.java
----------------------------------------------------------------------
diff --git a/flink-core/src/main/java/org/apache/flink/configuration/GlobalConfiguration.java b/flink-core/src/main/java/org/apache/flink/configuration/GlobalConfiguration.java
index 9d7aa31..23846ca 100644
--- a/flink-core/src/main/java/org/apache/flink/configuration/GlobalConfiguration.java
+++ b/flink-core/src/main/java/org/apache/flink/configuration/GlobalConfiguration.java
@@ -411,7 +411,7 @@ public final class GlobalConfiguration {
 						continue;
 					}
 
-					LOG.debug("Loading configuration property: " + key + ", " + value);
+					LOG.debug("Loading configuration property: {}, {}", key, value);
 
 					this.confData.put(key, value);
 				}
@@ -420,9 +420,11 @@ public final class GlobalConfiguration {
 			e.printStackTrace();
 		} finally {
 			try {
-				reader.close();
+				if(reader != null) {
+					reader.close();
+				}
 			} catch (IOException e) {
-				e.printStackTrace();
+				LOG.warn("Cannot to close reader with IOException.", e);
 			}
 		}
 	}
@@ -525,11 +527,10 @@ public final class GlobalConfiguration {
 
 					if (key != null && value != null) {
 						// Put key, value pair into the map
-						LOG.debug("Loading configuration property: " + key + ", " + value);
+						LOG.debug("Loading configuration property: {}, {}", key, value);
 						this.confData.put(key, value);
 					} else {
 						LOG.warn("Error while reading configuration: Cannot read property " + propNumber);
-						continue;
 					}
 				}
 			}

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/NettyConnectionManager.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/NettyConnectionManager.java b/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/NettyConnectionManager.java
index cb849f0..3b148e0 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/NettyConnectionManager.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/io/network/netty/NettyConnectionManager.java
@@ -339,7 +339,7 @@ public class NettyConnectionManager implements NetworkConnectionManager {
 				handInChannel(new OutboundConnectionQueue(future.channel()));
 			}
 			else if (this.numRetries > 0) {
-				LOG.debug(String.format("Connection request did not succeed, retrying (%d attempts left)", this.numRetries));
+				LOG.debug("Connection request did not succeed, retrying ({} attempts left)", numRetries);
 
 				this.out.connect(this.receiver.getConnectionAddress()).addListener(this);
 				this.numRetries--;

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-runtime/src/main/java/org/apache/flink/runtime/ipc/Server.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/ipc/Server.java b/flink-runtime/src/main/java/org/apache/flink/runtime/ipc/Server.java
index e66dc33..31203cb 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/ipc/Server.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/ipc/Server.java
@@ -331,7 +331,7 @@ public abstract class Server {
 
 		@Override
 		public void run() {
-			LOG.debug(getName() + ": starting");
+			LOG.debug("{} : starting", getName());
 			SERVER.set(Server.this);
 			while (running) {
 				SelectionKey key = null;
@@ -373,7 +373,7 @@ public abstract class Server {
 				}
 				cleanupConnections(false);
 			}
-			LOG.debug("Stopping " + this.getName());
+			LOG.debug("Stopping {}", this.getName());
 
 			synchronized (this) {
 				try {
@@ -494,7 +494,7 @@ public abstract class Server {
 
 		@Override
 		public void run() {
-			LOG.debug(getName() + ": starting");
+			LOG.debug("{} : starting", getName());
 			SERVER.set(Server.this);
 			long lastPurgeTime = 0; // last check for old calls.
 
@@ -560,7 +560,7 @@ public abstract class Server {
 					LOG.warn("Exception in Responder " + e.toString());
 				}
 			}
-			LOG.debug("Stopping " + this.getName());
+			LOG.debug("Stopping {}", getName());
 
 			this.shutDown = true;
 		}
@@ -931,7 +931,7 @@ public abstract class Server {
 
 		@Override
 		public void run() {
-			LOG.debug(getName() + ": starting");
+			LOG.debug("{} : starting", getName());
 			SERVER.set(Server.this);
 			ByteArrayOutputStream buf = new ByteArrayOutputStream(10240);
 			while (running) {
@@ -957,7 +957,7 @@ public abstract class Server {
 					LOG.error(getName() + " caught: ", e);
 				}
 			}
-			LOG.debug(getName() + ": exiting");
+			LOG.debug("{} : exiting", getName());
 
 			this.shutDown = true;
 		}
@@ -1072,7 +1072,7 @@ public abstract class Server {
 
 	/** Stops the service. No new calls will be handled after this is called. */
 	public synchronized void stop() {
-		LOG.debug("Stopping server on " + port);
+		LOG.debug("Stopping server on {}", port);
 		running = false;
 		if (handlers != null) {
 			for (int i = 0; i < handlerCount; i++) {

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/JobManager.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/JobManager.java b/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/JobManager.java
index 4133302..854708c 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/JobManager.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/JobManager.java
@@ -444,8 +444,10 @@ public class JobManager implements ExtendedManagementProtocol, InputSplitProvide
 
 		final ExecutionGraph eg = this.currentJobs.get(executionState.getJobID());
 		if (eg == null) {
-			LOG.debug("Orphaned execution task: UpdateTaskExecutionState call cannot find execution graph for ID " + executionState.getJobID() +
-					" to change state to " + executionState.getExecutionState());
+			if (LOG.isDebugEnabled()) {
+				LOG.debug("Orphaned execution task: UpdateTaskExecutionState call cannot find execution graph for ID " + executionState.getJobID() +
+						" to change state to " + executionState.getExecutionState());
+			}
 			return false;
 		}
 

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/web/JobmanagerInfoServlet.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/web/JobmanagerInfoServlet.java b/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/web/JobmanagerInfoServlet.java
index 7e585be..e63adbd 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/web/JobmanagerInfoServlet.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/jobmanager/web/JobmanagerInfoServlet.java
@@ -493,7 +493,7 @@ public class JobmanagerInfoServlet extends HttpServlet {
 		}
 		catch (IOException ioe) { // Connection closed by client
 			String message = "Info server for jobmanager: Connection closed by client - " + ioe.getClass().getSimpleName();
-			
+
 			if (LOG.isDebugEnabled()) {
 				LOG.debug(message, ioe);
 			}

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-runtime/src/main/java/org/apache/flink/runtime/operators/resettable/AbstractBlockResettableIterator.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/operators/resettable/AbstractBlockResettableIterator.java b/flink-runtime/src/main/java/org/apache/flink/runtime/operators/resettable/AbstractBlockResettableIterator.java
index 27157bf..421e0c7 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/operators/resettable/AbstractBlockResettableIterator.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/operators/resettable/AbstractBlockResettableIterator.java
@@ -85,7 +85,7 @@ abstract class AbstractBlockResettableIterator<T> implements MemoryBlockIterator
 		this.readView = new RandomAccessInputView(this.fullSegments, memoryManager.getPageSize());
 		
 		if (LOG.isDebugEnabled()) {
-			LOG.debug("Iterator initalized using " + numPages + " memory buffers.");
+			LOG.debug("Iterator initialized using " + numPages + " memory buffers.");
 		}
 	}
 	

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/JobManagerProfilerImpl.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/JobManagerProfilerImpl.java b/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/JobManagerProfilerImpl.java
index bccaa6f..23b0ab9 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/JobManagerProfilerImpl.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/JobManagerProfilerImpl.java
@@ -118,7 +118,9 @@ public class JobManagerProfilerImpl implements JobManagerProfiler, ProfilerImplP
 
 		final long profilingStart = getProfilingStart(profilingData.getJobID());
 		if (profilingStart < 0 && LOG.isDebugEnabled()) {
-			LOG.debug("Received profiling data for unregistered job " + profilingData.getJobID());
+			if (LOG.isDebugEnabled()) {
+				LOG.debug("Received profiling data for unregistered job {}", profilingData.getJobID());
+			}
 			return;
 		}
 

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/TaskManagerProfilerImpl.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/TaskManagerProfilerImpl.java b/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/TaskManagerProfilerImpl.java
index 046d22d..e194792 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/TaskManagerProfilerImpl.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/profiling/impl/TaskManagerProfilerImpl.java
@@ -176,7 +176,7 @@ public class TaskManagerProfilerImpl extends TimerTask implements TaskManagerPro
 	public void registerMainThreadForCPUProfiling(Environment environment, Thread thread, JobVertexID vertexId, int subtask, ExecutionAttemptID executionID) {
 
 		synchronized (this.monitoredThreads) {
-			LOG.debug("Registering thread " + thread.getName() + " for CPU monitoring");
+			LOG.debug("Registering thread {} for CPU monitoring", thread);
 			if (this.monitoredThreads.containsKey(environment)) {
 				LOG.error("There is already a main thread registered for environment object "
 					+ environment.getTaskName());
@@ -204,7 +204,7 @@ public class TaskManagerProfilerImpl extends TimerTask implements TaskManagerPro
 	public void unregisterMainThreadFromCPUProfiling(Environment environment, Thread thread) {
 
 		synchronized (this.monitoredThreads) {
-			LOG.debug("Unregistering thread " + thread.getName() + " from CPU monitoring");
+			LOG.debug("Unregistering thread {} from CPU monitoring", thread);
 			final EnvironmentThreadSet environmentThreadSet = this.monitoredThreads.remove(environment);
 			if (environmentThreadSet != null) {
 

http://git-wip-us.apache.org/repos/asf/incubator-flink/blob/ec2a646a/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/TaskManager.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/TaskManager.java b/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/TaskManager.java
index cbcc9f5..ae39d04 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/TaskManager.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/taskmanager/TaskManager.java
@@ -376,9 +376,10 @@ public class TaskManager implements TaskOperationProtocol {
 						while (!isShutDown()) {
 							Thread.sleep(logIntervalMs);
 
-							LOG.debug(getMemoryUsageStatsAsString(memoryMXBean));
-
-							LOG.debug(getGarbageCollectorStatsAsString(gcMXBeans));
+							if (LOG.isDebugEnabled()) {
+								LOG.debug(getMemoryUsageStatsAsString(memoryMXBean));
+								LOG.debug(getGarbageCollectorStatsAsString(gcMXBeans));
+							}
 						}
 					} catch (InterruptedException e) {
 						LOG.warn("Unexpected interruption of memory usage logger thread.");
@@ -1186,7 +1187,7 @@ public class TaskManager implements TaskOperationProtocol {
 				throw new RuntimeException("The TaskManager is unable to connect to the JobManager (Address: '"+jobManagerAddress+"').");
 			}
 			if (LOG.isDebugEnabled()) {
-				LOG.debug("Defaulting to detection strategy " + strategy);
+				LOG.debug("Defaulting to detection strategy {}", strategy);
 			}
 		}
 	}
@@ -1197,7 +1198,7 @@ public class TaskManager implements TaskOperationProtocol {
 	 * @return An available port.
 	 * @throws RuntimeException Thrown, if no free port was found.
 	 */
-	private static final int getAvailablePort() {
+	private static int getAvailablePort() {
 		for (int i = 0; i < 50; i++) {
 			ServerSocket serverSocket = null;
 			try {
@@ -1207,7 +1208,8 @@ public class TaskManager implements TaskOperationProtocol {
 					return port;
 				}
 			} catch (IOException e) {
-				LOG.debug("Unable to allocate port " + e.getMessage(), e);
+
+				LOG.debug("Unable to allocate port with exception {}", e);
 			} finally {
 				if (serverSocket != null) {
 					try { serverSocket.close(); } catch (Throwable t) {}