Guosmilesmile commented on code in PR #14243:
URL: https://github.com/apache/iceberg/pull/14243#discussion_r2415467460


##########
flink/v2.1/flink/src/test/java/org/apache/iceberg/flink/maintenance/api/TestZkLockFactory.java:
##########
@@ -29,7 +29,15 @@ public class TestZkLockFactory extends TestLockFactoryBase {
 
   @Override
   TriggerLockFactory lockFactory(String tableName) {
-    return new ZkLockFactory(zkTestServer.getConnectString(), tableName, 5000, 
3000, 1000, 3);
+    return new ZkLockFactory(
+        zkTestServer.getConnectString(),
+        tableName,
+        5000,
+        3000,
+        1000,
+        3,
+        2000,
+        ZKRetryPolicies.EXPONENTIAL_BACKOFF.name());

Review Comment:
   Can we add corresponding test cases to ensure that the chosen strategy 
actually works?



##########
flink/v2.1/flink/src/main/java/org/apache/iceberg/flink/maintenance/api/ZkLockFactory.java:
##########
@@ -222,4 +240,32 @@ public void unlock() {
       }
     }
   }
+
+  RetryPolicy createRetryPolicy() {
+
+    ZKRetryPolicies policy;
+    try {
+      policy = ZKRetryPolicies.valueOf(retryPolicyName);
+    } catch (IllegalArgumentException e) {
+      policy = ZKRetryPolicies.EXPONENTIAL_BACKOFF;

Review Comment:
   Can we add some log output? Swallowing exceptions directly can leave users 
puzzled.



##########
flink/v2.1/flink/src/main/java/org/apache/iceberg/flink/maintenance/api/ZkLockFactory.java:
##########
@@ -222,4 +240,32 @@ public void unlock() {
       }
     }
   }
+
+  RetryPolicy createRetryPolicy() {
+
+    ZKRetryPolicies policy;
+    try {
+      policy = ZKRetryPolicies.valueOf(retryPolicyName);
+    } catch (IllegalArgumentException e) {
+      policy = ZKRetryPolicies.EXPONENTIAL_BACKOFF;
+    }
+
+    switch (policy) {
+      case ONE_TIME:
+        return new RetryOneTime(baseSleepTimeMs);
+
+      case N_TIME:
+        return new RetryNTimes(maxRetries, baseSleepTimeMs);
+
+      case BOUNDED_EXPONENTIAL_BACKOFF:
+        return new BoundedExponentialBackoffRetry(baseSleepTimeMs, 
maxSleepTimeMs, maxRetries);

Review Comment:
   What would happen if `baseSleepTimeMs` equals `maxSleepTimeMs` and 
`maxRetries` is greater than 1?



##########
flink/v2.1/flink/src/main/java/org/apache/iceberg/flink/maintenance/api/ZkLockFactory.java:
##########
@@ -222,4 +240,32 @@ public void unlock() {
       }
     }
   }
+
+  RetryPolicy createRetryPolicy() {
+
+    ZKRetryPolicies policy;
+    try {
+      policy = ZKRetryPolicies.valueOf(retryPolicyName);
+    } catch (IllegalArgumentException e) {
+      policy = ZKRetryPolicies.EXPONENTIAL_BACKOFF;
+    }
+
+    switch (policy) {
+      case ONE_TIME:
+        return new RetryOneTime(baseSleepTimeMs);
+
+      case N_TIME:
+        return new RetryNTimes(maxRetries, baseSleepTimeMs);
+
+      case BOUNDED_EXPONENTIAL_BACKOFF:
+        return new BoundedExponentialBackoffRetry(baseSleepTimeMs, 
maxSleepTimeMs, maxRetries);
+
+      case UNTIL_ELAPSED:
+        return new RetryUntilElapsed(maxSleepTimeMs, baseSleepTimeMs);
+
+      case EXPONENTIAL_BACKOFF:
+      default:
+        return new ExponentialBackoffRetry(maxSleepTimeMs, maxRetries);

Review Comment:
   The `maxSleepTimeMs` here should be `baseSleepTimeMs`.



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