You are viewing a plain text version of this content. The canonical link for it is here.
Posted to dev@curator.apache.org by "Robin Singh (Jira)" <ji...@apache.org> on 2022/05/02 03:17:00 UTC

[jira] [Updated] (CURATOR-637) Get Leader zNode actual path from LeaderSelector

     [ https://issues.apache.org/jira/browse/CURATOR-637?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ]

Robin Singh updated CURATOR-637:
--------------------------------
    Description: 
We are exploring LeaderSelector to implement leader election in our spring application. To implement fencing, we have to fetch all the participating zNodes in the leader election and then match the leaderSelector Id with each child's data.

 

This code is being executed in a Timer.

 
{code:java}
if (isLeader.get()) {
                    try {
                        //Stat stat = zookeeper.exists("/spring-integration/fence", false);
                        curator.createContainers("/spring-integration/fence");
                        Stat stat = curator.checkExists().forPath("/spring-integration/leader");
                        if (null != stat) {
                            List<String> allChilds = curator.getChildren().forPath("/spring-integration/leader");
                            if (allChilds != null && !allChilds.isEmpty()) {
                                String myChildPath = null;
                                for (String child : allChilds) {
                                    String childPath = "/spring-integration/leader/" + child;
                                    try {
                                        String childId = new String(curator.getData().forPath(childPath), "UTF-8");
                                        if (myId.equals(childId))
{ // checking for my zNode                                             myChildPath = childPath;                                             break;                                         }
                                    } catch (Exception e)
{                                         //ignore                                     }
                                }
                                if (null != myChildPath)
{ // fencing, update only if my childPath exists                                     curator.transaction().forOperations(                                             curator.transactionOp().setData().forPath("/spring-integration/fence",                                                     new String("" + counter.incrementAndGet()).getBytes()),                                             curator.transactionOp().check().forPath(myChildPath));                                 }
                            }
                            String stringData = new String(curator.getData().forPath("/spring-integration/fence"));
                            System.out.println("SAVED DATA: " + stringData);
                        }
                    } catch (Exception e)
{                         e.printStackTrace();                     }
                } else
{                     System.out.println("I am Not the leader, I cannot write anything");                     leaderSelector.requeue();                 }
{code}
 

AutoRequeue is disabled, we are requeing leader selection if isLeader is false.

 

This is the leaderSelectorListener.
{code:java}
 
@Override
            public void takeLeadership(CuratorFramework client) throws Exception {
                try
{                     // when this method exits, the leadership will be revoked;                     // therefore this thread needs to be held up until the                     // candidate is no longer leader                     System.out.println("I AM THE LEADER");                     isLeader.set(true);                     Thread.sleep(Long.MAX_VALUE);                 }
catch (@SuppressWarnings("unused") InterruptedException e)
{                     // InterruptedException, like any other runtime exception,                     // is handled by the finally block below. No need to                     // reset the interrupt flag as the interrupt is handled.                 }
finally
{                     System.out.println("I AM NOT THE LEADER ANYMORE");                     myId = generateId();                     leaderSelector.setId(myId);                     isLeader.set(false);                 }
            }
{code}
 

Generating a new ID so that when requeing for leaderElection, a new node will be generated with a different ID.

Question 1. Do I have to re generate ID so that when requeing for leaderElection a new zNode will be created using that new ID? Can I assume that in all the scenarios the zNode will always be deleted and then only a new leader will be elected from the participants?

 

Improvement 1. If only we could get the zNode path for our application using leaderSelection.getCurrentPath() the code for fencing would be cleaned up.

 

I was checking some issues related to fencing and leader based dataStore. And from there I found this document [https://docs.google.com/document/d/1cBY1t0k5g1xNqzyfZby3LcPu4t-wpx57G1xf-nmWrCo/edit#]

  was:
We are exploring LeaderSelector to implement leader election in our spring application. To implement fencing, we have to fetch all the participating zNodes in the leader election and then match the leaderSelector Id with each child's data. 

 

This code is being executed in a Timer.

if (isLeader.get()) {
                    try {
                        //Stat stat = zookeeper.exists("/spring-integration/fence", false);
                        curator.createContainers("/spring-integration/fence");
                        Stat stat = curator.checkExists().forPath("/spring-integration/leader");
                        if (null != stat) {
                            List<String> allChilds = curator.getChildren().forPath("/spring-integration/leader");
                            if (allChilds != null && !allChilds.isEmpty()) {
                                String myChildPath = null;
                                for (String child : allChilds) {
                                    String childPath = "/spring-integration/leader/" + child;
                                    try {
                                        String childId = new String(curator.getData().forPath(childPath), "UTF-8");
                                        if (myId.equals(childId)) { // checking for my zNode
                                            myChildPath = childPath;
                                            break;
                                        }
                                    } catch (Exception e) {
                                        //ignore
                                    }
                                }
                                if (null != myChildPath) { // fencing, update only if my childPath exists
                                    curator.transaction().forOperations(
                                            curator.transactionOp().setData().forPath("/spring-integration/fence",
                                                    new String("" + counter.incrementAndGet()).getBytes()),
                                            curator.transactionOp().check().forPath(myChildPath));
                                }
                            }
                            String stringData = new String(curator.getData().forPath("/spring-integration/fence"));
                            System.out.println("SAVED DATA: " + stringData);
                        }
                    } catch (Exception e) {
                        e.printStackTrace();
                    }
                } else {
                    System.out.println("I am Not the leader, I cannot write anything");
                    leaderSelector.requeue();
                }

AutoRequeue is disabled, we are requeing leader selection if isLeader is false.

 

This is the leaderSelectorListener.

@Override
            public void takeLeadership(CuratorFramework client) throws Exception {
                try {
                    // when this method exits, the leadership will be revoked;
                    // therefore this thread needs to be held up until the
                    // candidate is no longer leader
                    System.out.println("I AM THE LEADER");
                    isLeader.set(true);
                    Thread.sleep(Long.MAX_VALUE);
                } catch (@SuppressWarnings("unused") InterruptedException e) {
                    // InterruptedException, like any other runtime exception,
                    // is handled by the finally block below. No need to
                    // reset the interrupt flag as the interrupt is handled.
                } finally {
                    System.out.println("I AM NOT THE LEADER ANYMORE");
                    myId = generateId();
                    leaderSelector.setId(myId);
                    isLeader.set(false);
                }
            }

Generating a new ID so that when requeing for leaderElection, a new node will be generated with a different ID.

Question 1. Do I have to re generate ID so that when requeing for leaderElection a new zNode will be created using that new ID? Can I assume that in all the scenarios the zNode will always be deleted and then only a new leader will be elected from the participants?

 

Improvement 1. If only we could get the zNode path for our application using leaderSelection.getCurrentPath() the code for fencing would be cleaned up.

 

I was checking some issues related to fencing and leader based dataStore. And from there I found this document https://docs.google.com/document/d/1cBY1t0k5g1xNqzyfZby3LcPu4t-wpx57G1xf-nmWrCo/edit#


> Get Leader zNode actual path from LeaderSelector
> ------------------------------------------------
>
>                 Key: CURATOR-637
>                 URL: https://issues.apache.org/jira/browse/CURATOR-637
>             Project: Apache Curator
>          Issue Type: Improvement
>          Components: Recipes
>    Affects Versions: 5.2.1
>            Reporter: Robin Singh
>            Priority: Minor
>             Fix For: TBD
>
>
> We are exploring LeaderSelector to implement leader election in our spring application. To implement fencing, we have to fetch all the participating zNodes in the leader election and then match the leaderSelector Id with each child's data.
>  
> This code is being executed in a Timer.
>  
> {code:java}
> if (isLeader.get()) {
>                     try {
>                         //Stat stat = zookeeper.exists("/spring-integration/fence", false);
>                         curator.createContainers("/spring-integration/fence");
>                         Stat stat = curator.checkExists().forPath("/spring-integration/leader");
>                         if (null != stat) {
>                             List<String> allChilds = curator.getChildren().forPath("/spring-integration/leader");
>                             if (allChilds != null && !allChilds.isEmpty()) {
>                                 String myChildPath = null;
>                                 for (String child : allChilds) {
>                                     String childPath = "/spring-integration/leader/" + child;
>                                     try {
>                                         String childId = new String(curator.getData().forPath(childPath), "UTF-8");
>                                         if (myId.equals(childId))
> { // checking for my zNode                                             myChildPath = childPath;                                             break;                                         }
>                                     } catch (Exception e)
> {                                         //ignore                                     }
>                                 }
>                                 if (null != myChildPath)
> { // fencing, update only if my childPath exists                                     curator.transaction().forOperations(                                             curator.transactionOp().setData().forPath("/spring-integration/fence",                                                     new String("" + counter.incrementAndGet()).getBytes()),                                             curator.transactionOp().check().forPath(myChildPath));                                 }
>                             }
>                             String stringData = new String(curator.getData().forPath("/spring-integration/fence"));
>                             System.out.println("SAVED DATA: " + stringData);
>                         }
>                     } catch (Exception e)
> {                         e.printStackTrace();                     }
>                 } else
> {                     System.out.println("I am Not the leader, I cannot write anything");                     leaderSelector.requeue();                 }
> {code}
>  
> AutoRequeue is disabled, we are requeing leader selection if isLeader is false.
>  
> This is the leaderSelectorListener.
> {code:java}
>  
> @Override
>             public void takeLeadership(CuratorFramework client) throws Exception {
>                 try
> {                     // when this method exits, the leadership will be revoked;                     // therefore this thread needs to be held up until the                     // candidate is no longer leader                     System.out.println("I AM THE LEADER");                     isLeader.set(true);                     Thread.sleep(Long.MAX_VALUE);                 }
> catch (@SuppressWarnings("unused") InterruptedException e)
> {                     // InterruptedException, like any other runtime exception,                     // is handled by the finally block below. No need to                     // reset the interrupt flag as the interrupt is handled.                 }
> finally
> {                     System.out.println("I AM NOT THE LEADER ANYMORE");                     myId = generateId();                     leaderSelector.setId(myId);                     isLeader.set(false);                 }
>             }
> {code}
>  
> Generating a new ID so that when requeing for leaderElection, a new node will be generated with a different ID.
> Question 1. Do I have to re generate ID so that when requeing for leaderElection a new zNode will be created using that new ID? Can I assume that in all the scenarios the zNode will always be deleted and then only a new leader will be elected from the participants?
>  
> Improvement 1. If only we could get the zNode path for our application using leaderSelection.getCurrentPath() the code for fencing would be cleaned up.
>  
> I was checking some issues related to fencing and leader based dataStore. And from there I found this document [https://docs.google.com/document/d/1cBY1t0k5g1xNqzyfZby3LcPu4t-wpx57G1xf-nmWrCo/edit#]



--
This message was sent by Atlassian Jira
(v8.20.7#820007)