You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@ozone.apache.org by GitBox <gi...@apache.org> on 2021/03/29 08:28:55 UTC

[GitHub] [ozone] elek opened a new pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

elek opened a new pull request #2089:
URL: https://github.com/apache/ozone/pull/2089


   ## What changes were proposed in this pull request?
   
   This patch introduces the usage of `ReplicationConfig` on mater instead of using `replicationType` and `replicationFactor`. It is required by EC (see #2068 and #1973)
   
   Using java based `ReplicationFactor` requires further changes in SCM (and OM) and we decidede to make the generic part of the refactor on master (it doesn't introduce any new logic only the java representation of the data is changing)  
   
   ## What is the link to the Apache JIRA
   
   https://issues.apache.org/jira/browse/HDDS-5011
   
   ## How was this patch tested?
   
   It contains unit tests. It introduces only new Java classes, therefore it's safe to include.


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

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



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


[GitHub] [ozone] elek commented on a change in pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

Posted by GitBox <gi...@apache.org>.
elek commented on a change in pull request #2089:
URL: https://github.com/apache/ozone/pull/2089#discussion_r607846097



##########
File path: hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/client/StandaloneReplicationConfig.java
##########
@@ -0,0 +1,72 @@
+/**
+ * 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.hadoop.hdds.client;
+
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType;
+
+import java.util.Objects;
+
+/**
+ * Replication configuration for STANDALONE replication.
+ */
+public class StandaloneReplicationConfig implements ReplicationConfig {
+
+  private final ReplicationFactor replicationFactor;
+
+  public StandaloneReplicationConfig(ReplicationFactor replicationFactor) {

Review comment:
       I am thinking about it, and I think we may have STANDALONE/THREE support. I think we upload the key only to one server but the key may be replicated to multiple nodes by `ReplicationManager` when the container is closed...




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

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



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


[GitHub] [ozone] sodonnel commented on a change in pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

Posted by GitBox <gi...@apache.org>.
sodonnel commented on a change in pull request #2089:
URL: https://github.com/apache/ozone/pull/2089#discussion_r605017982



##########
File path: hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/client/StandaloneReplicationConfig.java
##########
@@ -0,0 +1,72 @@
+/**
+ * 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.hadoop.hdds.client;
+
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType;
+
+import java.util.Objects;
+
+/**
+ * Replication configuration for STANDALONE replication.
+ */
+public class StandaloneReplicationConfig implements ReplicationConfig {
+
+  private final ReplicationFactor replicationFactor;
+
+  public StandaloneReplicationConfig(ReplicationFactor replicationFactor) {

Review comment:
       Is Standalone by definition factor 1? Should we drop the replicationFactor parameter and just hardcode Factor.ONE ?




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

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



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


[GitHub] [ozone] elek commented on pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

Posted by GitBox <gi...@apache.org>.
elek commented on pull request #2089:
URL: https://github.com/apache/ozone/pull/2089#issuecomment-814699171


   Merging it. Thanks the review @sodonnel and @umamaheswararao 


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

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



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


[GitHub] [ozone] umamaheswararao commented on a change in pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

Posted by GitBox <gi...@apache.org>.
umamaheswararao commented on a change in pull request #2089:
URL: https://github.com/apache/ozone/pull/2089#discussion_r607997184



##########
File path: hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/client/StandaloneReplicationConfig.java
##########
@@ -0,0 +1,72 @@
+/**
+ * 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.hadoop.hdds.client;
+
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType;
+
+import java.util.Objects;
+
+/**
+ * Replication configuration for STANDALONE replication.
+ */
+public class StandaloneReplicationConfig implements ReplicationConfig {
+
+  private final ReplicationFactor replicationFactor;
+
+  public StandaloneReplicationConfig(ReplicationFactor replicationFactor) {

Review comment:
       I agree. Changing options for existing things should be discussed if we really want to do that. Let's just scope this JIRA to ad ReplicationConfig.




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

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



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


[GitHub] [ozone] elek commented on a change in pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

Posted by GitBox <gi...@apache.org>.
elek commented on a change in pull request #2089:
URL: https://github.com/apache/ozone/pull/2089#discussion_r607987037



##########
File path: hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/client/StandaloneReplicationConfig.java
##########
@@ -0,0 +1,72 @@
+/**
+ * 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.hadoop.hdds.client;
+
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType;
+
+import java.util.Objects;
+
+/**
+ * Replication configuration for STANDALONE replication.
+ */
+public class StandaloneReplicationConfig implements ReplicationConfig {
+
+  private final ReplicationFactor replicationFactor;
+
+  public StandaloneReplicationConfig(ReplicationFactor replicationFactor) {

Review comment:
       I think even have RATIS/ONE is a valid option. But I agree, it's quite confusing.
   
   This is something what we can start to fix together with EC as we need to modify the client to define the replication config based on the replication type. For example in case of `replicationType=EC` the `3:2` can be a valid replication config, while `replicationType=STANDALONE` we may accept only `ONE` or `ONE/THREE`.
   
   I think this kind of validation is definitely something what we should do... 




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

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



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


[GitHub] [ozone] elek commented on a change in pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

Posted by GitBox <gi...@apache.org>.
elek commented on a change in pull request #2089:
URL: https://github.com/apache/ozone/pull/2089#discussion_r607841890



##########
File path: hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/client/StandaloneReplicationConfig.java
##########
@@ -0,0 +1,72 @@
+/**
+ * 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.hadoop.hdds.client;
+
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType;
+
+import java.util.Objects;
+
+/**
+ * Replication configuration for STANDALONE replication.
+ */
+public class StandaloneReplicationConfig implements ReplicationConfig {
+
+  private final ReplicationFactor replicationFactor;
+
+  public StandaloneReplicationConfig(ReplicationFactor replicationFactor) {

Review comment:
       Very good point. I agree that STANDALONE/THREE support should be removed if it's not available.
   
   However, I am not sure about the backward compatibility: let's say I already have a STANDALONE/THREE key persisted in OM rocksdb. If we don't allow THREE for STANDALONE it means that we will change the persisted THREE to ONE during de-serialization.
   
   I am not sure if it's good or not, but can be confusing.
   
   What do you think about this problem? 
   
   I don't have a strong opinion. If we remove STANDALONE/THREE support, it seems to be a bigger change (we need to implement proper validation, fix unit tests where STANDALONE/THREE is used, etc. But I am closer to your suggestion, to do this cleanup together with the other changes.




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

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



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


[GitHub] [ozone] sodonnel commented on a change in pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

Posted by GitBox <gi...@apache.org>.
sodonnel commented on a change in pull request #2089:
URL: https://github.com/apache/ozone/pull/2089#discussion_r607942705



##########
File path: hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/client/StandaloneReplicationConfig.java
##########
@@ -0,0 +1,72 @@
+/**
+ * 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.hadoop.hdds.client;
+
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType;
+
+import java.util.Objects;
+
+/**
+ * Replication configuration for STANDALONE replication.
+ */
+public class StandaloneReplicationConfig implements ReplicationConfig {
+
+  private final ReplicationFactor replicationFactor;
+
+  public StandaloneReplicationConfig(ReplicationFactor replicationFactor) {

Review comment:
       I did not realise we already had an option for a STANDALONE/THREE. I thought it was always STANDALONE/ONE or RATIS/THREE. If we already have STANDALONE/THREE then I believe you are correct and we need to keep the option to keep the factor.




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

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



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


[GitHub] [ozone] elek merged pull request #2089: HDDS-5011. Introduce Java based ReplicationConfig implementation

Posted by GitBox <gi...@apache.org>.
elek merged pull request #2089:
URL: https://github.com/apache/ozone/pull/2089


   


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

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



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