sumitagrawl commented on code in PR #8217: URL: https://github.com/apache/ozone/pull/8217#discussion_r2052305739
########## hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/lock/OmRequestGatekeeper.java: ########## @@ -0,0 +1,223 @@ +/* + * 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.ozone.om.lock; + +import com.google.common.util.concurrent.Striped; +import java.io.IOException; +import java.util.ArrayList; +import java.util.List; +import java.util.Optional; +import java.util.Set; +import java.util.Stack; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.TimeoutException; +import java.util.concurrent.locks.Lock; +import java.util.concurrent.locks.ReadWriteLock; +import org.apache.hadoop.hdds.utils.SimpleStriped; +import org.apache.hadoop.ozone.om.exceptions.OMException; +import org.apache.hadoop.util.Time; + +/** + * Manage locking of volume, bucket, keys and others. + */ +public class OmRequestGatekeeper { + private static final long LOCK_TIMEOUT_DEFAULT = 10 * 60 * 1000; + private static final int NUM_VOLUME_STRIPES = 1024; + private static final int NUM_BUCKET_STRIPES = 1024; + private static final int NUM_KEY_STRIPES = 4096; + + private final Striped<ReadWriteLock> volumeLocks; + private final Striped<ReadWriteLock> bucketLocks; + private final Striped<ReadWriteLock> keyLocks; + + public OmRequestGatekeeper() { + volumeLocks = SimpleStriped.readWriteLock(NUM_VOLUME_STRIPES, false); + bucketLocks = SimpleStriped.readWriteLock(NUM_BUCKET_STRIPES, false); + keyLocks = SimpleStriped.readWriteLock(NUM_KEY_STRIPES, false); + } + + public OmLockObject lock(OmLockInfo lockInfo) throws IOException { + OmLockObject omLockObject = new OmLockObject(lockInfo); + long startTime = Time.monotonicNowNanos(); + Optional<OmLockInfo.LockInfo> optionalVolumeLock = lockInfo.getVolumeLock(); + Optional<OmLockInfo.LockInfo> optionalBucketLock = lockInfo.getBucketLock(); + Optional<Set<OmLockInfo.LockInfo>> optionalKeyLocks = lockInfo.getKeyLocks(); + List<Lock> locks = new ArrayList<>(); + + if (optionalVolumeLock.isPresent()) { + OmLockInfo.LockInfo volumeLockInfo = optionalVolumeLock.get(); + if (volumeLockInfo.isWriteLock()) { + omLockObject.setReadStatsType(false); + locks.add(volumeLocks.get(volumeLockInfo.getName()).writeLock()); + } else { + locks.add(volumeLocks.get(volumeLockInfo.getName()).readLock()); + } + } + + if (optionalBucketLock.isPresent()) { + OmLockInfo.LockInfo bucketLockInfo = optionalBucketLock.get(); + if (bucketLockInfo.isWriteLock()) { + omLockObject.setReadStatsType(false); + locks.add(bucketLocks.get(bucketLockInfo.getName()).writeLock()); + } else { + locks.add(bucketLocks.get(bucketLockInfo.getName()).readLock()); + } + } + + if (optionalKeyLocks.isPresent()) { + for (ReadWriteLock keyLock: keyLocks.bulkGet(optionalKeyLocks.get())) { + omLockObject.setReadStatsType(false); + locks.add(keyLock.writeLock()); + } + } + + try { + acquireLocks(locks, omLockObject.getLocks()); + lockStatsBegin(omLockObject.getLockStats(), Time.monotonicNowNanos(), startTime); + } catch (InterruptedException e) { + Thread.currentThread().interrupt(); + throw new OMException("Waiting for locks is interrupted, " + lockInfo, OMException.ResultCodes.INTERNAL_ERROR); + } catch (TimeoutException e) { + throw new OMException("Timeout occurred for locks " + lockInfo, OMException.ResultCodes.TIMEOUT); + } + return omLockObject; + } + + private void acquireLocks(List<Lock> locks, Stack<Lock> acquiredLocks) throws TimeoutException, InterruptedException { Review Comment: added section for having timeout in design doc for lock -- 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]
