You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by gr...@apache.org on 2017/04/24 15:02:24 UTC

flink git commit: [FLINK-6280] [scripts] Allow logging with Java flags

Repository: flink
Updated Branches:
  refs/heads/master aa21f853a -> 166444767


[FLINK-6280] [scripts] Allow logging with Java flags

Evaluate user-defined Java options immediately before starting services
and rotate all log files.

This closes #3701


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

Branch: refs/heads/master
Commit: 1664447677404e66ef842350d30469c82286cb41
Parents: aa21f85
Author: Greg Hogan <co...@greghogan.com>
Authored: Fri Apr 7 13:34:16 2017 -0400
Committer: Greg Hogan <co...@greghogan.com>
Committed: Mon Apr 24 10:12:50 2017 -0400

----------------------------------------------------------------------
 docs/monitoring/application_profiling.md        | 54 ++++++++++++++++++++
 docs/setup/config.md                            |  8 ++-
 flink-dist/src/main/flink-bin/bin/config.sh     | 11 +++-
 .../src/main/flink-bin/bin/flink-daemon.sh      | 11 ++--
 4 files changed, 78 insertions(+), 6 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/16644476/docs/monitoring/application_profiling.md
----------------------------------------------------------------------
diff --git a/docs/monitoring/application_profiling.md b/docs/monitoring/application_profiling.md
new file mode 100644
index 0000000..65ef45e
--- /dev/null
+++ b/docs/monitoring/application_profiling.md
@@ -0,0 +1,54 @@
+---
+title: "Application Profiling"
+nav-parent_id: monitoring
+nav-pos: 15
+---
+<!--
+Licensed to the Apache Software Foundation (ASF) under one
+or more contributor license agreements.  See the NOTICE file
+distributed with this work for additional information
+regarding copyright ownership.  The ASF licenses this file
+to you under the Apache License, Version 2.0 (the
+"License"); you may not use this file except in compliance
+with the License.  You may obtain a copy of the License at
+
+  http://www.apache.org/licenses/LICENSE-2.0
+
+Unless required by applicable law or agreed to in writing,
+software distributed under the License is distributed on an
+"AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+KIND, either express or implied.  See the License for the
+specific language governing permissions and limitations
+under the License.
+-->
+
+* ToC
+{:toc}
+
+## Overview of Custom Logging with Apache Flink
+
+Each standalone JobManager, TaskManager, HistoryServer, and ZooKeeper daemon redirects `stdout` and `stderr` to a file
+with a `.out` filename suffix and writes internal logging to a file with a `.log` suffix. Java options configured by the
+user in `env.java.opts`, `env.java.opts.jobmanager`, and `env.java.opts.taskmanager` can likewise define log files with
+use of the script variable `FLINK_LOG_PREFIX` and by enclosing the options in double quotes for late evaluation. Log files
+using `FLINK_LOG_PREFIX` are rotated along with the default `.out` and `.log` files.
+
+# Profiling with Java Flight Recorder
+
+Java Flight Recorder is a profiling and event collection framework built into the Oracle JDK.
+[Java Mission Control](http://www.oracle.com/technetwork/java/javaseproducts/mission-control/java-mission-control-1998576.html)
+is an advanced set of tools that enables efficient and detailed analysis of the extensive of data collected by Java
+Flight Recorder. Example configuration:
+
+~~~
+env.java.opts: "-XX:+UnlockCommercialFeatures -XX:+UnlockDiagnosticVMOptions -XX:+FlightRecorder -XX:+DebugNonSafepoints -XX:FlightRecorderOptions=defaultrecording=true,dumponexit=true,dumponexitpath=${FLINK_LOG_PREFIX}.jfr"
+~~~
+
+# Profiling with JITWatch
+
+[JITWatch](https://github.com/AdoptOpenJDK/jitwatch/wiki) is a log analyser and visualizer for the Java HotSpot JIT
+compiler used to inspect inlining decisions, hot methods, bytecode, and assembly. Example configuration:
+
+~~~
+env.java.opts: "-XX:+UnlockDiagnosticVMOptions -XX:+TraceClassLoading -XX:+LogCompilation -XX:LogFile=${FLINK_LOG_PREFIX}.jit -XX:+PrintAssembly"
+~~~

http://git-wip-us.apache.org/repos/asf/flink/blob/16644476/docs/setup/config.md
----------------------------------------------------------------------
diff --git a/docs/setup/config.md b/docs/setup/config.md
index 6a5c1ff..e36f149 100644
--- a/docs/setup/config.md
+++ b/docs/setup/config.md
@@ -42,7 +42,11 @@ The configuration files for the TaskManagers can be different, Flink does not as
 
 - `env.java.home`: The path to the Java installation to use (DEFAULT: system's default Java installation, if found). Needs to be specified if the startup scripts fail to automatically resolve the java home directory. Can be specified to point to a specific java installation or version. If this option is not specified, the startup scripts also evaluate the `$JAVA_HOME` environment variable.
 
-- `env.java.opts`: Set custom JVM options. This value is respected by Flink's start scripts, both JobManager and TaskManager, and Flink's YARN client. This can be used to set different garbage collectors or to include remote debuggers into the JVMs running Flink's services. Use `env.java.opts.jobmanager` and `env.java.opts.taskmanager` for JobManager or TaskManager-specific options, respectively.
+- `env.java.opts`: Set custom JVM options. This value is respected by Flink's start scripts, both JobManager and
+TaskManager, and Flink's YARN client. This can be used to set different garbage collectors or to include remote
+debuggers into the JVMs running Flink's services. Enclosing options in double quotes delays parameter substitution
+allowing access to variables from Flink's startup scripts. Use `env.java.opts.jobmanager` and `env.java.opts.taskmanager`
+for JobManager or TaskManager-specific options, respectively.
 
 - `env.java.opts.jobmanager`: JobManager-specific JVM options. These are used in addition to the regular `env.java.opts`.
 
@@ -536,6 +540,8 @@ Previously this key was named `recovery.mode` and the default value was `standal
 
 - `env.log.dir`: (Defaults to the `log` directory under Flink's home) Defines the directory where the Flink logs are saved. It has to be an absolute path.
 
+- `env.log.max`: (Default: `5`) The maximum number of old log files to keep.
+
 - `env.ssh.opts`: Additional command line options passed to SSH clients when starting or stopping JobManager, TaskManager, and Zookeeper services (start-cluster.sh, stop-cluster.sh, start-zookeeper-quorum.sh, stop-zookeeper-quorum.sh).
 
 ### Queryable State

http://git-wip-us.apache.org/repos/asf/flink/blob/16644476/flink-dist/src/main/flink-bin/bin/config.sh
----------------------------------------------------------------------
diff --git a/flink-dist/src/main/flink-bin/bin/config.sh b/flink-dist/src/main/flink-bin/bin/config.sh
index a16529c..0894481 100755
--- a/flink-dist/src/main/flink-bin/bin/config.sh
+++ b/flink-dist/src/main/flink-bin/bin/config.sh
@@ -356,7 +356,16 @@ extractHostName() {
     echo $SLAVE
 }
 
-# Auxilliary function for log file rotation
+# Auxilliary functions for log file rotation
+rotateLogFilesWithPrefix() {
+    dir=$1
+    prefix=$2
+    while read -r log ; do
+        rotateLogFile $log
+    # find distinct set of log file names, ignoring the rotation number (trailing dot and digit)
+    done < <(find "$dir" ! -type d -path "${prefix}*" | sed -E s/\.[0-9]+$// | sort | uniq)
+}
+
 rotateLogFile() {
     log=$1;
     num=$MAX_LOG_FILE_NUMBER

http://git-wip-us.apache.org/repos/asf/flink/blob/16644476/flink-dist/src/main/flink-bin/bin/flink-daemon.sh
----------------------------------------------------------------------
diff --git a/flink-dist/src/main/flink-bin/bin/flink-daemon.sh b/flink-dist/src/main/flink-bin/bin/flink-daemon.sh
index b25358d..8bb6f0b 100644
--- a/flink-dist/src/main/flink-bin/bin/flink-daemon.sh
+++ b/flink-dist/src/main/flink-bin/bin/flink-daemon.sh
@@ -78,8 +78,9 @@ fi
 # This allows us to start multiple daemon of each type.
 id=$([ -f "$pid" ] && echo $(wc -l < $pid) || echo "0")
 
-log="${FLINK_LOG_DIR}/flink-${FLINK_IDENT_STRING}-${DAEMON}-${id}-${HOSTNAME}.log"
-out="${FLINK_LOG_DIR}/flink-${FLINK_IDENT_STRING}-${DAEMON}-${id}-${HOSTNAME}.out"
+FLINK_LOG_PREFIX="${FLINK_LOG_DIR}/flink-${FLINK_IDENT_STRING}-${DAEMON}-${id}-${HOSTNAME}"
+log="${FLINK_LOG_PREFIX}.log"
+out="${FLINK_LOG_PREFIX}.out"
 
 log_setting=("-Dlog.file=${log}" "-Dlog4j.configuration=file:${FLINK_CONF_DIR}/log4j.properties" "-Dlogback.configurationFile=file:${FLINK_CONF_DIR}/logback.xml")
 
@@ -96,8 +97,7 @@ case $STARTSTOP in
 
     (start)
         # Rotate log files
-        rotateLogFile $log
-        rotateLogFile $out
+        rotateLogFilesWithPrefix $FLINK_LOG_DIR $FLINK_LOG_PREFIX
 
         # Print a warning if daemons are already running on host
         if [ -f $pid ]; then
@@ -116,6 +116,9 @@ case $STARTSTOP in
           fi
         fi
 
+        # Evaluate user options for local variable expansion
+        FLINK_ENV_JAVA_OPTS=$(eval echo ${FLINK_ENV_JAVA_OPTS})
+
         echo "Starting $DAEMON daemon on host $HOSTNAME."
         $JAVA_RUN $JVM_ARGS ${FLINK_ENV_JAVA_OPTS} "${log_setting[@]}" -classpath "`manglePathList "$FLINK_TM_CLASSPATH:$INTERNAL_HADOOP_CLASSPATHS"`" ${CLASS_TO_RUN} "${ARGS[@]}" > "$out" 200<&- 2>&1 < /dev/null &