You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@uniffle.apache.org by "smallzhongfeng (via GitHub)" <gi...@apache.org> on 2023/03/30 07:44:26 UTC

[GitHub] [incubator-uniffle] smallzhongfeng commented on a diff in pull request #775: [#757] Separate flush thread pools for different storage type.

smallzhongfeng commented on code in PR #775:
URL: https://github.com/apache/incubator-uniffle/pull/775#discussion_r1152858842


##########
server/src/test/java/org/apache/uniffle/server/buffer/ShuffleBufferManagerTest.java:
##########
@@ -587,6 +587,7 @@ public void shuffleFlushThreshold() throws Exception {
     shuffleBufferManager.cacheShuffleData(appId, smallShuffleId, false, createData(0, 31));
     assertEquals(96 + 63, shuffleBufferManager.getUsedMemory());
     shuffleFlushManager.flush();
+    Thread.sleep(100);

Review Comment:
   Why do we need to sleep here?
   
   



##########
server/src/main/java/org/apache/uniffle/server/StorageTypeFlushEventHandler.java:
##########
@@ -0,0 +1,96 @@
+/*
+ * 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.
+ */
+
+package org.apache.uniffle.server;
+
+import java.util.concurrent.BlockingQueue;
+import java.util.concurrent.Executor;
+import java.util.concurrent.ThreadPoolExecutor;
+import java.util.concurrent.TimeUnit;
+
+import com.google.common.collect.Queues;
+
+import org.apache.uniffle.common.config.RssBaseConf;
+import org.apache.uniffle.common.exception.RssException;
+import org.apache.uniffle.common.util.ThreadUtils;
+import org.apache.uniffle.server.storage.StorageManager;
+import org.apache.uniffle.storage.common.HdfsStorage;
+import org.apache.uniffle.storage.common.LocalStorage;
+import org.apache.uniffle.storage.common.Storage;
+import org.apache.uniffle.storage.util.StorageType;
+
+public class StorageTypeFlushEventHandler implements FlushEventHandler {
+  private final ShuffleServerConf shuffleServerConf;
+  private final ShuffleFlushManager shuffleFlushManager;
+  private final StorageManager storageManager;
+  private Executor localFileThreadPoolExecutor;
+  private Executor hdfsThreadPoolExecutor;
+  private final StorageType storageType;
+
+  public StorageTypeFlushEventHandler(ShuffleServerConf conf, ShuffleFlushManager shuffleFlushManager,
+      StorageManager storageManager) {
+    this.shuffleServerConf = conf;
+    this.storageType = StorageType.valueOf(shuffleServerConf.get(RssBaseConf.RSS_STORAGE_TYPE));
+    this.shuffleFlushManager = shuffleFlushManager;
+    this.storageManager = storageManager;
+    initFlushEventExecutor();
+  }
+
+  @Override
+  public void handle(ShuffleDataFlushEvent event) {
+    Storage storage = storageManager.selectStorage(event);
+    if (storage instanceof HdfsStorage) {
+      hdfsThreadPoolExecutor.execute(() -> handleEventAndUpdateMetrics(event, false));
+    } else if (storage instanceof LocalStorage) {
+      localFileThreadPoolExecutor.execute(() -> handleEventAndUpdateMetrics(event, true));
+    } else {
+      throw new RssException("Unexpected storage type!");
+    }
+  }
+
+  private void handleEventAndUpdateMetrics(ShuffleDataFlushEvent event, boolean isLocalFile) {
+    try {
+      shuffleFlushManager.processEvent(event);
+    } finally {

Review Comment:
   Should we catch some exceptions and add logs here?



##########
server/src/test/java/org/apache/uniffle/server/ShuffleFlushManagerTest.java:
##########
@@ -580,4 +580,58 @@ public void processPendingEventsTest(@TempDir File tempDir) throws Exception {
     assertEquals(eventNum + 3, (int) ShuffleServerMetrics.counterTotalDroppedEventNum.get());
     assertEquals(0, manager.getPendingEventsSize());
   }
+
+  @Test
+  public void storageTypeFlushEventHandlerTest(@TempDir File tempDir) throws Exception {
+    shuffleServerConf.setLong(ShuffleServerConf.FLUSH_COLD_STORAGE_THRESHOLD_SIZE, 10000L);
+    shuffleServerConf.set(RssBaseConf.RSS_STORAGE_TYPE, StorageType.LOCALFILE_HDFS.toString());
+    shuffleServerConf.set(RssBaseConf.RSS_STORAGE_BASE_PATH, Arrays.asList(tempDir.getAbsolutePath()));
+    shuffleServerConf.set(ShuffleServerConf.DISK_CAPACITY, 100L);
+    shuffleServerConf.set(ShuffleServerConf.SERVER_FLUSH_HDFS_THREAD_POOL_SIZE, 1);

Review Comment:
   Maybe we can change it to default value now.



-- 
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: issues-unsubscribe@uniffle.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@uniffle.apache.org
For additional commands, e-mail: issues-help@uniffle.apache.org