You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@uniffle.apache.org by GitBox <gi...@apache.org> on 2023/01/06 09:56:40 UTC

[GitHub] [incubator-uniffle] zuston commented on a diff in pull request #449: [ISSUE-448][Feature] shuffle server report storage info

zuston commented on code in PR #449:
URL: https://github.com/apache/incubator-uniffle/pull/449#discussion_r1063265004


##########
common/src/main/java/org/apache/uniffle/common/storage/SingleStorageType.java:
##########
@@ -0,0 +1,67 @@
+/*
+ * 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.common.storage;
+
+import org.apache.uniffle.proto.RssProtos;
+
+public enum SingleStorageType {
+  Unknown(0),

Review Comment:
   Unknown -> `UNKNOWN?`



##########
common/src/main/java/org/apache/uniffle/common/storage/SingleStorageType.java:
##########
@@ -0,0 +1,67 @@
+/*
+ * 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.common.storage;
+
+import org.apache.uniffle.proto.RssProtos;
+
+public enum SingleStorageType {
+  Unknown(0),
+  HDD(1),
+  SSD(2),
+  HDFS(3),
+  OBJECT(4);

Review Comment:
   OBJECT -> `OBJECT_STORE` ?



##########
common/src/main/java/org/apache/uniffle/common/storage/SingleStorageType.java:
##########
@@ -0,0 +1,67 @@
+/*
+ * 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.common.storage;
+
+import org.apache.uniffle.proto.RssProtos;
+
+public enum SingleStorageType {
+  Unknown(0),
+  HDD(1),
+  SSD(2),
+  HDFS(3),
+  OBJECT(4);
+
+  private final byte val;
+
+  SingleStorageType(int code) {
+    assert (code >= -1 && code < 256);
+    this.val = (byte) code;
+  }
+
+  public RssProtos.StorageInfo.SingleStorageType toProto() {
+    switch (this) {
+      case Unknown:
+        return RssProtos.StorageInfo.SingleStorageType.STORAGE_TYPE_UNKNOWN;
+      case HDD:
+        return RssProtos.StorageInfo.SingleStorageType.HDD;
+      case SSD:
+        return RssProtos.StorageInfo.SingleStorageType.SSD;
+      case HDFS:
+        return RssProtos.StorageInfo.SingleStorageType.HDFS;
+      case OBJECT:
+        return RssProtos.StorageInfo.SingleStorageType.OBJECT;
+      default:
+        return RssProtos.StorageInfo.SingleStorageType.UNRECOGNIZED;

Review Comment:
   What's the different with `UNRECOGNIZED` and `STORAGE_TYPE_UNKNOWN` ?



##########
common/src/main/java/org/apache/uniffle/common/storage/StorageStatus.java:
##########
@@ -0,0 +1,66 @@
+/*
+ * 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.common.storage;
+
+import org.apache.uniffle.proto.RssProtos;
+
+public enum StorageStatus {
+  UNKNOWN(0),
+  NORMAL(1),
+  UNHEALTHY(2),

Review Comment:
   What will make status unhealthy ? For example Disk I/O is too high?



##########
proto/src/main/proto/Rss.proto:
##########
@@ -239,6 +239,7 @@ message ShuffleServerHeartBeatRequest {
   int32 eventNumInFlush = 5;
   repeated string tags = 6;
   google.protobuf.BoolValue isHealthy = 7;
+  map<string, StorageInfo> localStorages = 21; // mount point to storage info mapping.

Review Comment:
   Why the proto var number is 21 instead of 8?



##########
common/src/main/java/org/apache/uniffle/common/storage/StorageStatus.java:
##########
@@ -0,0 +1,66 @@
+/*
+ * 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.common.storage;
+
+import org.apache.uniffle.proto.RssProtos;
+
+public enum StorageStatus {
+  UNKNOWN(0),
+  NORMAL(1),
+  UNHEALTHY(2),
+  OVERUSED(3);
+
+  private final byte val;
+
+  StorageStatus(int code) {
+    assert (code >= -1 && code < 256);
+    this.val = (byte) code;
+  }
+
+  public final byte getCode() {
+    return val;
+  }
+
+  public RssProtos.StorageInfo.StorageStatus toProto() {
+    switch (this) {
+      case UNKNOWN:
+        return RssProtos.StorageInfo.StorageStatus.STORAGE_STATUS_UNKNOWN;
+      case NORMAL:
+        return RssProtos.StorageInfo.StorageStatus.NORMAL;
+      case UNHEALTHY:
+        return RssProtos.StorageInfo.StorageStatus.UNHEALTHY;
+      case OVERUSED:
+        return RssProtos.StorageInfo.StorageStatus.OVERUSED;
+      default:
+        return RssProtos.StorageInfo.StorageStatus.UNRECOGNIZED;

Review Comment:
   What's the different with `STORAGE_STATUS_UNKNOWN` and `UNRECOGNIZED`



##########
server/src/main/java/org/apache/uniffle/server/storage/LocalStorageManager.java:
##########
@@ -94,19 +101,26 @@ public class LocalStorageManager extends SingleStorageManager {
     // We must make sure the order of `storageBasePaths` and `localStorages` is same, or some unit test may be fail
     CountDownLatch countDownLatch = new CountDownLatch(storageBasePaths.size());
     AtomicInteger successCount = new AtomicInteger();
+    ServiceLoader<StorageTypeProvider> loader = ServiceLoader.load(StorageTypeProvider.class);

Review Comment:
   Why we need multiple storage type providers? I think one is enough.



##########
storage/src/main/java/org/apache/uniffle/storage/common/DefaultStorageTypeProvider.java:
##########
@@ -0,0 +1,27 @@
+/*
+ * 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.storage.common;
+
+import org.apache.uniffle.common.storage.SingleStorageType;
+
+public class DefaultStorageTypeProvider implements StorageTypeProvider {
+  @Override
+  public SingleStorageType getStorageTypeFor(String baseDir) {
+    return SingleStorageType.HDD;

Review Comment:
   I think this could be as default storage type provider, which could find out the storage type by linux command, https://unix.stackexchange.com/questions/65595/how-to-know-if-a-disk-is-an-ssd-or-an-hdd



##########
common/src/main/java/org/apache/uniffle/common/storage/SingleStorageType.java:
##########
@@ -0,0 +1,67 @@
+/*
+ * 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.common.storage;
+
+import org.apache.uniffle.proto.RssProtos;
+
+public enum SingleStorageType {

Review Comment:
   SingleStorageType rename to `UnderlyingStorageType` ? The word of `Single` looks general, and the name of `StorageType` has been occupied.



##########
server/src/main/java/org/apache/uniffle/server/storage/StorageTypeFromEnvProvider.java:
##########
@@ -0,0 +1,81 @@
+/*
+ * 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.storage;
+
+import java.io.File;
+import java.util.Map;
+
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import com.google.common.collect.Maps;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import org.apache.uniffle.common.config.RssConf;
+import org.apache.uniffle.common.storage.SingleStorageType;
+import org.apache.uniffle.server.ShuffleServerConf;
+import org.apache.uniffle.storage.common.StorageTypeProvider;
+
+public class StorageTypeFromEnvProvider implements StorageTypeProvider {

Review Comment:
   I think the storage type mapping in env is too implict. Why not using config file? 



##########
server/src/main/java/org/apache/uniffle/server/storage/StorageTypeFromEnvProvider.java:
##########
@@ -0,0 +1,81 @@
+/*
+ * 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.storage;
+
+import java.io.File;
+import java.util.Map;
+
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import com.google.common.collect.Maps;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import org.apache.uniffle.common.config.RssConf;
+import org.apache.uniffle.common.storage.SingleStorageType;
+import org.apache.uniffle.server.ShuffleServerConf;
+import org.apache.uniffle.storage.common.StorageTypeProvider;
+
+public class StorageTypeFromEnvProvider implements StorageTypeProvider {

Review Comment:
   If this is only used for test, it's ok for me.



-- 
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