blob: e5aa3e83ee1ddf4fb53219ddb3aed9ae6cace19f [file] [log] [blame]
alshabib339a3d92014-09-26 17:54:32 -07001package org.onlab.onos.store.flow.impl;
2
alshabib339a3d92014-09-26 17:54:32 -07003import static org.onlab.onos.net.flow.FlowRuleEvent.Type.RULE_REMOVED;
4import static org.slf4j.LoggerFactory.getLogger;
Yuta HIGUCHIf3d51bd2014-10-21 01:05:33 -07005import static org.onlab.onos.store.flow.impl.FlowStoreMessageSubjects.*;
Yuta HIGUCHI9def0472014-10-23 15:51:10 -07006import static org.onlab.util.Tools.namedThreads;
alshabib339a3d92014-09-26 17:54:32 -07007
Madan Jampani38b250d2014-10-17 11:02:38 -07008import java.io.IOException;
Madan Jampani117aaae2014-10-23 10:04:05 -07009import java.util.ArrayList;
10import java.util.Arrays;
alshabib339a3d92014-09-26 17:54:32 -070011import java.util.Collection;
12import java.util.Collections;
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070013import java.util.Map;
14import java.util.concurrent.ExecutorService;
15import java.util.concurrent.Executors;
Madan Jampani117aaae2014-10-23 10:04:05 -070016import java.util.concurrent.Future;
Madan Jampani38b250d2014-10-17 11:02:38 -070017import java.util.concurrent.TimeUnit;
18import java.util.concurrent.TimeoutException;
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070019import java.util.concurrent.atomic.AtomicInteger;
Madan Jampani117aaae2014-10-23 10:04:05 -070020import java.util.List;
alshabib339a3d92014-09-26 17:54:32 -070021
22import org.apache.felix.scr.annotations.Activate;
23import org.apache.felix.scr.annotations.Component;
24import org.apache.felix.scr.annotations.Deactivate;
Madan Jampani38b250d2014-10-17 11:02:38 -070025import org.apache.felix.scr.annotations.Reference;
26import org.apache.felix.scr.annotations.ReferenceCardinality;
alshabib339a3d92014-09-26 17:54:32 -070027import org.apache.felix.scr.annotations.Service;
28import org.onlab.onos.ApplicationId;
Madan Jampani38b250d2014-10-17 11:02:38 -070029import org.onlab.onos.cluster.ClusterService;
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070030import org.onlab.onos.net.Device;
alshabib339a3d92014-09-26 17:54:32 -070031import org.onlab.onos.net.DeviceId;
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070032import org.onlab.onos.net.device.DeviceService;
Madan Jampani117aaae2014-10-23 10:04:05 -070033import org.onlab.onos.net.flow.CompletedBatchOperation;
alshabib1c319ff2014-10-04 20:29:09 -070034import org.onlab.onos.net.flow.DefaultFlowEntry;
35import org.onlab.onos.net.flow.FlowEntry;
36import org.onlab.onos.net.flow.FlowEntry.FlowEntryState;
alshabib339a3d92014-09-26 17:54:32 -070037import org.onlab.onos.net.flow.FlowRule;
Madan Jampani117aaae2014-10-23 10:04:05 -070038import org.onlab.onos.net.flow.FlowRuleBatchEntry;
39import org.onlab.onos.net.flow.FlowRuleBatchEvent;
40import org.onlab.onos.net.flow.FlowRuleBatchOperation;
41import org.onlab.onos.net.flow.FlowRuleBatchRequest;
alshabib339a3d92014-09-26 17:54:32 -070042import org.onlab.onos.net.flow.FlowRuleEvent;
Madan Jampani117aaae2014-10-23 10:04:05 -070043import org.onlab.onos.net.flow.FlowRuleBatchEntry.FlowRuleOperation;
alshabib339a3d92014-09-26 17:54:32 -070044import org.onlab.onos.net.flow.FlowRuleEvent.Type;
45import org.onlab.onos.net.flow.FlowRuleStore;
46import org.onlab.onos.net.flow.FlowRuleStoreDelegate;
Yuta HIGUCHIf6f50a62014-10-19 15:58:49 -070047import org.onlab.onos.net.flow.StoredFlowEntry;
alshabib339a3d92014-09-26 17:54:32 -070048import org.onlab.onos.store.AbstractStore;
Madan Jampani38b250d2014-10-17 11:02:38 -070049import org.onlab.onos.store.cluster.messaging.ClusterCommunicationService;
50import org.onlab.onos.store.cluster.messaging.ClusterMessage;
Yuta HIGUCHIf3d51bd2014-10-21 01:05:33 -070051import org.onlab.onos.store.cluster.messaging.ClusterMessageHandler;
Madan Jampani38b250d2014-10-17 11:02:38 -070052import org.onlab.onos.store.cluster.messaging.ClusterMessageResponse;
53import org.onlab.onos.store.flow.ReplicaInfo;
Yuta HIGUCHIfe280eb2014-10-17 12:10:43 -070054import org.onlab.onos.store.flow.ReplicaInfoService;
Madan Jampani38b250d2014-10-17 11:02:38 -070055import org.onlab.onos.store.serializers.DistributedStoreSerializers;
56import org.onlab.onos.store.serializers.KryoSerializer;
Yuta HIGUCHI8d143d22014-10-19 23:15:09 -070057import org.onlab.util.KryoNamespace;
alshabib339a3d92014-09-26 17:54:32 -070058import org.slf4j.Logger;
59
60import com.google.common.collect.ArrayListMultimap;
61import com.google.common.collect.ImmutableSet;
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070062import com.google.common.collect.Iterables;
63import com.google.common.collect.Maps;
alshabib339a3d92014-09-26 17:54:32 -070064import com.google.common.collect.Multimap;
Madan Jampani117aaae2014-10-23 10:04:05 -070065import com.google.common.util.concurrent.Futures;
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070066import com.google.common.util.concurrent.ListenableFuture;
67import com.google.common.util.concurrent.SettableFuture;
alshabib339a3d92014-09-26 17:54:32 -070068
69/**
Madan Jampani38b250d2014-10-17 11:02:38 -070070 * Manages inventory of flow rules using a distributed state management protocol.
alshabib339a3d92014-09-26 17:54:32 -070071 */
alshabib339a3d92014-09-26 17:54:32 -070072@Component(immediate = true)
73@Service
74public class DistributedFlowRuleStore
Madan Jampani117aaae2014-10-23 10:04:05 -070075 extends AbstractStore<FlowRuleBatchEvent, FlowRuleStoreDelegate>
alshabib1c319ff2014-10-04 20:29:09 -070076 implements FlowRuleStore {
alshabib339a3d92014-09-26 17:54:32 -070077
78 private final Logger log = getLogger(getClass());
79
80 // store entries as a pile of rules, no info about device tables
Yuta HIGUCHIf6f50a62014-10-19 15:58:49 -070081 private final Multimap<DeviceId, StoredFlowEntry> flowEntries =
82 ArrayListMultimap.<DeviceId, StoredFlowEntry>create();
alshabib339a3d92014-09-26 17:54:32 -070083
alshabib92c65ad2014-10-08 21:56:05 -070084 private final Multimap<Short, FlowRule> flowEntriesById =
85 ArrayListMultimap.<Short, FlowRule>create();
alshabib339a3d92014-09-26 17:54:32 -070086
Madan Jampani38b250d2014-10-17 11:02:38 -070087 @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070088 protected ReplicaInfoService replicaInfoManager;
Madan Jampani38b250d2014-10-17 11:02:38 -070089
90 @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070091 protected ClusterCommunicationService clusterCommunicator;
Madan Jampani38b250d2014-10-17 11:02:38 -070092
93 @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
Yuta HIGUCHI9def0472014-10-23 15:51:10 -070094 protected ClusterService clusterService;
95
96 @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
97 protected DeviceService deviceService;
98
99 private final AtomicInteger localBatchIdGen = new AtomicInteger();
100
101
102 // FIXME switch to expiraing map/Cache?
103 private Map<Integer, SettableFuture<CompletedBatchOperation>> pendingFutures = Maps.newConcurrentMap();
104
105 private final ExecutorService futureListeners =
106 Executors.newCachedThreadPool(namedThreads("flowstore-peer-responders"));
107
Madan Jampani38b250d2014-10-17 11:02:38 -0700108
109 protected static final KryoSerializer SERIALIZER = new KryoSerializer() {
110 @Override
111 protected void setupKryoPool() {
Yuta HIGUCHI8d143d22014-10-19 23:15:09 -0700112 serializerPool = KryoNamespace.newBuilder()
Madan Jampani38b250d2014-10-17 11:02:38 -0700113 .register(DistributedStoreSerializers.COMMON)
114 .build()
115 .populate(1);
116 }
117 };
118
119 // TODO: make this configurable
Yuta HIGUCHIf3d51bd2014-10-21 01:05:33 -0700120 private static final long FLOW_RULE_STORE_TIMEOUT_MILLIS = 5000;
Madan Jampani38b250d2014-10-17 11:02:38 -0700121
alshabib339a3d92014-09-26 17:54:32 -0700122 @Activate
123 public void activate() {
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700124 clusterCommunicator.addSubscriber(APPLY_BATCH_FLOWS, new ClusterMessageHandler() {
Yuta HIGUCHIf3d51bd2014-10-21 01:05:33 -0700125
126 @Override
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700127 public void handle(final ClusterMessage message) {
128 FlowRuleBatchOperation operation = SERIALIZER.decode(message.payload());
129 log.info("received batch request {}", operation);
130 final ListenableFuture<CompletedBatchOperation> f = storeBatchInternal(operation);
Yuta HIGUCHIf3d51bd2014-10-21 01:05:33 -0700131
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700132 f.addListener(new Runnable() {
Yuta HIGUCHIf3d51bd2014-10-21 01:05:33 -0700133
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700134 @Override
135 public void run() {
136 CompletedBatchOperation result = Futures.getUnchecked(f);
137 try {
138 message.respond(SERIALIZER.encode(result));
139 } catch (IOException e) {
140 log.error("Failed to respond back", e);
141 }
142 }
143 }, futureListeners);
Yuta HIGUCHIf3d51bd2014-10-21 01:05:33 -0700144 }
145 });
Madan Jampani117aaae2014-10-23 10:04:05 -0700146
147 clusterCommunicator.addSubscriber(GET_FLOW_ENTRY, new ClusterMessageHandler() {
148
149 @Override
150 public void handle(ClusterMessage message) {
151 FlowRule rule = SERIALIZER.decode(message.payload());
152 log.info("received get flow entry request for {}", rule);
153 FlowEntry flowEntry = getFlowEntryInternal(rule);
154 try {
155 message.respond(SERIALIZER.encode(flowEntry));
156 } catch (IOException e) {
157 log.error("Failed to respond back", e);
158 }
159 }
160 });
161
alshabib339a3d92014-09-26 17:54:32 -0700162 log.info("Started");
163 }
164
165 @Deactivate
166 public void deactivate() {
167 log.info("Stopped");
168 }
169
170
Madan Jampani117aaae2014-10-23 10:04:05 -0700171 // TODO: This is not a efficient operation on a distributed sharded
172 // flow store. We need to revisit the need for this operation or at least
173 // make it device specific.
alshabib339a3d92014-09-26 17:54:32 -0700174 @Override
tom9b4030d2014-10-06 10:39:03 -0700175 public int getFlowRuleCount() {
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700176 // implementing in-efficient operation for debugging purpose.
177 int sum = 0;
178 for (Device device : deviceService.getDevices()) {
179 final DeviceId did = device.id();
180 sum += Iterables.size(getFlowEntries(did));
181 }
182 return sum;
tom9b4030d2014-10-06 10:39:03 -0700183 }
184
185 @Override
alshabib1c319ff2014-10-04 20:29:09 -0700186 public synchronized FlowEntry getFlowEntry(FlowRule rule) {
Madan Jampani117aaae2014-10-23 10:04:05 -0700187 ReplicaInfo replicaInfo = replicaInfoManager.getReplicaInfoFor(rule.deviceId());
188 if (replicaInfo.master().get().equals(clusterService.getLocalNode().id())) {
189 return getFlowEntryInternal(rule);
190 }
191
192 log.info("Forwarding getFlowEntry to {}, which is the primary (master) for device {}",
193 replicaInfo.master().orNull(), rule.deviceId());
194
195 ClusterMessage message = new ClusterMessage(
196 clusterService.getLocalNode().id(),
197 FlowStoreMessageSubjects.GET_FLOW_ENTRY,
198 SERIALIZER.encode(rule));
199
200 try {
201 ClusterMessageResponse response = clusterCommunicator.sendAndReceive(message, replicaInfo.master().get());
202 return SERIALIZER.decode(response.get(FLOW_RULE_STORE_TIMEOUT_MILLIS, TimeUnit.MILLISECONDS));
203 } catch (IOException | TimeoutException e) {
204 // FIXME: throw a FlowStoreException
205 throw new RuntimeException(e);
206 }
Yuta HIGUCHIf6f50a62014-10-19 15:58:49 -0700207 }
208
209 private synchronized StoredFlowEntry getFlowEntryInternal(FlowRule rule) {
210 for (StoredFlowEntry f : flowEntries.get(rule.deviceId())) {
alshabib339a3d92014-09-26 17:54:32 -0700211 if (f.equals(rule)) {
212 return f;
213 }
214 }
215 return null;
216 }
217
218 @Override
alshabib1c319ff2014-10-04 20:29:09 -0700219 public synchronized Iterable<FlowEntry> getFlowEntries(DeviceId deviceId) {
Yuta HIGUCHIf6f50a62014-10-19 15:58:49 -0700220 Collection<? extends FlowEntry> rules = flowEntries.get(deviceId);
alshabib339a3d92014-09-26 17:54:32 -0700221 if (rules == null) {
222 return Collections.emptyList();
223 }
224 return ImmutableSet.copyOf(rules);
225 }
226
227 @Override
alshabib1c319ff2014-10-04 20:29:09 -0700228 public synchronized Iterable<FlowRule> getFlowRulesByAppId(ApplicationId appId) {
alshabib92c65ad2014-10-08 21:56:05 -0700229 Collection<FlowRule> rules = flowEntriesById.get(appId.id());
alshabib339a3d92014-09-26 17:54:32 -0700230 if (rules == null) {
231 return Collections.emptyList();
232 }
233 return ImmutableSet.copyOf(rules);
234 }
235
236 @Override
Madan Jampani117aaae2014-10-23 10:04:05 -0700237 public void storeFlowRule(FlowRule rule) {
238 storeBatch(new FlowRuleBatchOperation(Arrays.asList(new FlowRuleBatchEntry(FlowRuleOperation.ADD, rule))));
239 }
240
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700241 @Override
Madan Jampani117aaae2014-10-23 10:04:05 -0700242 public Future<CompletedBatchOperation> storeBatch(FlowRuleBatchOperation operation) {
243 if (operation.getOperations().isEmpty()) {
244 return Futures.immediateFuture(new CompletedBatchOperation(true, Collections.<FlowEntry>emptySet()));
alshabib339a3d92014-09-26 17:54:32 -0700245 }
Madan Jampani38b250d2014-10-17 11:02:38 -0700246
Madan Jampani117aaae2014-10-23 10:04:05 -0700247 DeviceId deviceId = operation.getOperations().get(0).getTarget().deviceId();
248
249 ReplicaInfo replicaInfo = replicaInfoManager.getReplicaInfoFor(deviceId);
250
251 if (replicaInfo.master().get().equals(clusterService.getLocalNode().id())) {
252 return storeBatchInternal(operation);
253 }
254
255 log.info("Forwarding storeBatch to {}, which is the primary (master) for device {}",
256 replicaInfo.master().orNull(), deviceId);
Yuta HIGUCHIf3d51bd2014-10-21 01:05:33 -0700257
Madan Jampani38b250d2014-10-17 11:02:38 -0700258 ClusterMessage message = new ClusterMessage(
259 clusterService.getLocalNode().id(),
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700260 APPLY_BATCH_FLOWS,
Madan Jampani117aaae2014-10-23 10:04:05 -0700261 SERIALIZER.encode(operation));
Madan Jampani38b250d2014-10-17 11:02:38 -0700262
263 try {
264 ClusterMessageResponse response = clusterCommunicator.sendAndReceive(message, replicaInfo.master().get());
265 response.get(FLOW_RULE_STORE_TIMEOUT_MILLIS, TimeUnit.MILLISECONDS);
266 } catch (IOException | TimeoutException e) {
267 // FIXME: throw a FlowStoreException
268 throw new RuntimeException(e);
269 }
Madan Jampani117aaae2014-10-23 10:04:05 -0700270
271 return null;
Madan Jampani38b250d2014-10-17 11:02:38 -0700272 }
273
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700274 private ListenableFuture<CompletedBatchOperation> storeBatchInternal(FlowRuleBatchOperation operation) {
Madan Jampani117aaae2014-10-23 10:04:05 -0700275 List<FlowEntry> toRemove = new ArrayList<>();
276 List<FlowEntry> toAdd = new ArrayList<>();
277 // TODO: backup changes to hazelcast map
278 for (FlowRuleBatchEntry batchEntry : operation.getOperations()) {
279 FlowRule flowRule = batchEntry.getTarget();
280 FlowRuleOperation op = batchEntry.getOperator();
281 if (op.equals(FlowRuleOperation.REMOVE)) {
282 StoredFlowEntry entry = getFlowEntryInternal(flowRule);
283 if (entry != null) {
284 entry.setState(FlowEntryState.PENDING_REMOVE);
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700285 toRemove.add(entry);
Madan Jampani117aaae2014-10-23 10:04:05 -0700286 }
Madan Jampani117aaae2014-10-23 10:04:05 -0700287 } else if (op.equals(FlowRuleOperation.ADD)) {
288 StoredFlowEntry flowEntry = new DefaultFlowEntry(flowRule);
289 DeviceId deviceId = flowRule.deviceId();
290 if (!flowEntries.containsEntry(deviceId, flowEntry)) {
291 flowEntries.put(deviceId, flowEntry);
292 flowEntriesById.put(flowRule.appId(), flowEntry);
293 toAdd.add(flowEntry);
294 }
295 }
Madan Jampani38b250d2014-10-17 11:02:38 -0700296 }
Madan Jampani117aaae2014-10-23 10:04:05 -0700297 if (toAdd.isEmpty() && toRemove.isEmpty()) {
298 return Futures.immediateFuture(new CompletedBatchOperation(true, Collections.<FlowEntry>emptySet()));
299 }
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700300
301 SettableFuture<CompletedBatchOperation> r = SettableFuture.create();
302 final int batchId = localBatchIdGen.incrementAndGet();
303
304 pendingFutures.put(batchId, r);
305 notifyDelegate(FlowRuleBatchEvent.requested(new FlowRuleBatchRequest(batchId, toAdd, toRemove)));
306 return r;
alshabib339a3d92014-09-26 17:54:32 -0700307 }
308
309 @Override
Madan Jampani117aaae2014-10-23 10:04:05 -0700310 public void deleteFlowRule(FlowRule rule) {
311 storeBatch(new FlowRuleBatchOperation(Arrays.asList(new FlowRuleBatchEntry(FlowRuleOperation.REMOVE, rule))));
alshabib339a3d92014-09-26 17:54:32 -0700312 }
313
314 @Override
Madan Jampani38b250d2014-10-17 11:02:38 -0700315 public FlowRuleEvent addOrUpdateFlowRule(FlowEntry rule) {
316 ReplicaInfo replicaInfo = replicaInfoManager.getReplicaInfoFor(rule.deviceId());
Yuta HIGUCHI0858b352014-10-17 11:56:06 -0700317 if (replicaInfo.master().get().equals(clusterService.getLocalNode().id())) {
Madan Jampani38b250d2014-10-17 11:02:38 -0700318 return addOrUpdateFlowRuleInternal(rule);
319 }
320
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700321 log.error("Tried to update FlowRule {} state,"
322 + " while the Node was not the master.", rule);
323 return null;
Madan Jampani38b250d2014-10-17 11:02:38 -0700324 }
325
326 private synchronized FlowRuleEvent addOrUpdateFlowRuleInternal(FlowEntry rule) {
alshabib339a3d92014-09-26 17:54:32 -0700327 DeviceId did = rule.deviceId();
328
329 // check if this new rule is an update to an existing entry
Yuta HIGUCHIf6f50a62014-10-19 15:58:49 -0700330 StoredFlowEntry stored = getFlowEntryInternal(rule);
alshabib1c319ff2014-10-04 20:29:09 -0700331 if (stored != null) {
332 stored.setBytes(rule.bytes());
333 stored.setLife(rule.life());
334 stored.setPackets(rule.packets());
335 if (stored.state() == FlowEntryState.PENDING_ADD) {
336 stored.setState(FlowEntryState.ADDED);
337 return new FlowRuleEvent(Type.RULE_ADDED, rule);
338 }
alshabib339a3d92014-09-26 17:54:32 -0700339 return new FlowRuleEvent(Type.RULE_UPDATED, rule);
340 }
341
Yuta HIGUCHIf6f50a62014-10-19 15:58:49 -0700342 // TODO: Confirm if this behavior is correct. See SimpleFlowRuleStore
343 flowEntries.put(did, new DefaultFlowEntry(rule));
alshabib1c319ff2014-10-04 20:29:09 -0700344 return null;
Madan Jampani38b250d2014-10-17 11:02:38 -0700345
346 // TODO: also update backup.
alshabib339a3d92014-09-26 17:54:32 -0700347 }
348
349 @Override
Madan Jampani38b250d2014-10-17 11:02:38 -0700350 public FlowRuleEvent removeFlowRule(FlowEntry rule) {
351 ReplicaInfo replicaInfo = replicaInfoManager.getReplicaInfoFor(rule.deviceId());
Yuta HIGUCHI0858b352014-10-17 11:56:06 -0700352 if (replicaInfo.master().get().equals(clusterService.getLocalNode().id())) {
Madan Jampani38b250d2014-10-17 11:02:38 -0700353 // bypass and handle it locally
354 return removeFlowRuleInternal(rule);
355 }
356
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700357 log.error("Tried to remove FlowRule {},"
358 + " while the Node was not the master.", rule);
359 return null;
Madan Jampani38b250d2014-10-17 11:02:38 -0700360 }
361
362 private synchronized FlowRuleEvent removeFlowRuleInternal(FlowEntry rule) {
alshabib1c319ff2014-10-04 20:29:09 -0700363 // This is where one could mark a rule as removed and still keep it in the store.
alshabib339a3d92014-09-26 17:54:32 -0700364 if (flowEntries.remove(rule.deviceId(), rule)) {
365 return new FlowRuleEvent(RULE_REMOVED, rule);
366 } else {
367 return null;
368 }
Madan Jampani38b250d2014-10-17 11:02:38 -0700369 // TODO: also update backup.
alshabib339a3d92014-09-26 17:54:32 -0700370 }
Madan Jampani117aaae2014-10-23 10:04:05 -0700371
372 @Override
373 public void batchOperationComplete(FlowRuleBatchEvent event) {
Yuta HIGUCHI9def0472014-10-23 15:51:10 -0700374 SettableFuture<CompletedBatchOperation> future
375 = pendingFutures.get(event.subject().batchId());
376 if (future != null) {
377 future.set(event.result());
378 }
Madan Jampani117aaae2014-10-23 10:04:05 -0700379 notifyDelegate(event);
380 }
alshabib339a3d92014-09-26 17:54:32 -0700381}