summaryzb commented on code in PR #2412:
URL: https://github.com/apache/uniffle/pull/2412#discussion_r2000678142


##########
client-spark/common/src/main/java/org/apache/spark/shuffle/writer/WriteBufferManager.java:
##########
@@ -551,38 +550,18 @@ public List<AddBlockEvent> 
buildBlockEvents(List<ShuffleBlockInfo> shuffleBlockI
   @Override
   public long spill(long size, MemoryConsumer trigger) {
     // Only for the MemoryConsumer of this instance, it will flush buffer
-    if (!memorySpillEnabled || trigger != this) {
-      return 0L;
-    }
-
-    List<CompletableFuture<Long>> futures = 
spillFunc.apply(clear(bufferSpillRatio));
-    CompletableFuture<Void> allOfFutures =
-        CompletableFuture.allOf(futures.toArray(new 
CompletableFuture[futures.size()]));
-    try {
-      allOfFutures.get(memorySpillTimeoutSec, TimeUnit.SECONDS);
-    } catch (TimeoutException timeoutException) {
-      // A best effort strategy to wait.
-      // If timeout exception occurs, the underlying tasks won't be cancelled.
-      LOG.warn("[taskId: {}] Spill tasks timeout after {} seconds", taskId, 
memorySpillTimeoutSec);
-    } catch (Exception e) {
-      LOG.warn("[taskId: {}] Failed to spill buffers due to ", taskId, e);
-    } finally {
-      long releasedSize =
-          futures.stream()
-              .filter(x -> x.isDone())
-              .mapToLong(
-                  x -> {
-                    try {
-                      return x.get();
-                    } catch (Exception e) {
-                      return 0;
-                    }
-                  })
-              .sum();
-      LOG.info(
-          "[taskId: {}] Spill triggered by own, released memory size: {}", 
taskId, releasedSize);
-      return releasedSize;
+    if (memorySpillEnabled && trigger == this) {
+      List<CompletableFuture<Long>> futures = 
spillFunc.apply(clear(bufferSpillRatio));
+      CompletableFuture<Void> allOfFutures =
+          CompletableFuture.allOf(futures.toArray(new 
CompletableFuture[futures.size()]));
+      try {
+        // A best effort strategy to wait.
+        allOfFutures.get(memorySpillTimeoutSec, TimeUnit.SECONDS);
+      } catch (Exception e) {
+        // ignore
+      }
     }
+    return 0L;

Review Comment:
   I see, but does it really necessary to block here to calculate memory 
released. since the `free memory` is called in multiply callback thread



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to