|
189 | 189 | import static org.elasticsearch.core.Strings.format; |
190 | 190 | import static org.elasticsearch.index.seqno.RetentionLeaseActions.RETAIN_ALL; |
191 | 191 | import static org.elasticsearch.index.seqno.SequenceNumbers.UNASSIGNED_SEQ_NO; |
| 192 | +import static org.elasticsearch.index.shard.IndexShard.PrimaryPermitCheck.PRIMARY_MODE; |
192 | 193 |
|
193 | 194 | public class IndexShard extends AbstractIndexShardComponent implements IndicesClusterStateService.Shard { |
194 | 195 |
|
@@ -3568,58 +3569,96 @@ private EngineConfig newEngineConfig(LongSupplier globalCheckpointSupplier) { |
3568 | 3569 | ); |
3569 | 3570 | } |
3570 | 3571 |
|
| 3572 | + /** |
| 3573 | + * Checks to run before running the primary permit operation |
| 3574 | + */ |
| 3575 | + public enum PrimaryPermitCheck { |
| 3576 | + PRIMARY_MODE, |
| 3577 | + NONE |
| 3578 | + } |
| 3579 | + |
3571 | 3580 | /** |
3572 | 3581 | * Acquire a primary operation permit whenever the shard is ready for indexing. If a permit is directly available, the provided |
3573 | 3582 | * ActionListener will be called on the calling thread. During relocation hand-off, permit acquisition can be delayed. The provided |
3574 | 3583 | * ActionListener will then be called using the provided executor. |
3575 | | - * |
3576 | 3584 | */ |
3577 | 3585 | public void acquirePrimaryOperationPermit(ActionListener<Releasable> onPermitAcquired, Executor executorOnDelay) { |
3578 | | - acquirePrimaryOperationPermit(onPermitAcquired, executorOnDelay, false); |
| 3586 | + acquirePrimaryOperationPermit(onPermitAcquired, executorOnDelay, false, PRIMARY_MODE); |
3579 | 3587 | } |
3580 | 3588 |
|
3581 | 3589 | public void acquirePrimaryOperationPermit( |
3582 | 3590 | ActionListener<Releasable> onPermitAcquired, |
3583 | 3591 | Executor executorOnDelay, |
3584 | 3592 | boolean forceExecution |
| 3593 | + ) { |
| 3594 | + acquirePrimaryOperationPermit(onPermitAcquired, executorOnDelay, forceExecution, PRIMARY_MODE); |
| 3595 | + } |
| 3596 | + |
| 3597 | + public void acquirePrimaryOperationPermit( |
| 3598 | + ActionListener<Releasable> onPermitAcquired, |
| 3599 | + Executor executorOnDelay, |
| 3600 | + boolean forceExecution, |
| 3601 | + PrimaryPermitCheck primaryPermitCheck |
3585 | 3602 | ) { |
3586 | 3603 | verifyNotClosed(); |
3587 | 3604 | assert shardRouting.primary() : "acquirePrimaryOperationPermit should only be called on primary shard: " + shardRouting; |
3588 | | - indexShardOperationPermits.acquire(wrapPrimaryOperationPermitListener(onPermitAcquired), executorOnDelay, forceExecution); |
| 3605 | + indexShardOperationPermits.acquire( |
| 3606 | + wrapPrimaryOperationPermitListener(primaryPermitCheck, onPermitAcquired), |
| 3607 | + executorOnDelay, |
| 3608 | + forceExecution |
| 3609 | + ); |
3589 | 3610 | } |
3590 | 3611 |
|
3591 | 3612 | public boolean isPrimaryMode() { |
3592 | 3613 | assert indexShardOperationPermits.getActiveOperationsCount() != 0 : "must hold permit to check primary mode"; |
3593 | 3614 | return replicationTracker.isPrimaryMode(); |
3594 | 3615 | } |
3595 | 3616 |
|
| 3617 | + public void acquireAllPrimaryOperationsPermits(final ActionListener<Releasable> onPermitAcquired, final TimeValue timeout) { |
| 3618 | + acquireAllPrimaryOperationsPermits(onPermitAcquired, timeout, PRIMARY_MODE); |
| 3619 | + } |
| 3620 | + |
3596 | 3621 | /** |
3597 | 3622 | * Acquire all primary operation permits. Once all permits are acquired, the provided ActionListener is called. |
3598 | 3623 | * It is the responsibility of the caller to close the {@link Releasable}. |
3599 | 3624 | */ |
3600 | | - public void acquireAllPrimaryOperationsPermits(final ActionListener<Releasable> onPermitAcquired, final TimeValue timeout) { |
| 3625 | + public void acquireAllPrimaryOperationsPermits( |
| 3626 | + final ActionListener<Releasable> onPermitAcquired, |
| 3627 | + final TimeValue timeout, |
| 3628 | + final PrimaryPermitCheck primaryPermitCheck |
| 3629 | + ) { |
3601 | 3630 | verifyNotClosed(); |
3602 | 3631 | assert shardRouting.primary() : "acquireAllPrimaryOperationsPermits should only be called on primary shard: " + shardRouting; |
3603 | 3632 |
|
3604 | | - asyncBlockOperations(wrapPrimaryOperationPermitListener(onPermitAcquired), timeout.duration(), timeout.timeUnit()); |
| 3633 | + asyncBlockOperations( |
| 3634 | + wrapPrimaryOperationPermitListener(primaryPermitCheck, onPermitAcquired), |
| 3635 | + timeout.duration(), |
| 3636 | + timeout.timeUnit() |
| 3637 | + ); |
3605 | 3638 | } |
3606 | 3639 |
|
3607 | 3640 | /** |
3608 | | - * Wraps the action to run on a primary after acquiring permit. This wrapping is used to check if the shard is in primary mode before |
3609 | | - * executing the action. |
| 3641 | + * Wraps the action to run on a primary after acquiring permit. |
3610 | 3642 | * |
| 3643 | + * @param primaryPermitCheck check to run before the primary mode operation |
3611 | 3644 | * @param listener the listener to wrap |
3612 | 3645 | * @return the wrapped listener |
3613 | 3646 | */ |
3614 | | - private ActionListener<Releasable> wrapPrimaryOperationPermitListener(final ActionListener<Releasable> listener) { |
3615 | | - return listener.delegateFailure((l, r) -> { |
3616 | | - if (isPrimaryMode()) { |
3617 | | - l.onResponse(r); |
3618 | | - } else { |
3619 | | - r.close(); |
3620 | | - l.onFailure(new ShardNotInPrimaryModeException(shardId, state)); |
3621 | | - } |
3622 | | - }); |
| 3647 | + private ActionListener<Releasable> wrapPrimaryOperationPermitListener( |
| 3648 | + final PrimaryPermitCheck primaryPermitCheck, |
| 3649 | + final ActionListener<Releasable> listener |
| 3650 | + ) { |
| 3651 | + return switch (primaryPermitCheck) { |
| 3652 | + case PRIMARY_MODE -> listener.delegateFailure((l, r) -> { |
| 3653 | + if (isPrimaryMode()) { |
| 3654 | + l.onResponse(r); |
| 3655 | + } else { |
| 3656 | + r.close(); |
| 3657 | + l.onFailure(new ShardNotInPrimaryModeException(shardId, state)); |
| 3658 | + } |
| 3659 | + }); |
| 3660 | + case NONE -> listener; |
| 3661 | + }; |
3623 | 3662 | } |
3624 | 3663 |
|
3625 | 3664 | private void asyncBlockOperations(ActionListener<Releasable> onPermitAcquired, long timeout, TimeUnit timeUnit) { |
@@ -3657,7 +3696,7 @@ public void runUnderPrimaryPermit(final Runnable runnable, final Consumer<Except |
3657 | 3696 | runnable.run(); |
3658 | 3697 | } |
3659 | 3698 | }, onFailure); |
3660 | | - acquirePrimaryOperationPermit(onPermitAcquired, executorOnDelay); |
| 3699 | + acquirePrimaryOperationPermit(onPermitAcquired, executorOnDelay, false, PRIMARY_MODE); |
3661 | 3700 | } |
3662 | 3701 |
|
3663 | 3702 | private <E extends Exception> void bumpPrimaryTerm( |
|
0 commit comments