blob: 123c3b6a40241174ae799d224417be0227e9b302 [file] [log] [blame]
Madan Jampani15b8ef52016-02-02 17:35:05 -08001/*
Brian O'Connora09fe5b2017-08-03 21:12:30 -07002 * Copyright 2016-present Open Networking Foundation
Madan Jampani15b8ef52016-02-02 17:35:05 -08003 *
4 * Licensed under the Apache License, Version 2.0 (the "License");
5 * you may not use this file except in compliance with the License.
6 * You may obtain a copy of the License at
7 *
8 * http://www.apache.org/licenses/LICENSE-2.0
9 *
10 * Unless required by applicable law or agreed to in writing, software
11 * distributed under the License is distributed on an "AS IS" BASIS,
12 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
13 * See the License for the specific language governing permissions and
14 * limitations under the License.
15 */
16package org.onosproject.store.primitives.impl;
17
Madan Jampani15b8ef52016-02-02 17:35:05 -080018import java.io.File;
Jordan Halterman980a8c12017-09-22 18:01:19 -070019import java.io.IOException;
20import java.nio.file.Files;
Jordan Halterman19201232017-09-12 17:20:26 -070021import java.time.Duration;
Madan Jampani15b8ef52016-02-02 17:35:05 -080022import java.util.Collection;
Madan Jampani15b8ef52016-02-02 17:35:05 -080023import java.util.concurrent.CompletableFuture;
Madan Jampani15b8ef52016-02-02 17:35:05 -080024
Jordan Halterman2bf177c2017-06-29 01:49:08 -070025import io.atomix.protocols.raft.RaftServer;
26import io.atomix.protocols.raft.cluster.MemberId;
Jordan Halterman980a8c12017-09-22 18:01:19 -070027import io.atomix.protocols.raft.cluster.RaftMember;
Jordan Halterman2bf177c2017-06-29 01:49:08 -070028import io.atomix.protocols.raft.storage.RaftStorage;
29import io.atomix.storage.StorageLevel;
Jordan Halterman980a8c12017-09-22 18:01:19 -070030import org.onosproject.core.Version;
31import org.onosproject.store.cluster.messaging.UnifiedClusterCommunicationService;
Jordan Halterman2bf177c2017-06-29 01:49:08 -070032import org.onosproject.store.primitives.resources.impl.AtomixSerializerAdapter;
Madan Jampanie14a09c2016-02-11 10:43:21 -080033import org.onosproject.store.service.PartitionInfo;
Jordan Halterman2bf177c2017-06-29 01:49:08 -070034import org.onosproject.store.service.Serializer;
Madan Jampanifc981772016-02-16 09:46:42 -080035import org.slf4j.Logger;
Madan Jampani15b8ef52016-02-02 17:35:05 -080036
Jordan Halterman2bf177c2017-06-29 01:49:08 -070037import static org.slf4j.LoggerFactory.getLogger;
38
Madan Jampani15b8ef52016-02-02 17:35:05 -080039/**
40 * {@link StoragePartition} server.
41 */
42public class StoragePartitionServer implements Managed<StoragePartitionServer> {
43
Madan Jampanifc981772016-02-16 09:46:42 -080044 private final Logger log = getLogger(getClass());
45
Jordan Halterman035231e2017-07-18 08:39:07 -070046 private static final int MAX_SEGMENT_SIZE = 1024 * 1024 * 64;
Jordan Halterman19201232017-09-12 17:20:26 -070047 private static final long ELECTION_TIMEOUT_MILLIS = 2500;
48 private static final long HEARTBEAT_INTERVAL_MILLIS = 1000;
49
Jordan Halterman2bf177c2017-06-29 01:49:08 -070050 private final MemberId localMemberId;
Madan Jampani15b8ef52016-02-02 17:35:05 -080051 private final StoragePartition partition;
Jordan Halterman980a8c12017-09-22 18:01:19 -070052 private final UnifiedClusterCommunicationService clusterCommunicator;
Jordan Halterman2bf177c2017-06-29 01:49:08 -070053 private RaftServer server;
Madan Jampani15b8ef52016-02-02 17:35:05 -080054
Jordan Halterman2bf177c2017-06-29 01:49:08 -070055 public StoragePartitionServer(
Madan Jampani15b8ef52016-02-02 17:35:05 -080056 StoragePartition partition,
Jordan Halterman2bf177c2017-06-29 01:49:08 -070057 MemberId localMemberId,
Jordan Halterman980a8c12017-09-22 18:01:19 -070058 UnifiedClusterCommunicationService clusterCommunicator) {
Madan Jampani15b8ef52016-02-02 17:35:05 -080059 this.partition = partition;
Jordan Halterman2bf177c2017-06-29 01:49:08 -070060 this.localMemberId = localMemberId;
Jordan Halterman980a8c12017-09-22 18:01:19 -070061 this.clusterCommunicator = clusterCommunicator;
Madan Jampani15b8ef52016-02-02 17:35:05 -080062 }
63
64 @Override
65 public CompletableFuture<Void> open() {
Jordan Halterman980a8c12017-09-22 18:01:19 -070066 log.info("Starting server for partition {} ({})", partition.getId(), partition.getVersion());
Jordan Halterman2bf177c2017-06-29 01:49:08 -070067 CompletableFuture<RaftServer> serverOpenFuture;
68 if (partition.getMemberIds().contains(localMemberId)) {
Madan Jampani65f24bb2016-03-15 15:16:18 -070069 if (server != null && server.isRunning()) {
Madan Jampani15b8ef52016-02-02 17:35:05 -080070 return CompletableFuture.completedFuture(null);
71 }
72 synchronized (this) {
Madan Jampani630e7ac2016-05-31 11:34:05 -070073 server = buildServer();
Madan Jampani15b8ef52016-02-02 17:35:05 -080074 }
Jordan Halterman2bf177c2017-06-29 01:49:08 -070075 serverOpenFuture = server.bootstrap(partition.getMemberIds());
Madan Jampani15b8ef52016-02-02 17:35:05 -080076 } else {
77 serverOpenFuture = CompletableFuture.completedFuture(null);
78 }
Madan Jampanifc981772016-02-16 09:46:42 -080079 return serverOpenFuture.whenComplete((r, e) -> {
80 if (e == null) {
Jordan Halterman980a8c12017-09-22 18:01:19 -070081 log.info("Successfully started server for partition {} ({})",
82 partition.getId(), partition.getVersion());
Madan Jampanifc981772016-02-16 09:46:42 -080083 } else {
Jordan Halterman980a8c12017-09-22 18:01:19 -070084 log.info("Failed to start server for partition {} ({})",
85 partition.getId(), partition.getVersion(), e);
Madan Jampanifc981772016-02-16 09:46:42 -080086 }
87 }).thenApply(v -> null);
Madan Jampani15b8ef52016-02-02 17:35:05 -080088 }
89
90 @Override
91 public CompletableFuture<Void> close() {
Madan Jampani630e7ac2016-05-31 11:34:05 -070092 return server.shutdown();
Madan Jampani15b8ef52016-02-02 17:35:05 -080093 }
94
Madan Jampani33547452016-02-29 16:45:04 -080095 /**
96 * Closes the server and exits the partition.
97 * @return future that is completed when the operation is complete
98 */
99 public CompletableFuture<Void> closeAndExit() {
Madan Jampani630e7ac2016-05-31 11:34:05 -0700100 return server.leave();
Madan Jampani33547452016-02-29 16:45:04 -0800101 }
102
Jordan Halterman980a8c12017-09-22 18:01:19 -0700103 /**
104 * Forks the existing partition into a new partition.
105 *
106 * @param version the version from which to fork the server
107 * @return future to be completed once the fork operation is complete
108 */
109 public CompletableFuture<Void> fork(Version version) {
110 log.info("Forking server for partition {} ({}->{})", partition.getId(), version, partition.getVersion());
Jordan Halterman2bf177c2017-06-29 01:49:08 -0700111 RaftServer.Builder builder = RaftServer.newBuilder(localMemberId)
Jordan Halterman980a8c12017-09-22 18:01:19 -0700112 .withName(partition.getName(version))
113 .withType(RaftMember.Type.PASSIVE)
114 .withProtocol(new RaftServerCommunicator(
115 partition.getName(version),
116 Serializer.using(StorageNamespaces.RAFT_PROTOCOL),
117 clusterCommunicator))
Jordan Halterman19201232017-09-12 17:20:26 -0700118 .withElectionTimeout(Duration.ofMillis(ELECTION_TIMEOUT_MILLIS))
119 .withHeartbeatInterval(Duration.ofMillis(HEARTBEAT_INTERVAL_MILLIS))
Jordan Halterman2bf177c2017-06-29 01:49:08 -0700120 .withStorage(RaftStorage.newBuilder()
Jordan Halterman19201232017-09-12 17:20:26 -0700121 .withStorageLevel(StorageLevel.MAPPED)
Jordan Halterman2bf177c2017-06-29 01:49:08 -0700122 .withSerializer(new AtomixSerializerAdapter(Serializer.using(StorageNamespaces.RAFT_STORAGE)))
Jordan Halterman980a8c12017-09-22 18:01:19 -0700123 .withDirectory(partition.getDataFolder())
124 .withMaxSegmentSize(MAX_SEGMENT_SIZE)
125 .build());
126 StoragePartition.RAFT_SERVICES.forEach(builder::addService);
127 RaftServer server = builder.build();
128 return server.join(partition.getMemberIds(version))
129 .thenCompose(v -> server.shutdown())
130 .thenCompose(v -> {
131 // Delete the cluster configuration file from the forked partition.
132 try {
133 Files.delete(new File(partition.getDataFolder(), "atomix.conf").toPath());
134 } catch (IOException e) {
135 log.error("Failed to delete partition configuration: {}", e);
136 }
137
138 // Build and bootstrap a new server.
139 this.server = buildServer();
140 return this.server.bootstrap();
141 }).whenComplete((r, e) -> {
142 if (e == null) {
143 log.info("Successfully forked server for partition {} ({}->{})",
144 partition.getId(), version, partition.getVersion());
145 } else {
146 log.info("Failed to fork server for partition {} ({}->{})",
147 partition.getId(), version, partition.getVersion(), e);
148 }
149 }).thenApply(v -> null);
150 }
151
152 private RaftServer buildServer() {
153 RaftServer.Builder builder = RaftServer.newBuilder(localMemberId)
154 .withName(partition.getName())
155 .withProtocol(new RaftServerCommunicator(
156 partition.getName(),
157 Serializer.using(StorageNamespaces.RAFT_PROTOCOL),
158 clusterCommunicator))
159 .withElectionTimeout(Duration.ofMillis(ELECTION_TIMEOUT_MILLIS))
160 .withHeartbeatInterval(Duration.ofMillis(HEARTBEAT_INTERVAL_MILLIS))
161 .withStorage(RaftStorage.newBuilder()
162 .withStorageLevel(StorageLevel.MAPPED)
163 .withSerializer(new AtomixSerializerAdapter(Serializer.using(StorageNamespaces.RAFT_STORAGE)))
164 .withDirectory(partition.getDataFolder())
Jordan Halterman035231e2017-07-18 08:39:07 -0700165 .withMaxSegmentSize(MAX_SEGMENT_SIZE)
Jordan Halterman2bf177c2017-06-29 01:49:08 -0700166 .build());
167 StoragePartition.RAFT_SERVICES.forEach(builder::addService);
168 return builder.build();
Madan Jampani15b8ef52016-02-02 17:35:05 -0800169 }
170
Jordan Halterman2bf177c2017-06-29 01:49:08 -0700171 public CompletableFuture<Void> join(Collection<MemberId> otherMembers) {
Jordan Halterman980a8c12017-09-22 18:01:19 -0700172 log.info("Joining partition {} ({})", partition.getId(), partition.getVersion());
Madan Jampani630e7ac2016-05-31 11:34:05 -0700173 server = buildServer();
174 return server.join(otherMembers).whenComplete((r, e) -> {
Madan Jampanif172d402016-03-04 00:56:38 -0800175 if (e == null) {
Jordan Halterman980a8c12017-09-22 18:01:19 -0700176 log.info("Successfully joined partition {} ({})", partition.getId(), partition.getVersion());
Madan Jampanif172d402016-03-04 00:56:38 -0800177 } else {
Jordan Halterman980a8c12017-09-22 18:01:19 -0700178 log.info("Failed to join partition {} ({})", partition.getId(), partition.getVersion(), e);
Madan Jampanif172d402016-03-04 00:56:38 -0800179 }
180 }).thenApply(v -> null);
181 }
182
Madan Jampani15b8ef52016-02-02 17:35:05 -0800183 @Override
184 public boolean isOpen() {
Madan Jampani65f24bb2016-03-15 15:16:18 -0700185 return server.isRunning();
Madan Jampani15b8ef52016-02-02 17:35:05 -0800186 }
Madan Jampanie14a09c2016-02-11 10:43:21 -0800187
188 /**
189 * Returns the partition information.
190 * @return partition info
191 */
192 public PartitionInfo info() {
193 return new StoragePartitionDetails(partition.getId(),
Jordan Halterman2bf177c2017-06-29 01:49:08 -0700194 server.cluster().getMembers(),
195 server.cluster().getMembers(),
196 server.cluster().getLeader(),
197 server.cluster().getTerm()).toPartitionInfo();
Madan Jampanie14a09c2016-02-11 10:43:21 -0800198 }
Madan Jampani15b8ef52016-02-02 17:35:05 -0800199}