This is an automated email from the ASF dual-hosted git repository.

bbovenzi pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/airflow.git


The following commit(s) were added to refs/heads/main by this push:
     new 54d01809f9 Feature/add log grouping to UI (#38021)
54d01809f9 is described below

commit 54d01809f9389680b0ae358ee72bc162503dce3e
Author: Jens Scheffler <[email protected]>
AuthorDate: Thu Mar 14 01:47:33 2024 +0100

    Feature/add log grouping to UI (#38021)
    
    * Add folding feature for log group to web UI
    
    * Add folding feature for log groups, documentation
    
    * Group logs from pre-task execution and post-execution logs
    
    * Add state to the log block in grid view when task is running and logs are 
updated
    
    * Spelling
    
    * Review feedback
    
    * Review feedback v2
    
    * Enhance unicode arrow symbol compatability
---
 airflow/example_dags/example_python_operator.py    |  4 ++
 airflow/jobs/local_task_job_runner.py              |  6 +++
 airflow/models/taskinstance.py                     |  6 +++
 .../js/dag/details/taskInstance/Logs/LogBlock.tsx  | 50 +++++++++++++++++++++-
 .../js/dag/details/taskInstance/Logs/index.tsx     |  8 +++-
 .../dag/details/taskInstance/Logs/utils.test.tsx   | 17 ++++++--
 .../js/dag/details/taskInstance/Logs/utils.ts      | 39 ++++++++++++++---
 airflow/www/static/js/ti_log.js                    | 39 +++++++++++++++++
 .../logging-monitoring/logging-tasks.rst           | 41 ++++++++++++++++++
 9 files changed, 198 insertions(+), 12 deletions(-)

diff --git a/airflow/example_dags/example_python_operator.py 
b/airflow/example_dags/example_python_operator.py
index 0398dc0ae2..a5c2f145a3 100644
--- a/airflow/example_dags/example_python_operator.py
+++ b/airflow/example_dags/example_python_operator.py
@@ -51,8 +51,12 @@ with DAG(
     # [START howto_operator_python]
     def print_context(ds=None, **kwargs):
         """Print the Airflow context and ds variable from the context."""
+        print("::group::All kwargs")
         pprint(kwargs)
+        print("::endgroup::")
+        print("::group::Context variable ds")
         print(ds)
+        print("::endgroup::")
         return "Whatever you return gets printed in the logs"
 
     run_this = PythonOperator(task_id="print_the_context", 
python_callable=print_context)
diff --git a/airflow/jobs/local_task_job_runner.py 
b/airflow/jobs/local_task_job_runner.py
index e12ecefcc0..37b062b3a0 100644
--- a/airflow/jobs/local_task_job_runner.py
+++ b/airflow/jobs/local_task_job_runner.py
@@ -116,6 +116,9 @@ class LocalTaskJobRunner(BaseJobRunner, LoggingMixin):
 
         self.task_runner = get_task_runner(self)
 
+        # Print a marker post execution for internals of post task processing
+        self.log.info("::group::Pre task execution logs")
+
         def signal_handler(signum, frame):
             """Set kill signal handler."""
             self.log.error("Received SIGTERM. Terminating subprocesses")
@@ -215,6 +218,9 @@ class LocalTaskJobRunner(BaseJobRunner, LoggingMixin):
                     )
             return return_code
         finally:
+            # Print a marker for log grouping of details before task execution
+            self.log.info("::endgroup::")
+
             self.on_kill()
 
     def handle_task_exit(self, return_code: int) -> None:
diff --git a/airflow/models/taskinstance.py b/airflow/models/taskinstance.py
index 066c87b42f..433f454f1a 100644
--- a/airflow/models/taskinstance.py
+++ b/airflow/models/taskinstance.py
@@ -414,6 +414,9 @@ def _execute_task(task_instance: TaskInstance | 
TaskInstancePydantic, context: C
 
     def _execute_callable(context, **execute_callable_kwargs):
         try:
+            # Print a marker for log grouping of details before task execution
+            log.info("::endgroup::")
+
             return execute_callable(context=context, **execute_callable_kwargs)
         except SystemExit as e:
             # Handle only successful cases here. Failure cases will be handled 
upper
@@ -421,6 +424,9 @@ def _execute_task(task_instance: TaskInstance | 
TaskInstancePydantic, context: C
             if e.code is not None and e.code != 0:
                 raise
             return None
+        finally:
+            # Print a marker post execution for internals of post task 
processing
+            log.info("::group::Post task execution logs")
 
     # If a timeout is specified for the task, make it fail
     # if it goes beyond
diff --git a/airflow/www/static/js/dag/details/taskInstance/Logs/LogBlock.tsx 
b/airflow/www/static/js/dag/details/taskInstance/Logs/LogBlock.tsx
index dad3fda391..fe2e12aae2 100644
--- a/airflow/www/static/js/dag/details/taskInstance/Logs/LogBlock.tsx
+++ b/airflow/www/static/js/dag/details/taskInstance/Logs/LogBlock.tsx
@@ -25,9 +25,17 @@ interface Props {
   parsedLogs: string;
   wrap: boolean;
   tryNumber: number;
+  unfoldedGroups: Array<string>;
+  setUnfoldedLogGroup: React.Dispatch<React.SetStateAction<string[]>>;
 }
 
-const LogBlock = ({ parsedLogs, wrap, tryNumber }: Props) => {
+const LogBlock = ({
+  parsedLogs,
+  wrap,
+  tryNumber,
+  unfoldedGroups,
+  setUnfoldedLogGroup,
+}: Props) => {
   const [autoScroll, setAutoScroll] = useState(true);
 
   const logBoxRef = useRef<HTMLPreElement>(null);
@@ -59,10 +67,50 @@ const LogBlock = ({ parsedLogs, wrap, tryNumber }: Props) 
=> {
     }
   };
 
+  const onClick = (e: React.MouseEvent<HTMLElement>) => {
+    const target = e.target as HTMLElement;
+    const unfoldIdSuffix = "_unfold";
+    const foldIdSuffix = "_fold";
+    if (target.id?.endsWith(unfoldIdSuffix)) {
+      const gId = target.id.substring(
+        0,
+        target.id.length - unfoldIdSuffix.length
+      );
+      // remember the folding state if logs re-loaded
+      unfoldedGroups.push(gId);
+      setUnfoldedLogGroup(unfoldedGroups);
+      // now do the folding
+      target.style.display = "none";
+      if (target.nextElementSibling) {
+        (target.nextElementSibling as HTMLElement).style.display = "inline";
+      }
+    } else if (target.id?.endsWith(foldIdSuffix)) {
+      const gId = target.id.substring(
+        0,
+        target.id.length - foldIdSuffix.length
+      );
+      // remember the folding state if logs re-loaded
+      if (unfoldedGroups.indexOf(gId) >= 0) {
+        unfoldedGroups.splice(unfoldedGroups.indexOf(gId), 1);
+      }
+      setUnfoldedLogGroup(unfoldedGroups);
+      // now do the folding
+      if (target.parentElement) {
+        target.parentElement.style.display = "none";
+        if (target.parentElement.previousSibling) {
+          (target.parentElement.previousSibling as HTMLElement).style.display =
+            "inline";
+        }
+      }
+    }
+    return false;
+  };
+
   return (
     <Code
       ref={logBoxRef}
       onScroll={onScroll}
+      onClick={onClick}
       maxHeight={`calc(100% - ${offsetTop}px)`}
       overflowY="auto"
       p={3}
diff --git a/airflow/www/static/js/dag/details/taskInstance/Logs/index.tsx 
b/airflow/www/static/js/dag/details/taskInstance/Logs/index.tsx
index 12d64f4238..2b1035fb23 100644
--- a/airflow/www/static/js/dag/details/taskInstance/Logs/index.tsx
+++ b/airflow/www/static/js/dag/details/taskInstance/Logs/index.tsx
@@ -117,6 +117,7 @@ const Logs = ({
   const [fileSourceFilters, setFileSourceFilters] = useState<
     Array<FileSourceOption>
   >([]);
+  const [unfoldedLogGroups, setUnfoldedLogGroup] = useState<Array<string>>([]);
   const { timezone } = useTimezone();
 
   const taskTryNumber = selectedTryNumber || tryNumber || 1;
@@ -148,9 +149,10 @@ const Logs = ({
         data,
         timezone,
         logLevelFilters.map((option) => option.value),
-        fileSourceFilters.map((option) => option.value)
+        fileSourceFilters.map((option) => option.value),
+        unfoldedLogGroups
       ),
-    [data, fileSourceFilters, logLevelFilters, timezone]
+    [data, fileSourceFilters, logLevelFilters, timezone, unfoldedLogGroups]
   );
 
   const logAttemptDropdownLimit = 10;
@@ -315,6 +317,8 @@ const Logs = ({
                 parsedLogs={parsedLogs}
                 wrap={wrap}
                 tryNumber={taskTryNumber}
+                unfoldedGroups={unfoldedLogGroups}
+                setUnfoldedLogGroup={setUnfoldedLogGroup}
               />
             )
           )}
diff --git a/airflow/www/static/js/dag/details/taskInstance/Logs/utils.test.tsx 
b/airflow/www/static/js/dag/details/taskInstance/Logs/utils.test.tsx
index 23082e4282..aa835985d7 100644
--- a/airflow/www/static/js/dag/details/taskInstance/Logs/utils.test.tsx
+++ b/airflow/www/static/js/dag/details/taskInstance/Logs/utils.test.tsx
@@ -42,7 +42,13 @@ const mockTaskLog = `
 
 describe("Test Logs Utils.", () => {
   test("parseLogs function replaces datetimes", () => {
-    const { parsedLogs, fileSources } = parseLogs(mockTaskLog, "UTC", [], []);
+    const { parsedLogs, fileSources } = parseLogs(
+      mockTaskLog,
+      "UTC",
+      [],
+      [],
+      []
+    );
 
     expect(parsedLogs).toContain("2022-06-04, 00:00:01 UTC");
     expect(fileSources).toEqual([
@@ -51,7 +57,7 @@ describe("Test Logs Utils.", () => {
       "task_command.py",
       "taskinstance.py",
     ]);
-    const result = parseLogs(mockTaskLog, "America/Los_Angeles", [], []);
+    const result = parseLogs(mockTaskLog, "America/Los_Angeles", [], [], []);
     expect(result.parsedLogs).toContain("2022-06-03, 17:00:01 PDT");
   });
 
@@ -77,6 +83,7 @@ describe("Test Logs Utils.", () => {
         mockTaskLog,
         null,
         logLevelFilters,
+        [],
         []
       );
 
@@ -93,7 +100,8 @@ describe("Test Logs Utils.", () => {
       mockTaskLog,
       null,
       [],
-      ["taskinstance.py"]
+      ["taskinstance.py"],
+      []
     );
 
     expect(fileSources).toEqual([
@@ -112,7 +120,8 @@ describe("Test Logs Utils.", () => {
       mockTaskLog,
       null,
       [LogLevel.INFO, LogLevel.WARNING],
-      ["taskinstance.py"]
+      ["taskinstance.py"],
+      []
     );
 
     expect(fileSources).toEqual([
diff --git a/airflow/www/static/js/dag/details/taskInstance/Logs/utils.ts 
b/airflow/www/static/js/dag/details/taskInstance/Logs/utils.ts
index 44e8d4a46c..a7043cb32f 100644
--- a/airflow/www/static/js/dag/details/taskInstance/Logs/utils.ts
+++ b/airflow/www/static/js/dag/details/taskInstance/Logs/utils.ts
@@ -42,7 +42,8 @@ export const parseLogs = (
   data: string | undefined,
   timezone: string | null,
   logLevelFilters: Array<LogLevel>,
-  fileSourceFilters: Array<string>
+  fileSourceFilters: Array<string>,
+  unfoldedLogGroups: Array<string>
 ) => {
   if (!data) {
     return {};
@@ -62,6 +63,17 @@ export const parseLogs = (
   const fileSources: Set<string> = new Set();
   const ansiUp = new AnsiUp();
 
+  const urlRegex = /((https?:\/\/|http:\/\/)[^\s]+)/g;
+  // Detect log groups which can be collapsed
+  // Either in Github like format '::group::<group name>' to '::endgroup::'
+  // see 
https://docs.github.com/en/actions/using-workflows/workflow-commands-for-github-actions#grouping-log-lines
+  // Or in ADO pipeline like format '##[group]<group name>' to '##[endgroup]'
+  // see 
https://learn.microsoft.com/en-us/azure/devops/pipelines/scripts/logging-commands?view=azure-devops&tabs=powershell#formatting-commands
+  const logGroupStart = / INFO - (::|##\[])group(::|\])([^\n])*/g;
+  const logGroupEnd = / INFO - (::|##\[])endgroup(::|\])/g;
+  // Coloring (blue-60 as chakra style, is #0060df) and style such that log 
group appears like a link
+  const logGroupStyle = "color:#0060df;cursor:pointer;font-weight:bold;";
+
   lines.forEach((line) => {
     let parsedLine = line;
 
@@ -103,10 +115,27 @@ export const parseLogs = (
       const coloredLine = ansiUp.ansi_to_html(parsedLine);
 
       // for lines with links, transform to hyperlinks
-      const lineWithHyperlinks = coloredLine.replace(
-        /((https?:\/\/|http:\/\/)[^\s]+)/g,
-        '<a href="$1" target="_blank" style="color: blue; text-decoration: 
underline;">$1</a>'
-      );
+      const lineWithHyperlinks = coloredLine
+        .replace(
+          urlRegex,
+          '<a href="$1" target="_blank" style="color: blue; text-decoration: 
underline;">$1</a>'
+        )
+        .replace(logGroupStart, (textLine) => {
+          const unfoldIdSuffix = "_unfold";
+          const foldIdSuffix = "_fold";
+          const gName = textLine.substring(17);
+          const gId = gName.replace(/\W+/g, "_").toLowerCase();
+          const isFolded = unfoldedLogGroups.indexOf(gId) === -1;
+          const ufDisplay = isFolded ? "" : "display:none;";
+          const unfold = `<span id="${gId}${unfoldIdSuffix}" 
style="${ufDisplay}${logGroupStyle}"> &#9654; ${gName}</span>`;
+          const fDisplay = isFolded ? "display:none;" : "";
+          const fold = `<span style="${fDisplay}"><span 
id="${gId}${foldIdSuffix}" style="${logGroupStyle}"> &#9660; ${gName}</span>`;
+          return unfold + fold;
+        })
+        .replace(
+          logGroupEnd,
+          " <span style='color:#0060df;'>&#9650;&#9650;&#9650; Log group 
end</span></span>"
+        );
       parsedLines.push(lineWithHyperlinks);
     }
   });
diff --git a/airflow/www/static/js/ti_log.js b/airflow/www/static/js/ti_log.js
index cbafeccbc3..465af4be6d 100644
--- a/airflow/www/static/js/ti_log.js
+++ b/airflow/www/static/js/ti_log.js
@@ -31,6 +31,8 @@ const DELAY = parseInt(getMetaValue("delay"), 10);
 const AUTO_TAILING_OFFSET = parseInt(getMetaValue("auto_tailing_offset"), 10);
 const ANIMATION_SPEED = parseInt(getMetaValue("animation_speed"), 10);
 const TOTAL_ATTEMPTS = parseInt(getMetaValue("total_attempts"), 10);
+const unfoldIdSuffix = "_unfold";
+const foldIdSuffix = "_fold";
 
 function recurse(delay = DELAY) {
   return new Promise((resolve) => {
@@ -115,6 +117,14 @@ function autoTailingLog(tryNumber, metadata = null, 
autoTailing = false) {
       const dateRegex = /\d{4}[./-]\d{2}[./-]\d{2} \d{2}:\d{2}:\d{2},\d{3}/g;
       const iso8601Regex =
         /\d{4}[./-]\d{2}[./-]\d{2}T\d{2}:\d{2}:\d{2}.\d{3}[+-]\d{4}/g;
+      // Detect log groups which can be collapsed
+      // Either in Github like format '::group::<group name>' to '::endgroup::'
+      // see 
https://docs.github.com/en/actions/using-workflows/workflow-commands-for-github-actions#grouping-log-lines
+      // Or in ADO pipeline like format '##[group]<group name>' to 
'##[endgroup]'
+      // see 
https://learn.microsoft.com/en-us/azure/devops/pipelines/scripts/logging-commands?view=azure-devops&tabs=powershell#formatting-commands
+      const logGroupStart = / INFO - (::|##\[])group(::|\])([^\n])*/g;
+      const logGroupEnd = / INFO - (::|##\[])endgroup(::|\])/g;
+      const logGroupStyle = "color:#0060df;cursor:pointer;font-weight: bold;";
 
       res.message.forEach((item) => {
         const logBlockElementId = `try-${tryNumber}-${item[0]}`;
@@ -151,6 +161,17 @@ function autoTailingLog(tryNumber, metadata = null, 
autoTailing = false) {
               `<time datetime="${date}" data-with-tz="true">${formatDateTime(
                 `${date}`
               )}</time>`
+          )
+          .replaceAll(logGroupStart, (line) => {
+            const gName = line.substring(17);
+            const gId = gName.replaceAll(/\W+/g, "_").toLowerCase();
+            const unfold = `<span id="${gId}${unfoldIdSuffix}" 
style="${logGroupStyle}"> &#9654; ${gName}</span>`;
+            const fold = `<span style="display:none;"><span 
id="${gId}${foldIdSuffix}" style="${logGroupStyle}"> &#9660; ${gName}</span>`;
+            return unfold + fold;
+          })
+          .replaceAll(
+            logGroupEnd,
+            " <span style='color:#0060df;'>&#9650;&#9650;&#9650; Log group 
end</span></span>"
           );
         logBlock.innerHTML += `${linkifiedMessage}`;
       });
@@ -169,6 +190,17 @@ function autoTailingLog(tryNumber, metadata = null, 
autoTailing = false) {
   });
 }
 
+function handleLogGroupClick(e) {
+  if (e.target.id?.endsWith(unfoldIdSuffix)) {
+    e.target.style.display = "none";
+    e.target.nextSibling.style.display = "inline";
+  } else if (e.target.id?.endsWith(foldIdSuffix)) {
+    e.target.parentNode.style.display = "none";
+    e.target.parentNode.previousSibling.style.display = "inline";
+  }
+  return false;
+}
+
 function setDownloadUrl(tryNumber) {
   let tryNumberData = tryNumber;
   if (!tryNumberData) {
@@ -204,4 +236,11 @@ $(document).ready(() => {
 
     setDownloadUrl(tryNumber);
   });
+
+  console.debug(
+    `Attaching log grouping event handler for ${TOTAL_ATTEMPTS} attempts`
+  );
+  for (let i = 1; i <= TOTAL_ATTEMPTS; i += 1) {
+    document.getElementById(`log-group-${i}`).onclick = handleLogGroupClick;
+  }
 });
diff --git 
a/docs/apache-airflow/administration-and-deployment/logging-monitoring/logging-tasks.rst
 
b/docs/apache-airflow/administration-and-deployment/logging-monitoring/logging-tasks.rst
index 42b52ea9fc..3ea26deed6 100644
--- 
a/docs/apache-airflow/administration-and-deployment/logging-monitoring/logging-tasks.rst
+++ 
b/docs/apache-airflow/administration-and-deployment/logging-monitoring/logging-tasks.rst
@@ -82,6 +82,47 @@ This is the usual way loggers are used directly in Python 
code:
   logger = logging.getLogger(__name__)
   logger.info("This is a log message")
 
+Grouping of log lines
+---------------------
+
+.. versionadded:: 2.9.0
+
+Like CI pipelines also Airflow logs can be quite large and become hard to 
read. Sometimes therefore it is useful to group sections of log areas
+and provide folding of text areas to hide non relevant content. Airflow 
therefore implements a compatible log message grouping like
+`Github 
<https://docs.github.com/en/actions/using-workflows/workflow-commands-for-github-actions#grouping-log-lines>`_
 and
+`Azure DevOps 
<https://learn.microsoft.com/en-us/azure/devops/pipelines/scripts/logging-commands?view=azure-devops&tabs=powershell#formatting-commands>`_
+such that areas of text can be folded. The implemented scheme is compatible 
such that tools making output in CI can leverage the same experience
+in Airflow directly.
+
+By adding log markers with the starting and ending positions like for example 
below log messages can be grouped:
+
+.. code-block:: python
+
+   print("Here is some standard text.")
+   print("::group::Non important details")
+   print("bla")
+   print("debug messages...")
+   print("::endgroup::")
+   print("Here is again some standard text.")
+
+When displaying the logs in web UI, the display of logs will be condensed:
+
+.. code-block:: text
+
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} INFO - Here is some 
standard text.
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} ⯈ Non important details
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} INFO - Here is again some 
standard text.
+
+If you click on the log text label, the detailed log lies will be displayed.
+
+.. code-block:: text
+
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} INFO - Here is some 
standard text.
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} ⯆ Non important details
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} INFO - bla
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} INFO - debug messages...
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} ⯅⯅⯅ Log group end
+   [2024-03-08, 23:30:18 CET] {logging_mixin.py:188} INFO - Here is again some 
standard text.
 
 Interleaving of logs
 --------------------

Reply via email to