Uploaded image for project: 'Apache Curator'
  1. Apache Curator
  2. CURATOR-637

Get Leader zNode actual path from LeaderSelector

    XMLWordPrintableJSON

Details

    • Improvement
    • Status: Open
    • Minor
    • Resolution: Unresolved
    • 5.2.1
    • TBD
    • Recipes
    • None

    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.

       

      if (isLeader.get()) {
           try {
                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)){
                                         myChildPath = childPath;                                             
                                         break;                                         
                                    }
                               } catch (Exception e){
                                   //ignore                                     
                               }
                          }
                     if (null != myChildPath){
                         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{                    
             System.out.println("I AM THE LEADER");                     
             isLeader.set(true);                     
             Thread.sleep(Long.MAX_VALUE);                 
         } catch (@SuppressWarnings("unused") InterruptedException e){                     
         } 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() as InterProcessMutex stores that path in threadData map 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#

      Attachments

        Activity

          People

            kezhuw Kezhu Wang
            robinsingh@ameyo.com Robin Singh
            Votes:
            0 Vote for this issue
            Watchers:
            2 Start watching this issue

            Dates

              Created:
              Updated: