WIP [VOL-2811] - Incorporate preliminary onu-adapter-go code into opencord repo
- reason "discovery-mibsync-complete" reached (via full MibUpload only, received data won't be stored yet)
- first review comments of patchset #4 considered
(please have a look into our inline-comments in Gerrit to know more about the current state)
- no refactoring done yet
Change-Id: Iac47817f8ce4bd28dd8132f530b0570d57ae99b8
Signed-off-by: Holger Hildebrandt <holger.hildebrandt@adtran.com>
diff --git a/adaptercoreont/device_handler.go b/adaptercoreont/device_handler.go
new file mode 100644
index 0000000..c6e81ad
--- /dev/null
+++ b/adaptercoreont/device_handler.go
@@ -0,0 +1,661 @@
+/*
+ * Copyright 2020-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ "context"
+ "encoding/hex"
+ "errors"
+ "fmt"
+ "sync"
+ "time"
+
+ "github.com/gogo/protobuf/proto"
+ "github.com/golang/protobuf/ptypes"
+ "github.com/opencord/voltha-lib-go/v3/pkg/adapters/adapterif"
+ "github.com/opencord/voltha-lib-go/v3/pkg/log"
+ ic "github.com/opencord/voltha-protos/v3/go/inter_container"
+ oop "github.com/opencord/voltha-protos/v3/go/openolt"
+ "github.com/opencord/voltha-protos/v3/go/voltha"
+)
+
+/*
+// Constants for number of retries and for timeout
+const (
+ MaxRetry = 10
+ MaxTimeOutInMs = 500
+)
+*/
+
+//DeviceHandler will interact with the ONU ? device.
+type DeviceHandler struct {
+ deviceID string
+ DeviceType string
+ adminState string
+ device *voltha.Device
+ logicalDeviceID string
+ ProxyAddressID string
+ ProxyAddressType string
+
+ coreProxy adapterif.CoreProxy
+ AdapterProxy adapterif.AdapterProxy
+ EventProxy adapterif.EventProxy
+ openOnuAc *OpenONUAC
+ transitionMap *TransitionMap
+ omciAgent *OpenOMCIAgent
+ ponPort *voltha.Port
+ onuOmciDevice *OnuDeviceEntry
+ exitChannel chan int
+ lockDevice sync.RWMutex
+
+ //Client oop.OpenoltClient
+ //clientCon *grpc.ClientConn
+ //flowMgr *OpenOltFlowMgr
+ //eventMgr *OpenOltEventMgr
+ //resourceMgr *rsrcMgr.OpenOltResourceMgr
+
+ //discOnus sync.Map
+ //onus sync.Map
+ //portStats *OpenOltStatisticsMgr
+ //metrics *pmmetrics.PmMetrics
+ stopCollector chan bool
+ stopHeartbeatCheck chan bool
+ activePorts sync.Map
+}
+
+/*
+//OnuDevice represents ONU related info
+type OnuDevice struct {
+ deviceID string
+ deviceType string
+ serialNumber string
+ onuID uint32
+ intfID uint32
+ proxyDeviceID string
+ uniPorts map[uint32]struct{}
+}
+
+//NewOnuDevice creates a new Onu Device
+func NewOnuDevice(devID, deviceTp, serialNum string, onuID, intfID uint32, proxyDevID string) *OnuDevice {
+ var device OnuDevice
+ device.deviceID = devID
+ device.deviceType = deviceTp
+ device.serialNumber = serialNum
+ device.onuID = onuID
+ device.intfID = intfID
+ device.proxyDeviceID = proxyDevID
+ device.uniPorts = make(map[uint32]struct{})
+ return &device
+}
+*/
+
+//NewDeviceHandler creates a new device handler
+func NewDeviceHandler(cp adapterif.CoreProxy, ap adapterif.AdapterProxy, ep adapterif.EventProxy, device *voltha.Device, adapter *OpenONUAC) *DeviceHandler {
+ var dh DeviceHandler
+ dh.coreProxy = cp
+ dh.AdapterProxy = ap
+ dh.EventProxy = ep
+ cloned := (proto.Clone(device)).(*voltha.Device)
+ dh.deviceID = cloned.Id
+ dh.DeviceType = cloned.Type
+ dh.adminState = "up"
+ dh.device = cloned
+ dh.openOnuAc = adapter
+ dh.transitionMap = nil
+ dh.exitChannel = make(chan int, 1)
+ dh.lockDevice = sync.RWMutex{}
+ dh.stopCollector = make(chan bool, 2)
+ dh.stopHeartbeatCheck = make(chan bool, 2)
+ //dh.metrics = pmmetrics.NewPmMetrics(cloned.Id, pmmetrics.Frequency(150), pmmetrics.FrequencyOverride(false), pmmetrics.Grouped(false), pmmetrics.Metrics(pmNames))
+ dh.activePorts = sync.Map{}
+ //TODO initialize the support classes.
+ return &dh
+}
+
+// start save the device to the data model
+func (dh *DeviceHandler) Start(ctx context.Context) {
+ dh.lockDevice.Lock()
+ defer dh.lockDevice.Unlock()
+ log.Debugw("starting-device-handler", log.Fields{"device": dh.device, "deviceId": dh.deviceID})
+ // Add the initial device to the local model
+ log.Debug("device-handler-started")
+}
+
+// stop stops the device dh. Not much to do for now
+func (dh *DeviceHandler) stop(ctx context.Context) {
+ dh.lockDevice.Lock()
+ defer dh.lockDevice.Unlock()
+ log.Debug("stopping-device-agent")
+ dh.exitChannel <- 1
+ log.Debug("device-agent-stopped")
+}
+
+// ##########################################################################################
+// DeviceHandler methods that implement the adapters interface requests ##### begin #########
+
+//AdoptDevice adopts the OLT device
+func (dh *DeviceHandler) AdoptDevice(ctx context.Context, device *voltha.Device) {
+ log.Debugw("Adopt_device", log.Fields{"deviceID": device.Id, "Address": device.GetHostAndPort()})
+
+ if dh.transitionMap == nil {
+ dh.transitionMap = NewTransitionMap(dh)
+ dh.transitionMap.Handle(ctx, DeviceInit)
+ } else {
+ log.Debug("AdoptDevice: Agent/device init already done")
+ }
+
+ /*
+ // Now, set the initial PM configuration for that device
+ if err := dh.coreProxy.DevicePMConfigUpdate(nil, dh.metrics.ToPmConfigs()); err != nil {
+ log.Errorw("error-updating-PMs", log.Fields{"deviceId": device.Id, "error": err})
+ }
+
+ go startCollector(dh)
+ go startHeartbeatCheck(dh)
+ */
+}
+
+//ProcessInterAdapterMessage sends the proxied messages to the target device
+// If the proxy address is not found in the unmarshalled message, it first fetches the onu device for which the message
+// is meant, and then send the unmarshalled omci message to this onu
+func (dh *DeviceHandler) ProcessInterAdapterMessage(msg *ic.InterAdapterMessage) error {
+ msgID := msg.Header.Id
+ msgType := msg.Header.Type
+ fromTopic := msg.Header.FromTopic
+ toTopic := msg.Header.ToTopic
+ toDeviceID := msg.Header.ToDeviceId
+ proxyDeviceID := msg.Header.ProxyDeviceId
+ log.Debugw("InterAdapter message header", log.Fields{"msgID": msgID, "msgType": msgType,
+ "fromTopic": fromTopic, "toTopic": toTopic, "toDeviceID": toDeviceID, "proxyDeviceID": proxyDeviceID})
+
+ switch msgType {
+ case ic.InterAdapterMessageType_OMCI_REQUEST:
+ {
+ /* TOBECHECKED: I assume, ONU Adapter receives the message hier already 'unmarshalled'? else: (howTo?)*/
+ msgBody := msg.GetBody()
+
+ omciMsg := &ic.InterAdapterOmciMessage{}
+ if err := ptypes.UnmarshalAny(msgBody, omciMsg); err != nil {
+ log.Warnw("cannot-unmarshal-omci-msg-body", log.Fields{"error": err})
+ return err
+ }
+
+ //assuming omci message content is hex coded!
+ // with restricted output of 16(?) bytes would be ...omciMsg.Message[:16]
+ log.Debugw("inter-adapter-recv-omci",
+ log.Fields{"RxOmciMessage": hex.EncodeToString(omciMsg.Message)})
+ //receive_message(omci_msg.message)
+ return dh.onuOmciDevice.PDevOmciCC.ReceiveMessage(context.TODO(), omciMsg.Message)
+ }
+ case ic.InterAdapterMessageType_ONU_IND_REQUEST:
+ {
+ /* TOBECHECKED: I assume, ONU Adapter receives the message hier already 'unmarshalled'? else: see above omci block */
+ msgBody := msg.GetBody()
+
+ onu_indication := &oop.OnuIndication{}
+ if err := ptypes.UnmarshalAny(msgBody, onu_indication); err != nil {
+ log.Warnw("cannot-unmarshal-onu-indication-msg-body", log.Fields{"error": err})
+ return err
+ }
+
+ onu_operstate := onu_indication.GetOperState()
+ log.Debugw("inter-adapter-recv-onu-ind", log.Fields{"OnuId": onu_indication.GetOnuId(),
+ "AdminState": onu_indication.GetAdminState(), "OperState": onu_operstate,
+ "SNR": onu_indication.GetSerialNumber()})
+
+ //interface related functioons might be error checked ....
+ if onu_operstate == "up" {
+ dh.create_interface(onu_indication)
+ } else if (onu_operstate == "down") || (onu_operstate == "unreachable") {
+ dh.update_interface(onu_indication)
+ } else {
+ log.Errorw("unknown-onu-indication operState", log.Fields{"OnuId": onu_indication.GetOnuId()})
+ return errors.New("InvalidOperState")
+ }
+ }
+ default:
+ {
+ log.Errorw("inter-adapter-unhandled-type", log.Fields{"msgType": msg.Header.Type})
+ return errors.New("unimplemented")
+ }
+ }
+
+ /* form py code:
+ elif request.header.type == InterAdapterMessageType.TECH_PROFILE_DOWNLOAD_REQUEST:
+ tech_msg = InterAdapterTechProfileDownloadMessage()
+ request.body.Unpack(tech_msg)
+ self.log.debug('inter-adapter-recv-tech-profile', tech_msg=tech_msg)
+
+ self.load_and_configure_tech_profile(tech_msg.uni_id, tech_msg.path)
+
+ elif request.header.type == InterAdapterMessageType.DELETE_GEM_PORT_REQUEST:
+ del_gem_msg = InterAdapterDeleteGemPortMessage()
+ request.body.Unpack(del_gem_msg)
+ self.log.debug('inter-adapter-recv-del-gem', gem_del_msg=del_gem_msg)
+
+ self.delete_tech_profile(uni_id=del_gem_msg.uni_id,
+ gem_port_id=del_gem_msg.gem_port_id,
+ tp_path=del_gem_msg.tp_path)
+
+ elif request.header.type == InterAdapterMessageType.DELETE_TCONT_REQUEST:
+ del_tcont_msg = InterAdapterDeleteTcontMessage()
+ request.body.Unpack(del_tcont_msg)
+ self.log.debug('inter-adapter-recv-del-tcont', del_tcont_msg=del_tcont_msg)
+
+ self.delete_tech_profile(uni_id=del_tcont_msg.uni_id,
+ alloc_id=del_tcont_msg.alloc_id,
+ tp_path=del_tcont_msg.tp_path)
+ else:
+ self.log.error("inter-adapter-unhandled-type", request=request)
+ */
+ return nil
+}
+
+// DeviceHandler methods that implement the adapters interface requests## end #########
+// #####################################################################################
+
+// ################ to be updated acc. needs of ONU Device ########################
+// DeviceHandler StateMachine related state transition methods ##### begin #########
+
+// doStateInit provides the device update to the core
+func (dh *DeviceHandler) doStateInit(ctx context.Context) error {
+ log.Debug("doStateInit-started")
+ /*
+ var err error
+ dh.clientCon, err = grpc.Dial(dh.device.GetHostAndPort(), grpc.WithInsecure(), grpc.WithBlock())
+ if err != nil {
+ log.Errorw("Failed to dial device", log.Fields{"DeviceId": dh.deviceID, "HostAndPort": dh.device.GetHostAndPort(), "err": err})
+ return err
+ }
+ return nil
+ */
+ // populate what we know. rest comes later after mib sync
+ dh.device.Root = false
+ dh.device.Vendor = "OpenONU"
+ dh.device.Model = "go"
+ dh.device.Reason = "activating-onu"
+ dh.logicalDeviceID = dh.deviceID
+
+ dh.coreProxy.DeviceUpdate(ctx, dh.device)
+
+ // store proxy parameters for later communication - assumption: invariant, else they have to be requested dynamically!!
+ dh.ProxyAddressID = dh.device.ProxyAddress.GetDeviceId()
+ dh.ProxyAddressType = dh.device.ProxyAddress.GetDeviceType()
+ log.Debugw("device-updated", log.Fields{"deviceID": dh.deviceID, "proxyAddressID": dh.ProxyAddressID,
+ "proxyAddressType": dh.ProxyAddressType, "SNR": dh.device.SerialNumber,
+ "ParentId": dh.device.ParentId, "ParentPortNo": dh.device.ParentPortNo})
+
+ /*
+ self._pon = PonPort.create(self, self._pon_port_number)
+ self._pon.add_peer(self.parent_id, self._pon_port_number)
+ self.log.debug('adding-pon-port-to-agent',
+ type=self._pon.get_port().type,
+ admin_state=self._pon.get_port().admin_state,
+ oper_status=self._pon.get_port().oper_status,
+ )
+ */
+ log.Debug("adding-pon-port")
+ ponPortNo := uint32(1)
+ if dh.device.ParentPortNo != 0 {
+ ponPortNo = dh.device.ParentPortNo
+ }
+
+ ponPort := &voltha.Port{
+ PortNo: ponPortNo,
+ Label: fmt.Sprintf("pon-%d", ponPortNo),
+ Type: voltha.Port_PON_ONU,
+ OperStatus: voltha.OperStatus_ACTIVE,
+ Peers: []*voltha.Port_PeerPort{{DeviceId: dh.device.ParentId, // Peer device is OLT
+ PortNo: dh.device.ParentPortNo}}, // Peer port is parent's port number
+ }
+ var err error
+ if err = dh.coreProxy.PortCreated(context.TODO(), dh.deviceID, ponPort); err != nil {
+ log.Fatalf("PortCreated-failed-%s", err)
+ }
+
+ log.Debug("doStateInit-done")
+ return nil
+}
+
+// postInit setups the DeviceEntry for the conerned device
+func (dh *DeviceHandler) postInit(ctx context.Context) error {
+ /*
+ dh.Client = oop.NewOpenoltClient(dh.clientCon)
+ dh.transitionMap.Handle(ctx, GrpcConnected)
+ return nil
+ */
+ //start the Agent object with no specific FSM setting
+ dh.omciAgent = NewOpenOMCIAgent(ctx, dh.coreProxy, dh.AdapterProxy)
+ dh.omciAgent.Start(ctx)
+ // might be updated with some error handling !!!
+ dh.onuOmciDevice, _ = dh.omciAgent.Add_device(ctx, dh.deviceID, dh)
+ //dh.transitionMap.Handle(ctx, GrpcConnected)
+
+ /*
+ ############################################################################
+ # Setup Alarm handler
+ self.events = AdapterEvents(self.core_proxy, device.id, self.logical_device_id,
+ device.serial_number)
+ ############################################################################
+ # Setup PM configuration for this device
+ # Pass in ONU specific options
+ kwargs = {
+ OnuPmMetrics.DEFAULT_FREQUENCY_KEY: OnuPmMetrics.DEFAULT_ONU_COLLECTION_FREQUENCY,
+ 'heartbeat': self.heartbeat,
+ OnuOmciPmMetrics.OMCI_DEV_KEY: self._onu_omci_device
+ }
+ self.log.debug('create-pm-metrics', device_id=device.id, serial_number=device.serial_number)
+ self._pm_metrics = OnuPmMetrics(self.events, self.core_proxy, self.device_id,
+ self.logical_device_id, device.serial_number,
+ grouped=True, freq_override=False, **kwargs)
+ pm_config = self._pm_metrics.make_proto()
+ self._onu_omci_device.set_pm_config(self._pm_metrics.omci_pm.openomci_interval_pm)
+ self.log.info("initial-pm-config", device_id=device.id, serial_number=device.serial_number)
+ yield self.core_proxy.device_pm_config_update(pm_config, init=True)
+
+ # Note, ONU ID and UNI intf set in add_uni_port method
+ self._onu_omci_device.alarm_synchronizer.set_alarm_params(mgr=self.events,
+ ani_ports=[self._pon])
+
+ # Code to Run OMCI Test Action
+ kwargs_omci_test_action = {
+ OmciTestRequest.DEFAULT_FREQUENCY_KEY:
+ OmciTestRequest.DEFAULT_COLLECTION_FREQUENCY
+ }
+ serial_number = device.serial_number
+ self._test_request = OmciTestRequest(self.core_proxy,
+ self.omci_agent, self.device_id,
+ AniG, serial_number,
+ self.logical_device_id,
+ exclusive=False,
+ **kwargs_omci_test_action)
+
+ self.enabled = True
+ else:
+ self.log.info('onu-already-activated')
+ */
+
+ return nil
+}
+
+// doStateUp handle the onu up indication and update to voltha core
+func (dh *DeviceHandler) doStateUp(ctx context.Context) error {
+ /*
+ // Synchronous call to update device state - this method is run in its own go routine
+ if err := dh.coreProxy.DeviceStateUpdate(ctx, dh.device.Id, voltha.ConnectStatus_REACHABLE,
+ voltha.OperStatus_ACTIVE); err != nil {
+ log.Errorw("Failed to update device with OLT UP indication", log.Fields{"deviceID": dh.device.Id, "error": err})
+ return err
+ }
+ return nil
+ */
+ return errors.New("unimplemented")
+}
+
+// doStateDown handle the onu down indication
+func (dh *DeviceHandler) doStateDown(ctx context.Context) error {
+ dh.lockDevice.Lock()
+ defer dh.lockDevice.Unlock()
+ log.Debug("do-state-down-start")
+
+ device, err := dh.coreProxy.GetDevice(ctx, dh.device.Id, dh.device.Id)
+ if err != nil || device == nil {
+ /*TODO: needs to handle error scenarios */
+ log.Errorw("Failed to fetch device device", log.Fields{"err": err})
+ return errors.New("failed to fetch device device")
+ }
+
+ cloned := proto.Clone(device).(*voltha.Device)
+ log.Debugw("do-state-down", log.Fields{"ClonedDeviceID": cloned.Id})
+ /*
+ // Update the all ports state on that device to disable
+ if er := dh.coreProxy.PortsStateUpdate(ctx, cloned.Id, voltha.OperStatus_UNKNOWN); er != nil {
+ log.Errorw("updating-ports-failed", log.Fields{"deviceID": device.Id, "error": er})
+ return er
+ }
+
+ //Update the device oper state and connection status
+ cloned.OperStatus = voltha.OperStatus_UNKNOWN
+ cloned.ConnectStatus = common.ConnectStatus_UNREACHABLE
+ dh.device = cloned
+
+ if er := dh.coreProxy.DeviceStateUpdate(ctx, cloned.Id, cloned.ConnectStatus, cloned.OperStatus); er != nil {
+ log.Errorw("error-updating-device-state", log.Fields{"deviceID": device.Id, "error": er})
+ return er
+ }
+
+ //get the child device for the parent device
+ onuDevices, err := dh.coreProxy.GetChildDevices(ctx, dh.device.Id)
+ if err != nil {
+ log.Errorw("failed to get child devices information", log.Fields{"deviceID": dh.device.Id, "error": err})
+ return err
+ }
+ for _, onuDevice := range onuDevices.Items {
+
+ // Update onu state as down in onu adapter
+ onuInd := oop.OnuIndication{}
+ onuInd.OperState = "down"
+ er := dh.AdapterProxy.SendInterAdapterMessage(ctx, &onuInd, ic.InterAdapterMessageType_ONU_IND_REQUEST,
+ "openolt", onuDevice.Type, onuDevice.Id, onuDevice.ProxyAddress.DeviceId, "")
+ if er != nil {
+ log.Errorw("Failed to send inter-adapter-message", log.Fields{"OnuInd": onuInd,
+ "From Adapter": "openolt", "DevieType": onuDevice.Type, "DeviceID": onuDevice.Id})
+ //Do not return here and continue to process other ONUs
+ }
+ }
+ // * Discovered ONUs entries need to be cleared , since after OLT
+ // is up, it starts sending discovery indications again* /
+ dh.discOnus = sync.Map{}
+ log.Debugw("do-state-down-end", log.Fields{"deviceID": device.Id})
+ return nil
+ */
+ return errors.New("unimplemented")
+}
+
+// doStateConnected get the device info and update to voltha core
+// for comparison of the original method (not that easy to uncomment): compare here:
+// voltha-openolt-adapter/adaptercore/device_handler.go
+// -> this one obviously initiates all communication interfaces of the device ...?
+func (dh *DeviceHandler) doStateConnected(ctx context.Context) error {
+ log.Debug("OLT device has been connected")
+ return errors.New("unimplemented")
+}
+
+// DeviceHandler StateMachine related state transition methods ##### end #########
+// #################################################################################
+
+// ###################################################
+// DeviceHandler utility methods ##### begin #########
+
+// doStateInit provides the device update to the core
+func (dh *DeviceHandler) create_interface(onuind *oop.OnuIndication) error {
+ log.Debug("create_interface-started - not yet fully implemented (only device state update)")
+
+ if err := dh.coreProxy.DeviceStateUpdate(context.TODO(), dh.deviceID, voltha.ConnectStatus_REACHABLE, voltha.OperStatus_ACTIVATING); err != nil {
+ log.Errorw("error-updating-device-state", log.Fields{"deviceID": dh.deviceID, "error": err})
+ }
+
+ device, err := dh.coreProxy.GetDevice(context.TODO(), dh.device.Id, dh.device.Id)
+ if err != nil || device == nil {
+ /*TODO: needs to handle error scenarios */
+ log.Errorw("Failed to fetch device device at creating If", log.Fields{"err": err})
+ return errors.New("Voltha Device not found")
+ }
+
+ dh.onuOmciDevice.Start(context.TODO())
+ if err := dh.coreProxy.DeviceReasonUpdate(context.TODO(), dh.deviceID, "starting-openomci"); err != nil {
+ log.Errorw("error-DeviceReasonUpdate to starting-openomci", log.Fields{"deviceID": dh.deviceID, "error": err})
+ }
+
+ /* this might be a good time for Omci Verify message? */
+ verifyExec := make(chan bool)
+ omci_verify := NewOmciTestRequest(context.TODO(),
+ dh.device.Id, dh.onuOmciDevice.PDevOmciCC,
+ true, true) //eclusive and allowFailure (anyway not yet checked)
+ omci_verify.PerformOmciTest(context.TODO(), verifyExec)
+
+ /* give the handler some time here to wait for the OMCi verification result
+ after Timeout start and try MibUpload FSM anyway
+ (to prevent stopping on just not supported OMCI verification from ONU) */
+ select {
+ case <-time.After(2 * time.Second):
+ log.Warn("omci start-verification timed out (continue normal)")
+ case testresult := <-verifyExec:
+ log.Infow("Omci start verification done", log.Fields{"result": testresult})
+ }
+
+ /* In py code it looks earlier (on activate ..)
+ # Code to Run OMCI Test Action
+ kwargs_omci_test_action = {
+ OmciTestRequest.DEFAULT_FREQUENCY_KEY:
+ OmciTestRequest.DEFAULT_COLLECTION_FREQUENCY
+ }
+ serial_number = device.serial_number
+ self._test_request = OmciTestRequest(self.core_proxy,
+ self.omci_agent, self.device_id,
+ AniG, serial_number,
+ self.logical_device_id,
+ exclusive=False,
+ **kwargs_omci_test_action)
+ ...
+ # Start test requests after a brief pause
+ if not self._test_request_started:
+ self._test_request_started = True
+ tststart = _STARTUP_RETRY_WAIT * (random.randint(1, 5))
+ reactor.callLater(tststart, self._test_request.start_collector)
+
+ */
+ /* which is then: in omci_test_request.py : */
+ /*
+ def start_collector(self, callback=None):
+ """
+ Start the collection loop for an adapter if the frequency > 0
+
+ :param callback: (callable) Function to call to collect PM data
+ """
+ self.log.info("starting-pm-collection", device_name=self.name, default_freq=self.default_freq)
+ if callback is None:
+ callback = self.perform_test_omci
+
+ if self.lc is None:
+ self.lc = LoopingCall(callback)
+
+ if self.default_freq > 0:
+ self.lc.start(interval=self.default_freq / 10)
+
+ def perform_test_omci(self):
+ """
+ Perform the initial test request
+ """
+ ani_g_entities = self._device.configuration.ani_g_entities
+ ani_g_entities_ids = list(ani_g_entities.keys()) if ani_g_entities \
+ is not None else None
+ self._entity_id = ani_g_entities_ids[0]
+ self.log.info('perform-test', entity_class=self._entity_class,
+ entity_id=self._entity_id)
+ try:
+ frame = MEFrame(self._entity_class, self._entity_id, []).test()
+ result = yield self._device.omci_cc.send(frame)
+ if not result.fields['omci_message'].fields['success_code']:
+ self.log.info('Self-Test Submitted Successfully',
+ code=result.fields[
+ 'omci_message'].fields['success_code'])
+ else:
+ raise TestFailure('Test Failure: {}'.format(
+ result.fields['omci_message'].fields['success_code']))
+ except TimeoutError as e:
+ self.deferred.errback(failure.Failure(e))
+
+ except Exception as e:
+ self.log.exception('perform-test-Error', e=e,
+ class_id=self._entity_class,
+ entity_id=self._entity_id)
+ self.deferred.errback(failure.Failure(e))
+
+ */
+
+ // PM related heartbeat??? !!!TODO....
+ //self._heartbeat.enabled = True
+
+ //example how to call FSM - transition up to state "uploading"
+ if dh.onuOmciDevice.MibSyncFsm.Is("disabled") {
+
+ if err := dh.onuOmciDevice.MibSyncFsm.Event("start"); err != nil {
+ log.Errorw("MibSyncFsm: Can't go to state starting", log.Fields{"err": err})
+ return errors.New("Can't go to state starting")
+ } else {
+ log.Debug("MibSyncFsm", log.Fields{"state": string(dh.onuOmciDevice.MibSyncFsm.Current())})
+ //Determine ONU status and start/re-start MIB Synchronization tasks
+ //Determine if this ONU has ever synchronized
+ if true { //TODO: insert valid check
+ if err := dh.onuOmciDevice.MibSyncFsm.Event("load_mib_template"); err != nil {
+ log.Errorw("MibSyncFsm: Can't go to state loading_mib_template", log.Fields{"err": err})
+ return errors.New("Can't go to state loading_mib_template")
+ } else {
+ log.Debug("MibSyncFsm", log.Fields{"state": string(dh.onuOmciDevice.MibSyncFsm.Current())})
+ //Find and load a mib template. If not found proceed with mib_upload
+ // callbacks to be handled:
+ // Event("success")
+ // Event("timeout")
+ //no mib template found
+ if true { //TODO: insert valid check
+ if err := dh.onuOmciDevice.MibSyncFsm.Event("upload_mib"); err != nil {
+ log.Errorw("MibSyncFsm: Can't go to state uploading", log.Fields{"err": err})
+ return errors.New("Can't go to state uploading")
+ } else {
+ log.Debug("state of MibSyncFsm", log.Fields{"state": string(dh.onuOmciDevice.MibSyncFsm.Current())})
+ //Begin full MIB data upload, starting with a MIB RESET
+ // callbacks to be handled:
+ // success: e.Event("success")
+ // failure: e.Event("timeout")
+ }
+ }
+ }
+ } else {
+ dh.onuOmciDevice.MibSyncFsm.Event("examine_mds")
+ log.Debug("state of MibSyncFsm", log.Fields{"state": string(dh.onuOmciDevice.MibSyncFsm.Current())})
+ //Examine the MIB Data Sync
+ // callbacks to be handled:
+ // Event("success")
+ // Event("timeout")
+ // Event("mismatch")
+ }
+ }
+ } else {
+ log.Errorw("wrong state of MibSyncFsm - want: disabled", log.Fields{"have": string(dh.onuOmciDevice.MibSyncFsm.Current())})
+ return errors.New("wrong state of MibSyncFsm")
+ }
+ return nil
+}
+
+func (dh *DeviceHandler) update_interface(onuind *oop.OnuIndication) error {
+ log.Debug("update_interface-started - not yet implemented")
+ return nil
+}
+
+func (dh *DeviceHandler) DeviceStateUpdate(dev_Event OnuDeviceEvent) {
+ if dev_Event == MibDatabaseSync {
+ log.Debug("MibInSync Event: update dev state to 'MibSync complete'")
+ //initiate DevStateUpdate
+ if err := dh.coreProxy.DeviceReasonUpdate(context.TODO(), dh.deviceID, "discovery-mibsync-complete"); err != nil {
+ log.Errorw("error-DeviceReasonUpdate to mibsync-complete", log.Fields{"deviceID": dh.deviceID, "error": err})
+ }
+ } else {
+ log.Warnw("unhandled-device-event", log.Fields{"event": dev_Event})
+ }
+}
diff --git a/adaptercoreont/messageTypes.go b/adaptercoreont/messageTypes.go
new file mode 100644
index 0000000..e2261ed
--- /dev/null
+++ b/adaptercoreont/messageTypes.go
@@ -0,0 +1,59 @@
+/*
+ * Copyright 2018-present Open Networking Foundation
+
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+
+ * http://www.apache.org/licenses/LICENSE-2.0
+
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ gp "github.com/google/gopacket"
+ "github.com/opencord/omci-lib-go"
+)
+
+type MessageType int
+
+const (
+ TestMsg MessageType = 0
+ OMCI MessageType = 1
+)
+
+func (m MessageType) String() string {
+ names := [...]string{
+ "TestMsg",
+ "OMCI",
+ }
+ return names[m]
+}
+
+type Message struct {
+ Type MessageType
+ Data interface{}
+}
+
+const (
+ AnyTriggerForMibSyncUploadMib = 0
+)
+
+//TODO: place holder to have a second interface variant - to be replaced by real variant later on
+type TestMessage struct {
+ TestMessageVal uint32
+}
+
+type OmciMessage struct {
+ //OnuSN *openolt.SerialNumber
+ //OnuID uint32
+ OmciMsg *omci.OMCI
+ OmciPacket *gp.Packet
+}
diff --git a/adaptercoreont/mib_sync.go b/adaptercoreont/mib_sync.go
new file mode 100644
index 0000000..16dc49f
--- /dev/null
+++ b/adaptercoreont/mib_sync.go
@@ -0,0 +1,216 @@
+/*
+ * Copyright 2020-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ "context"
+ "errors"
+
+ "github.com/looplab/fsm"
+
+ //"sync"
+ //"time"
+
+ //"github.com/opencord/voltha-lib-go/v3/pkg/kafka"
+ "github.com/opencord/omci-lib-go"
+ me "github.com/opencord/omci-lib-go/generated"
+ "github.com/opencord/voltha-lib-go/v3/pkg/log"
+ //ic "github.com/opencord/voltha-protos/v3/go/inter_container"
+ //"github.com/opencord/voltha-protos/v3/go/openflow_13"
+ //"github.com/opencord/voltha-protos/v3/go/voltha"
+)
+
+func (onuDeviceEntry *OnuDeviceEntry) logStateChange(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"event name": string(e.Event), "src state": string(e.Src), "dst state": string(e.Dst), "device-id": onuDeviceEntry.deviceID})
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) enterStartingState(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"Start processing MibSync-msgs in State": e.FSM.Current(), "device-id": onuDeviceEntry.deviceID})
+
+ // create channel and start go routine for processing of MibSync messages
+ onuDeviceEntry.MibSyncChan = make(chan Message, 2048)
+ go onuDeviceEntry.ProcessMibSyncMessages()
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) enterLoadingMibTemplateState(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"Start MibTemplate processing in State": e.FSM.Current(), "device-id": onuDeviceEntry.deviceID})
+ log.Debug("function not implemented yet")
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) enterUploadingState(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"send mibReset in State": e.FSM.Current(), "device-id": onuDeviceEntry.deviceID})
+ onuDeviceEntry.PDevOmciCC.sendMibReset(context.TODO(), ConstDefaultOmciTimeout, true)
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) enterInSyncState(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"send notification to core in State": e.FSM.Current(), "device-id": onuDeviceEntry.deviceID})
+ onuDeviceEntry.transferSystemEvent(MibDatabaseSync)
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) enterExaminingMdsState(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"Start GetMds processing in State": e.FSM.Current(), "device-id": onuDeviceEntry.deviceID})
+ log.Debug("function not implemented yet")
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) enterResynchronizingState(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"Start MibResync processing in State": e.FSM.Current(), "device-id": onuDeviceEntry.deviceID})
+ log.Debug("function not implemented yet")
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) enterAuditingState(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"Start MibResync processing in State": e.FSM.Current(), "device-id": onuDeviceEntry.deviceID})
+ log.Debug("function not implemented yet")
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) enterOutOfSyncState(e *fsm.Event) {
+ log.Debugw("MibSync FSM", log.Fields{"Start MibReconcile processing in State": e.FSM.Current(), "device-id": onuDeviceEntry.deviceID})
+ log.Debug("function not implemented yet")
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) ProcessMibSyncMessages( /*ctx context.Context*/ ) {
+ log.Debugw("MibSync Msg", log.Fields{"Start routine to process OMCI-messages for device-id": onuDeviceEntry.deviceID})
+loop:
+ for {
+ select {
+ // case <-ctx.Done():
+ // log.Info("MibSync Msg", log.Fields{"Message handling canceled via context for device-id": onuDeviceEntry.deviceID})
+ // break loop
+ case message, ok := <-onuDeviceEntry.MibSyncChan:
+ if !ok {
+ log.Info("MibSync Msg", log.Fields{"Message couldn't be read from channel for device-id": onuDeviceEntry.deviceID})
+ break loop
+ }
+ log.Debugw("MibSync Msg", log.Fields{"Received message on ONU MibSyncChan for device-id": onuDeviceEntry.deviceID})
+
+ switch message.Type {
+ case TestMsg:
+ msg, _ := message.Data.(TestMessage)
+ onuDeviceEntry.handleTestMsg(msg)
+ case OMCI:
+ msg, _ := message.Data.(OmciMessage)
+ onuDeviceEntry.handleOmciMessage(msg)
+ default:
+ log.Warn("MibSync Msg", log.Fields{"Unknown message type received for device-id": onuDeviceEntry.deviceID, "message.Type": message.Type})
+ }
+ }
+ }
+ log.Info("MibSync Msg", log.Fields{"Stopped handling of MibSyncChan for device-id": onuDeviceEntry.deviceID})
+ // TODO: only this action?
+ onuDeviceEntry.MibSyncFsm.Event("stop")
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) handleTestMsg(msg TestMessage) {
+
+ log.Debugw("MibSync Msg", log.Fields{"TestMessage received for device-id": onuDeviceEntry.deviceID, "msg.TestMessageVal": msg.TestMessageVal})
+
+ switch msg.TestMessageVal {
+ case AnyTriggerForMibSyncUploadMib:
+ onuDeviceEntry.MibSyncFsm.Event("upload_mib")
+ log.Debugw("MibSync Msg", log.Fields{"state": string(onuDeviceEntry.MibSyncFsm.Current())})
+ default:
+ log.Warn("MibSync Msg", log.Fields{"Unknown message type received for device-id": onuDeviceEntry.deviceID, "msg.TestMessageVal": msg.TestMessageVal})
+ }
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) handleOmciMessage(msg OmciMessage) {
+
+ log.Debugw("MibSync Msg", log.Fields{"OmciMessage received for device-id": onuDeviceEntry.deviceID,
+ "msgType": msg.OmciMsg.MessageType})
+
+ //further analysis could be done here based on msg.OmciMsg.Payload, e.g. verification of error code ...
+ switch msg.OmciMsg.MessageType {
+ case omci.MibResetResponseType:
+ msgLayer := (*msg.OmciPacket).Layer(omci.LayerTypeMibResetResponse)
+ if msgLayer == nil {
+ log.Error("Omci Msg layer could not be detected")
+ return
+ }
+ msgObj, msgOk := msgLayer.(*omci.MibResetResponse)
+ if !msgOk {
+ log.Error("Omci Msg layer could not be assigned")
+ return
+ }
+ log.Debugw("MibResetResponse Data", log.Fields{"data-fields": msgObj})
+ if msgObj.Result != me.Success {
+ log.Errorw("Omci MibResetResponse Error - strange - what to do?", log.Fields{"Error": msgObj.Result})
+ return
+ }
+ onuDeviceEntry.PDevOmciCC.sendMibUpload(context.TODO(), ConstDefaultOmciTimeout, true)
+ case omci.MibUploadResponseType:
+ msgLayer := (*msg.OmciPacket).Layer(omci.LayerTypeMibUploadResponse)
+ if msgLayer == nil {
+ log.Error("Omci Msg layer could not be detected")
+ return
+ }
+ msgObj, msgOk := msgLayer.(*omci.MibUploadResponse)
+ if !msgOk {
+ log.Error("Omci Msg layer could not be assigned")
+ return
+ }
+ log.Debugw("MibUploadResponse Data for:", log.Fields{"deviceId": onuDeviceEntry.PDevOmciCC.deviceID, "data-fields": msgObj})
+ /* to be verified / reworked !!! */
+ onuDeviceEntry.PDevOmciCC.uploadNoOfCmds = msgObj.NumberOfCommands
+ if onuDeviceEntry.PDevOmciCC.uploadSequNo < onuDeviceEntry.PDevOmciCC.uploadNoOfCmds {
+ onuDeviceEntry.PDevOmciCC.sendMibUploadNext(context.TODO(), ConstDefaultOmciTimeout, true)
+ } else {
+ log.Error("Invalid number of commands received for:", log.Fields{"deviceId": onuDeviceEntry.PDevOmciCC.deviceID, "uploadNoOfCmds": onuDeviceEntry.PDevOmciCC.uploadNoOfCmds})
+ //TODO right action?
+ onuDeviceEntry.MibSyncFsm.Event("timeout")
+ }
+ case omci.MibUploadNextResponseType:
+ msgLayer := (*msg.OmciPacket).Layer(omci.LayerTypeMibUploadNextResponse)
+ if msgLayer == nil {
+ log.Error("Omci Msg layer could not be detected")
+ return
+ }
+ msgObj, msgOk := msgLayer.(*omci.MibUploadNextResponse)
+ if !msgOk {
+ log.Error("Omci Msg layer could not be assigned")
+ return
+ }
+ log.Debugw("MibUploadNextResponse Data for:", log.Fields{"deviceId": onuDeviceEntry.PDevOmciCC.deviceID, "data-fields": msgObj})
+ // TODO !!! content evaluation ?????
+ if onuDeviceEntry.PDevOmciCC.uploadSequNo < onuDeviceEntry.PDevOmciCC.uploadNoOfCmds {
+ onuDeviceEntry.PDevOmciCC.sendMibUploadNext(context.TODO(), ConstDefaultOmciTimeout, true)
+ } else {
+ //TODO
+ onuDeviceEntry.MibSyncFsm.Event("success")
+ }
+ }
+}
+
+func (onuDeviceEntry *OnuDeviceEntry) MibDbVolatileDict() error {
+ log.Debug("MibVolatileDict- running")
+ return errors.New("not_implemented")
+}
+
+// func (onuDeviceEntry *OnuDeviceEntry) MibTemplateTask() error {
+// return errors.New("not_implemented")
+// }
+// func (onuDeviceEntry *OnuDeviceEntry) MibUploadTask() error {
+// return errors.New("not_implemented")
+// }
+// func (onuDeviceEntry *OnuDeviceEntry) GetMdsTask() error {
+// return errors.New("not_implemented")
+// }
+// func (onuDeviceEntry *OnuDeviceEntry) MibResyncTask() error {
+// return errors.New("not_implemented")
+// }
+// func (onuDeviceEntry *OnuDeviceEntry) MibReconcileTask() error {
+// return errors.New("not_implemented")
+// }
diff --git a/adaptercoreont/omci_agent.go b/adaptercoreont/omci_agent.go
new file mode 100644
index 0000000..ff462f0
--- /dev/null
+++ b/adaptercoreont/omci_agent.go
@@ -0,0 +1,184 @@
+/*
+ * Copyright 2020-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ "context"
+ //"errors"
+ //"sync"
+ //"time"
+
+ "github.com/opencord/voltha-lib-go/v3/pkg/adapters/adapterif"
+
+ //"github.com/opencord/voltha-lib-go/v3/pkg/kafka"
+ "github.com/opencord/voltha-lib-go/v3/pkg/log"
+ //ic "github.com/opencord/voltha-protos/v3/go/inter_container"
+ //"github.com/opencord/voltha-protos/v3/go/openflow_13"
+ //"github.com/opencord/voltha-protos/v3/go/voltha"
+)
+
+/*
+OpenOmciAgentDefaults = {
+ 'mib-synchronizer': {
+ 'state-machine': MibSynchronizer, # Implements the MIB synchronization state machine
+ 'database': MibDbVolatileDict, # Implements volatile ME MIB database
+ # 'database': MibDbExternal, # Implements persistent ME MIB database
+ 'advertise-events': True, # Advertise events on OpenOMCI event bus
+ 'audit-delay': 60, # Time to wait between MIB audits. 0 to disable audits.
+ 'tasks': {
+ 'mib-upload': MibUploadTask,
+ 'mib-template': MibTemplateTask,
+ 'get-mds': GetMdsTask,
+ 'mib-audit': GetMdsTask,
+ 'mib-resync': MibResyncTask,
+ 'mib-reconcile': MibReconcileTask
+ }
+ },
+ 'omci-capabilities': {
+ 'state-machine': OnuOmciCapabilities, # Implements OMCI capabilities state machine
+ 'advertise-events': False, # Advertise events on OpenOMCI event bus
+ 'tasks': {
+ 'get-capabilities': OnuCapabilitiesTask # Get supported ME and Commands
+ }
+ },
+ 'performance-intervals': {
+ 'state-machine': PerformanceIntervals, # Implements PM Intervals State machine
+ 'advertise-events': False, # Advertise events on OpenOMCI event bus
+ 'tasks': {
+ 'sync-time': SyncTimeTask,
+ 'collect-data': IntervalDataTask,
+ 'create-pm': OmciCreatePMRequest,
+ 'delete-pm': OmciDeletePMRequest,
+ },
+ },
+ 'alarm-synchronizer': {
+ 'state-machine': AlarmSynchronizer, # Implements the Alarm sync state machine
+ 'database': AlarmDbExternal, # For any State storage needs
+ 'advertise-events': True, # Advertise events on OpenOMCI event bus
+ 'tasks': {
+ 'alarm-resync': AlarmResyncTask
+ }
+ },
+ 'image_downloader': {
+ 'state-machine': ImageDownloadeSTM,
+ 'advertise-event': True,
+ 'tasks': {
+ 'download-file': FileDownloadTask
+ }
+ },
+ 'image_upgrader': {
+ 'state-machine': OmciSoftwareImageDownloadSTM,
+ 'advertise-event': True,
+ 'tasks': {
+ 'omci_upgrade_task': OmciSwImageUpgradeTask
+ }
+ }
+ # 'image_activator': {
+ # 'state-machine': OmciSoftwareImageActivateSTM,
+ # 'advertise-event': True,
+ # }
+}
+*/
+
+//OpenOMCIAgent structure holds the ONU core information
+type OpenOMCIAgent struct {
+ coreProxy adapterif.CoreProxy
+ adapterProxy adapterif.AdapterProxy
+ started bool
+ deviceEntries map[string]*OnuDeviceEntry
+ mibDbClass func() error
+}
+
+//NewOpenOMCIAgent returns a new instance of OpenOMCIAgent
+func NewOpenOMCIAgent(ctx context.Context,
+ coreProxy adapterif.CoreProxy, adapterProxy adapterif.AdapterProxy) *OpenOMCIAgent {
+ log.Info("init-openOmciAgent")
+ var openomciagent OpenOMCIAgent
+ openomciagent.started = false
+ openomciagent.coreProxy = coreProxy
+ openomciagent.adapterProxy = adapterProxy
+ openomciagent.deviceEntries = make(map[string]*OnuDeviceEntry)
+ return &openomciagent
+}
+
+//Start starts (logs) the omci agent
+func (oo *OpenOMCIAgent) Start(ctx context.Context) error {
+ log.Info("starting-openOmciAgent")
+ //TODO .....
+ //mib_db.start()
+ oo.started = true
+ log.Info("openOmciAgent-started")
+ return nil
+}
+
+//Stop terminates the session
+func (oo *OpenOMCIAgent) Stop(ctx context.Context) error {
+ log.Info("stopping-openOmciAgent")
+ oo.started = false
+ //oo.exitChannel <- 1
+ log.Info("openOmciAgent-stopped")
+ return nil
+}
+
+//
+//Add a new ONU to be managed.
+
+//To provide vendor-specific or custom Managed Entities, create your own Entity
+// ID to class mapping dictionary.
+
+//Since ONU devices can be added at any time (even during Device Handler
+// startup), the ONU device handler is responsible for calling start()/stop()
+// for this object.
+
+//:param device_id: (str) Device ID of ONU to add
+//:param core_proxy: (CoreProxy) Remote API to VOLTHA core
+//:param adapter_proxy: (AdapterProxy) Remote API to other adapters via VOLTHA core
+//:param custom_me_map: (dict) Additional/updated ME to add to class map
+//:param support_classes: (dict) State machines and tasks for this ONU
+
+//:return: (OnuDeviceEntry) The ONU device
+//
+func (oo *OpenOMCIAgent) Add_device(ctx context.Context, device_id string,
+ dh *DeviceHandler) (*OnuDeviceEntry, error) {
+ log.Info("openOmciAgent-adding-deviceEntry")
+
+ deviceEntry := oo.GetDevice(device_id)
+ if deviceEntry == nil {
+ /* costum_me_map in python code seems always to be None,
+ we omit that here first (declaration unclear) -> todo at Adapter specialization ...*/
+ /* also no 'clock' argument - usage open ...*/
+ /* and no alarm_db yet (oo.alarm_db) */
+ deviceEntry = NewOnuDeviceEntry(ctx, device_id, dh, oo.coreProxy, oo.adapterProxy,
+ oo.mibDbClass, nil)
+ oo.deviceEntries[device_id] = deviceEntry
+ log.Infow("openOmciAgent-OnuDeviceEntry-added", log.Fields{"for deviceId": device_id})
+ } else {
+ log.Infow("openOmciAgent-OnuDeviceEntry-add: Device already exists", log.Fields{"for deviceId": device_id})
+ }
+ // might be updated with some error handling !!!
+ return deviceEntry, nil
+}
+
+// Get ONU device entry for a specific Id
+func (oo *OpenOMCIAgent) GetDevice(device_id string) *OnuDeviceEntry {
+ if _, exist := oo.deviceEntries[device_id]; !exist {
+ return nil
+ } else {
+ return oo.deviceEntries[device_id]
+ }
+}
diff --git a/adaptercoreont/omci_cc.go b/adaptercoreont/omci_cc.go
new file mode 100644
index 0000000..17e7e40
--- /dev/null
+++ b/adaptercoreont/omci_cc.go
@@ -0,0 +1,609 @@
+/*
+ * Copyright 2020-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ "container/list"
+ "context"
+ "encoding/binary"
+ "encoding/hex"
+ "errors"
+ "sync"
+
+ //"time"
+
+ "github.com/google/gopacket"
+ // TODO!!! Some references could be resolved auto, but some need specific context ....
+ gp "github.com/google/gopacket"
+
+ "github.com/opencord/omci-lib-go"
+ me "github.com/opencord/omci-lib-go/generated"
+ "github.com/opencord/voltha-lib-go/v3/pkg/adapters/adapterif"
+
+ //"github.com/opencord/voltha-lib-go/v3/pkg/kafka"
+ "github.com/opencord/voltha-lib-go/v3/pkg/log"
+ ic "github.com/opencord/voltha-protos/v3/go/inter_container"
+ //"github.com/opencord/voltha-protos/v3/go/openflow_13"
+ //"github.com/opencord/voltha-protos/v3/go/voltha"
+)
+
+const ConstDefaultOmciTimeout = 10 // ( 3 ?) Seconds
+
+//CallbackPair to be used for ReceiveCallback init
+type CallbackPair struct {
+ cbKey uint16
+ cbFunction func(*omci.OMCI, *gp.Packet) error
+}
+
+type omciTransferStructure struct {
+ txFrame []byte
+ timeout int
+ retry int
+ highPrio bool
+}
+
+//OmciCC structure holds information needed for OMCI communication (to/from OLT Adapter)
+type OmciCC struct {
+ enabled bool
+ pOnuDeviceEntry *OnuDeviceEntry
+ deviceID string
+ pBaseDeviceHandler *DeviceHandler
+ coreProxy adapterif.CoreProxy
+ adapterProxy adapterif.AdapterProxy
+ supportExtMsg bool
+ //txRequest
+ //rxResponse
+ //pendingRequest
+ txFrames, txOnuFrames uint32
+ rxFrames, rxOnuFrames, rxOnuDiscards uint32
+
+ // OMCI params
+ mutexTid sync.Mutex
+ tid uint16
+ mutexHpTid sync.Mutex
+ hpTid uint16
+ uploadSequNo uint16
+ uploadNoOfCmds uint16
+
+ mutexTxQueue sync.Mutex
+ txQueue *list.List
+ mutexRxSchedMap sync.Mutex
+ rxSchedulerMap map[uint16]func(*omci.OMCI, *gp.Packet) error
+}
+
+//NewOmciCC constructor returns a new instance of a OmciCC
+//mib_db (as well as not inluded alarm_db not really used in this code? VERIFY!!)
+func NewOmciCC(ctx context.Context, onu_device_entry *OnuDeviceEntry,
+ device_id string, device_handler *DeviceHandler,
+ core_proxy adapterif.CoreProxy, adapter_proxy adapterif.AdapterProxy) *OmciCC {
+ log.Infow("init-omciCC", log.Fields{"deviceId": device_id})
+ var omciCC OmciCC
+ omciCC.enabled = false
+ omciCC.pOnuDeviceEntry = onu_device_entry
+ omciCC.deviceID = device_id
+ omciCC.pBaseDeviceHandler = device_handler
+ omciCC.coreProxy = core_proxy
+ omciCC.adapterProxy = adapter_proxy
+ omciCC.supportExtMsg = false
+ omciCC.txFrames = 0
+ omciCC.txOnuFrames = 0
+ omciCC.rxFrames = 0
+ omciCC.rxOnuFrames = 0
+ omciCC.rxOnuDiscards = 0
+ omciCC.tid = 0x1
+ omciCC.hpTid = 0x8000
+ omciCC.uploadSequNo = 0
+ omciCC.uploadNoOfCmds = 0
+
+ omciCC.txQueue = list.New()
+ omciCC.rxSchedulerMap = make(map[uint16]func(*omci.OMCI, *gp.Packet) error)
+
+ return &omciCC
+}
+
+// Rx handler for omci messages
+func (oo *OmciCC) ReceiveOnuMessage(ctx context.Context, omciMsg *omci.OMCI) error {
+ log.Debugw("rx-onu-autonomous-message", log.Fields{"omciMsgType": omciMsg.MessageType,
+ "payload": hex.EncodeToString(omciMsg.Payload)})
+ /*
+ msgType = rxFrame.fields["message_type"] //assumed OmciOperationsValue
+ rxOnuFrames++
+
+ switch msgType {
+ case AlarmNotification:
+ {
+ log.Info("Unhandled: received-onu-alarm-message")
+ // python code was:
+ //if msg_type == EntityOperations.AlarmNotification.value:
+ // topic = OMCI_CC.event_bus_topic(self._device_id, RxEvent.Alarm_Notification)
+ // self.reactor.callLater(0, self.event_bus.publish, topic, msg)
+ //
+ return errors.New("RxAlarmNotification unimplemented")
+ }
+ case AttributeValueChange:
+ {
+ log.Info("Unhandled: received-attribute-value-change")
+ // python code was:
+ //elif msg_type == EntityOperations.AttributeValueChange.value:
+ // topic = OMCI_CC.event_bus_topic(self._device_id, RxEvent.AVC_Notification)
+ // self.reactor.callLater(0, self.event_bus.publish, topic, msg)
+ //
+ return errors.New("RxAttributeValueChange unimplemented")
+ }
+ case TestResult:
+ {
+ log.Info("Unhandled: received-test-result")
+ // python code was:
+ //elif msg_type == EntityOperations.TestResult.value:
+ // topic = OMCI_CC.event_bus_topic(self._device_id, RxEvent.Test_Result)
+ // self.reactor.callLater(0, self.event_bus.publish, topic, msg)
+ //
+ return errors.New("RxTestResult unimplemented")
+ }
+ default:
+ {
+ log.Errorw("rx-onu-unsupported-autonomous-message", log.Fields{"msgType": msgType})
+ rxOnuDiscards++
+ return errors.New("RxOnuMsgType unimplemented")
+ }
+ }
+ */
+ return errors.New("ReceiveOnuMessage unimplemented")
+}
+
+// Rx handler for onu messages
+// e.g. would call ReceiveOnuMessage() in case of TID=0 or Action=test ...
+func (oo *OmciCC) ReceiveMessage(ctx context.Context, rxMsg []byte) error {
+ //log.Debugw("cc-receive-omci-message", log.Fields{"RxOmciMessage-x2s": hex.EncodeToString(rxMsg)})
+ if len(rxMsg) >= 44 { // then it should normally include the BaseFormat trailer Len
+ // NOTE: autocorrection only valid for OmciBaseFormat, which is not specifically verified here!!!
+ // (am extendedFormat message could be destroyed this way!)
+ trailerLenData := rxMsg[42:44]
+ trailerLen := binary.BigEndian.Uint16(trailerLenData)
+ log.Infow("omci-received-trailer-len", log.Fields{"Length": trailerLen})
+ if trailerLen != 40 { // invalid base Format entry -> autocorrect
+ binary.BigEndian.PutUint16(rxMsg[42:44], 40)
+ log.Debug("cc-corrected-omci-message: trailer len inserted")
+ }
+ } else {
+ log.Errorw("received omci-message to small for OmciBaseFormat - abort", log.Fields{"Length": len(rxMsg)})
+ return errors.New("RxOmciMessage to small for BaseFormat")
+ }
+
+ packet := gopacket.NewPacket(rxMsg, omci.LayerTypeOMCI, gopacket.NoCopy)
+ if packet == nil {
+ log.Error("omci-message could not be decoded")
+ return errors.New("could not decode rxMsg as OMCI")
+ }
+ omciLayer := packet.Layer(omci.LayerTypeOMCI)
+ if omciLayer == nil {
+ log.Error("omci-message could not decode omci layer")
+ return errors.New("could not decode omci layer")
+ }
+ omciMsg, ok := omciLayer.(*omci.OMCI)
+ if !ok {
+ log.Error("omci-message could not assign omci layer")
+ return errors.New("could not assign omci layer")
+ }
+ log.Debugw("omci-message-decoded:", log.Fields{"omciMsgType": omciMsg.MessageType,
+ "transCorrId": omciMsg.TransactionID, "DeviceIdent": omciMsg.DeviceIdentifier})
+ if byte(omciMsg.MessageType) & ^me.AK == 0 {
+ // Not a response
+ log.Debug("RxMsg is no Omci Response Message")
+ if omciMsg.TransactionID == 0 {
+ return oo.ReceiveOnuMessage(ctx, omciMsg)
+ } else {
+ log.Errorw("Unexpected TransCorrId != 0 not accepted for autonomous messages",
+ log.Fields{"msgType": omciMsg.MessageType, "payload": hex.EncodeToString(omciMsg.Payload)})
+ return errors.New("Autonomous Omci Message with TranSCorrId != 0 not acccepted")
+ }
+ } else {
+ log.Debug("RxMsg is a Omci Response Message: try to schedule it to the requester")
+ oo.mutexRxSchedMap.Lock()
+ rxCallback, ok := oo.rxSchedulerMap[omciMsg.TransactionID]
+ if ok && rxCallback != nil {
+ //disadvantage of decoupling: error verification made difficult, but anyway the question is
+ // how to react on erroneous frame reception, maybe can simply be ignored
+ go rxCallback(omciMsg, &packet)
+ // having posted the response the request is regarded as 'done'
+ delete(oo.rxSchedulerMap, omciMsg.TransactionID)
+ oo.mutexRxSchedMap.Unlock()
+ } else {
+ oo.mutexRxSchedMap.Unlock()
+ log.Error("omci-message-response for not registered transCorrId")
+ return errors.New("could not find registered response handler tor transCorrId")
+ }
+ }
+
+ return nil
+ /* py code was:
+ Receive and OMCI message from the proxy channel to the OLT.
+
+ Call this from your ONU Adapter on a new OMCI Rx on the proxy channel
+ :param msg: (str) OMCI binary message (used as input to Scapy packet decoder)
+ """
+ if not self.enabled:
+ return
+
+ try:
+ now = arrow.utcnow()
+ d = None
+
+ # NOTE: Since we may need to do an independent ME map on a per-ONU basis
+ # save the current value of the entity_id_to_class_map, then
+ # replace it with our custom one before decode, and then finally
+ # restore it later. Tried other ways but really made the code messy.
+ saved_me_map = omci_entities.entity_id_to_class_map
+ omci_entities.entity_id_to_class_map = self._me_map
+
+ try:
+ rx_frame = msg if isinstance(msg, OmciFrame) else OmciFrame(msg)
+ self.log.debug('recv-omci-msg', omci_msg=hexlify(msg))
+ except KeyError as e:
+ # Unknown, Unsupported, or vendor-specific ME. Key is the unknown classID
+ self.log.debug('frame-decode-key-error', omci_msg=hexlify(msg), e=e)
+ rx_frame = self._decode_unknown_me(msg)
+ self._rx_unknown_me += 1
+
+ except Exception as e:
+ self.log.exception('frame-decode', omci_msg=hexlify(msg), e=e)
+ return
+
+ finally:
+ omci_entities.entity_id_to_class_map = saved_me_map # Always restore it.
+
+ rx_tid = rx_frame.fields['transaction_id']
+ msg_type = rx_frame.fields['message_type']
+ self.log.debug('Received message for rx_tid', rx_tid = rx_tid, msg_type = msg_type)
+ # Filter the Test Result frame and route through receive onu
+ # message method.
+ if rx_tid == 0 or msg_type == EntityOperations.TestResult.value:
+ self.log.debug('Receive ONU message', rx_tid=0)
+ return self._receive_onu_message(rx_frame)
+
+ # Previously unreachable if this is the very first round-trip Rx or we
+ # have been running consecutive errors
+ if self._rx_frames == 0 or self._consecutive_errors != 0:
+ self.log.debug('Consecutive errors for rx', err = self._consecutive_errors)
+ self.reactor.callLater(0, self._publish_connectivity_event, True)
+
+ self._rx_frames += 1
+ self._consecutive_errors = 0
+
+ try:
+ high_priority = self._tid_is_high_priority(rx_tid)
+ index = self._get_priority_index(high_priority)
+
+ # (timestamp, defer, frame, timeout, retry, delayedCall)
+ last_tx_tuple = self._tx_request[index]
+
+ if last_tx_tuple is None or \
+ last_tx_tuple[OMCI_CC.REQUEST_FRAME].fields.get('transaction_id') != rx_tid:
+ # Possible late Rx on a message that timed-out
+ if last_tx_tuple:
+ self.log.debug('Unknown message', rx_tid=rx_tid,
+ tx_id=last_tx_tuple[OMCI_CC.REQUEST_FRAME].fields.get('transaction_id'))
+ self._rx_unknown_tid += 1
+ self._rx_late += 1
+ return
+
+ ts, d, tx_frame, timeout, retry, dc = last_tx_tuple
+ if dc is not None and not dc.cancelled and not dc.called:
+ dc.cancel()
+
+ _secs = self._update_rx_tx_stats(now, ts)
+
+ # Late arrival already serviced by a timeout?
+ if d.called:
+ self._rx_late += 1
+ self.log.debug('Serviced by timeout. Late arrival', rx_late = self._rx_late)
+ return
+
+ except Exception as e:
+ self.log.exception('frame-match', msg=hexlify(msg), e=e)
+ if d is not None:
+ return d.errback(failure.Failure(e))
+ return
+
+ # Publish Rx event to listeners in a different task
+ self.log.debug('Publish rx event', rx_tid = rx_tid,
+ tx_tid = tx_frame.fields['transaction_id'])
+ reactor.callLater(0, self._publish_rx_frame, tx_frame, rx_frame)
+
+ # begin success callback chain (will cancel timeout and queue next Tx message)
+ self._rx_response[index] = rx_frame
+ d.callback(rx_frame)
+
+ except Exception as e:
+ self.log.exception('rx-msg', e=e)
+ */
+}
+
+func (oo *OmciCC) PublishRxResponseFrame(ctx context.Context, txFrame []byte, rxFrame []byte) error {
+ return errors.New("PublishRxResponseFrame unimplemented")
+ /*
+ def _publish_rx_frame(self, tx_frame, rx_frame):
+ */
+}
+
+//Queue the OMCI Frame for a transmit to the ONU via the proxy_channel
+func (oo *OmciCC) Send(ctx context.Context, txFrame []byte, timeout int, retry int, highPrio bool,
+ receiveCallbackPair CallbackPair) error {
+
+ log.Debugw("register-response-callback:", log.Fields{"for TansCorrId": receiveCallbackPair.cbKey})
+ // it could be checked, if the callback keay is already registered - but simply overwrite may be acceptable ...
+ oo.mutexRxSchedMap.Lock()
+ oo.rxSchedulerMap[receiveCallbackPair.cbKey] = receiveCallbackPair.cbFunction
+ oo.mutexRxSchedMap.Unlock()
+
+ //just use a simple list for starting - might need some more effort, especially for multi source write access
+ omciTxRequest := omciTransferStructure{
+ txFrame,
+ timeout,
+ retry,
+ highPrio,
+ }
+ oo.mutexTxQueue.Lock()
+ oo.txQueue.PushBack(omciTxRequest) // enqueue
+ oo.mutexTxQueue.Unlock()
+
+ // for first test just bypass and send directly:
+ go oo.sendNextRequest(ctx)
+ return nil
+}
+
+//Pull next tx request and send it
+func (oo *OmciCC) sendNextRequest(ctx context.Context) error {
+ // return errors.New("sendNextRequest unimplemented")
+
+ // just try to get something transferred !!
+ // avoid accessing the txQueue from parallel send requests
+ // block parallel omci send requests at least until SendIAP is 'committed'
+ // that should be feasible for an onu instance as on OMCI anyway window size 1 is assumed
+ oo.mutexTxQueue.Lock()
+ for oo.txQueue.Len() > 0 {
+ queueElement := oo.txQueue.Front() // First element
+ omciTxRequest := queueElement.Value.(omciTransferStructure)
+ /* compare olt device handler code:
+ func (dh *DeviceHandler) omciIndication(omciInd *oop.OmciIndication) {
+ log.Debugw("omci indication", log.Fields{"intfID": omciInd.IntfId, "onuID": omciInd.OnuId})
+ var deviceType string
+ var deviceID string
+ var proxyDeviceID string
+
+ onuKey := dh.formOnuKey(omciInd.IntfId, omciInd.OnuId)
+
+ if onuInCache, ok := dh.onus.Load(onuKey); !ok {
+
+ log.Debugw("omci indication for a device not in cache.", log.Fields{"intfID": omciInd.IntfId, "onuID": omciInd.OnuId})
+ ponPort := IntfIDToPortNo(omciInd.GetIntfId(), voltha.Port_PON_OLT)
+ kwargs := make(map[string]interface{})
+ kwargs["onu_id"] = omciInd.OnuId
+ kwargs["parent_port_no"] = ponPort
+
+ onuDevice, err := dh.coreProxy.GetChildDevice(context.TODO(), dh.device.Id, kwargs)
+ if err != nil {
+ log.Errorw("onu not found", log.Fields{"intfID": omciInd.IntfId, "onuID": omciInd.OnuId, "error": err})
+ return
+ }
+ deviceType = onuDevice.Type
+ deviceID = onuDevice.Id
+ proxyDeviceID = onuDevice.ProxyAddress.DeviceId
+ //if not exist in cache, then add to cache.
+ dh.onus.Store(onuKey, NewOnuDevice(deviceID, deviceType, onuDevice.SerialNumber, omciInd.OnuId, omciInd.IntfId, proxyDeviceID))
+ } else {
+ //found in cache
+ log.Debugw("omci indication for a device in cache.", log.Fields{"intfID": omciInd.IntfId, "onuID": omciInd.OnuId})
+ deviceType = onuInCache.(*OnuDevice).deviceType
+ deviceID = onuInCache.(*OnuDevice).deviceID
+ proxyDeviceID = onuInCache.(*OnuDevice).proxyDeviceID
+ }
+ */
+ /* and compare onu_adapter py code:
+ omci_msg = InterAdapterOmciMessage(
+ message=bytes(frame),
+ proxy_address=self._proxy_address,
+ connect_status=self._device.connect_status)
+
+ self.log.debug('sent-omci-msg', tid=tx_tid, omci_msg=hexlify(bytes(frame)))
+
+ yield self._adapter_proxy.send_inter_adapter_message(
+ msg=omci_msg,
+ type=InterAdapterMessageType.OMCI_REQUEST,
+ from_adapter=self._device.type,
+ to_adapter=self._proxy_address.device_type,
+ to_device_id=self._device_id,
+ proxy_device_id=self._proxy_address.device_id
+ )
+ */
+ device, err := oo.coreProxy.GetDevice(ctx,
+ oo.pBaseDeviceHandler.deviceID, oo.deviceID) //parent, child
+ if err != nil || device == nil {
+ /*TODO: needs to handle error scenarios */
+ log.Errorw("Failed to fetch device", log.Fields{"err": err, "ParentId": oo.pBaseDeviceHandler.deviceID,
+ "ChildId": oo.deviceID})
+ return errors.New("failed to fetch device")
+ }
+
+ log.Debugw("omci-message-sending", log.Fields{"fromDeviceType": oo.pBaseDeviceHandler.DeviceType,
+ "toDeviceType": oo.pBaseDeviceHandler.ProxyAddressType,
+ "onuDeviceID": oo.deviceID, "proxyDeviceID": oo.pBaseDeviceHandler.ProxyAddressID})
+ log.Debugw("omci-message-to-send:",
+ log.Fields{"TxOmciMessage": hex.EncodeToString(omciTxRequest.txFrame)})
+
+ omciMsg := &ic.InterAdapterOmciMessage{Message: omciTxRequest.txFrame}
+ if sendErr := oo.adapterProxy.SendInterAdapterMessage(context.Background(), omciMsg,
+ ic.InterAdapterMessageType_OMCI_REQUEST,
+ //fromType,toType,toDevId, ProxyDevId
+ oo.pBaseDeviceHandler.DeviceType, oo.pBaseDeviceHandler.ProxyAddressType,
+ oo.deviceID, oo.pBaseDeviceHandler.ProxyAddressID, ""); sendErr != nil {
+ log.Errorw("send omci request error", log.Fields{"error": sendErr})
+ return sendErr
+ }
+ oo.txQueue.Remove(queueElement) // Dequeue
+ }
+ oo.mutexTxQueue.Unlock()
+ return nil
+}
+
+func (oo *OmciCC) GetNextTid(highPriority bool) uint16 {
+ var next uint16
+ if highPriority {
+ oo.mutexTid.Lock()
+ next = oo.hpTid
+ oo.hpTid += 1
+ if oo.hpTid < 0x8000 {
+ oo.hpTid = 0x8000
+ }
+ oo.mutexTid.Unlock()
+ } else {
+ oo.mutexHpTid.Lock()
+ next = oo.tid
+ oo.tid += 1
+ if oo.tid >= 0x8000 {
+ oo.tid = 1
+ }
+ oo.mutexHpTid.Unlock()
+ }
+ return next
+}
+
+// ###################################################################################
+// # utility methods provided to work on OMCI messages
+func serialize(msgType omci.MessageType, request gopacket.SerializableLayer, tid uint16) ([]byte, error) {
+ omciLayer := &omci.OMCI{
+ TransactionID: tid,
+ MessageType: msgType,
+ }
+ var options gopacket.SerializeOptions
+ options.FixLengths = true
+
+ buffer := gopacket.NewSerializeBuffer()
+ err := gopacket.SerializeLayers(buffer, options, omciLayer, request)
+ if err != nil {
+ return nil, err
+ }
+ return buffer.Bytes(), nil
+}
+
+func hexEncode(omciPkt []byte) ([]byte, error) {
+ dst := make([]byte, hex.EncodedLen(len(omciPkt)))
+ hex.Encode(dst, omciPkt)
+ return dst, nil
+}
+
+// ###################################################################################
+// # MIB Action shortcuts - still dummies - TODO!!!!!
+
+//supply a response handler for the MibSync omci response messages
+func (oo *OmciCC) receiveMibSyncResponse(omciMsg *omci.OMCI, packet *gp.Packet) error {
+
+ log.Debugw("mib-sync-omci-message-response received:", log.Fields{"omciMsgType": omciMsg.MessageType,
+ "transCorrId": omciMsg.TransactionID, "deviceId": oo.deviceID})
+
+ if oo.pOnuDeviceEntry == nil {
+ log.Error("Abort Receive MibSync OMCI, DeviceEntryPointer is nil")
+ return errors.New("DeviceEntryPointer is nil")
+ }
+
+ // no further test on SeqNo is done here, assignment from rxScheduler is trusted
+ // MibSync responses are simply transferred via deviceEntry to MibSync, no specific analysis here
+ mibSyncMsg := Message{
+ Type: OMCI,
+ Data: OmciMessage{
+ OmciMsg: omciMsg,
+ OmciPacket: packet,
+ },
+ }
+ //log.Debugw("Message to be sent into channel:", log.Fields{"mibSyncMsg": mibSyncMsg})
+ (*oo.pOnuDeviceEntry).MibSyncChan <- mibSyncMsg
+
+ return nil
+}
+
+func (oo *OmciCC) sendMibReset(ctx context.Context, timeout int, highPrio bool) error {
+
+ log.Debugw("send MibReset-msg to:", log.Fields{"deviceId": oo.deviceID})
+ request := &omci.MibResetRequest{
+ MeBasePacket: omci.MeBasePacket{
+ EntityClass: me.OnuDataClassID,
+ },
+ }
+ tid := oo.GetNextTid(highPrio)
+ pkt, err := serialize(omci.MibResetRequestType, request, tid)
+ if err != nil {
+ log.Errorw("Cannot serialize MibResetRequest", log.Fields{"Err": err})
+ return err
+ }
+ omciRxCallbackPair := CallbackPair{tid, oo.receiveMibSyncResponse}
+ return oo.Send(ctx, pkt, timeout, 0, highPrio, omciRxCallbackPair)
+}
+
+func (oo *OmciCC) sendMibUpload(ctx context.Context, timeout int, highPrio bool) error {
+
+ log.Debugw("send MibUpload-msg to:", log.Fields{"deviceId": oo.deviceID})
+ request := &omci.MibUploadRequest{
+ MeBasePacket: omci.MeBasePacket{
+ EntityClass: me.OnuDataClassID,
+ },
+ }
+ tid := oo.GetNextTid(highPrio)
+ pkt, err := serialize(omci.MibUploadRequestType, request, tid)
+ if err != nil {
+ log.Errorw("Cannot serialize MibUploadRequest", log.Fields{"Err": err})
+ return err
+ }
+ oo.uploadSequNo = 0
+ oo.uploadNoOfCmds = 0
+
+ omciRxCallbackPair := CallbackPair{tid, oo.receiveMibSyncResponse}
+ return oo.Send(ctx, pkt, timeout, 0, highPrio, omciRxCallbackPair)
+}
+
+func (oo *OmciCC) sendMibUploadNext(ctx context.Context, timeout int, highPrio bool) error {
+
+ log.Debugw("send MibUploadNext-msg to:", log.Fields{"deviceId": oo.deviceID, "uploadSequNo": oo.uploadSequNo})
+ request := &omci.MibUploadNextRequest{
+ MeBasePacket: omci.MeBasePacket{
+ EntityClass: me.OnuDataClassID,
+ },
+ CommandSequenceNumber: oo.uploadSequNo,
+ }
+ tid := oo.GetNextTid(highPrio)
+ pkt, err := serialize(omci.MibUploadNextRequestType, request, tid)
+ if err != nil {
+ log.Errorw("Cannot serialize MibUploadNextRequest", log.Fields{"Err": err})
+ return err
+ }
+ oo.uploadSequNo++
+
+ omciRxCallbackPair := CallbackPair{tid, oo.receiveMibSyncResponse}
+ return oo.Send(ctx, pkt, timeout, 0, highPrio, omciRxCallbackPair)
+}
+
+/* py code example
+...
+def send_mib_upload(self, timeout=DEFAULT_OMCI_TIMEOUT, high_priority=False):
+ frame = OntDataFrame().mib_upload()
+ return self.send(frame, timeout=timeout, high_priority=high_priority)
+
+def send_mib_upload_next(self, seq_no, timeout=DEFAULT_OMCI_TIMEOUT, high_priority=False):
+ frame = OntDataFrame(sequence_number=seq_no).mib_upload_next()
+ return self.send(frame, timeout=timeout, high_priority=high_priority)
+...
+*/
diff --git a/adaptercoreont/omci_test_request.go b/adaptercoreont/omci_test_request.go
new file mode 100644
index 0000000..a4c6eda
--- /dev/null
+++ b/adaptercoreont/omci_test_request.go
@@ -0,0 +1,141 @@
+/*
+ * Copyright 2020-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ "context"
+ "errors"
+
+ //"sync"
+ //"time"
+
+ gp "github.com/google/gopacket"
+ "github.com/opencord/omci-lib-go"
+ me "github.com/opencord/omci-lib-go/generated"
+
+ //"github.com/opencord/voltha-lib-go/v3/pkg/kafka"
+ "github.com/opencord/voltha-lib-go/v3/pkg/log"
+ //ic "github.com/opencord/voltha-protos/v3/go/inter_container"
+ //"github.com/opencord/voltha-protos/v3/go/openflow_13"
+ //"github.com/opencord/voltha-protos/v3/go/voltha"
+)
+
+//OmciTestRequest structure holds the information for the OMCI test
+type OmciTestRequest struct {
+ deviceID string
+ pDevOmciCC *OmciCC
+ started bool
+ result bool
+ exclusive_cc bool
+ allowFailure bool
+ txSeqNo uint16
+ verifyDone chan<- bool
+}
+
+//NewOmciTestRequest returns a new instance of OmciTestRequest
+func NewOmciTestRequest(ctx context.Context,
+ device_id string, omci_cc *OmciCC,
+ exclusive bool, allow_failure bool) *OmciTestRequest {
+ log.Debug("omciTestRequest-init")
+ var omciTestRequest OmciTestRequest
+ omciTestRequest.deviceID = device_id
+ omciTestRequest.pDevOmciCC = omci_cc
+ omciTestRequest.started = false
+ omciTestRequest.result = false
+ omciTestRequest.exclusive_cc = exclusive
+ omciTestRequest.allowFailure = allow_failure
+
+ return &omciTestRequest
+}
+
+//
+func (oo *OmciTestRequest) PerformOmciTest(ctx context.Context, exec_Channel chan<- bool) {
+ log.Debug("omciTestRequest-start-test")
+
+ if oo.pDevOmciCC != nil {
+ oo.verifyDone = exec_Channel
+ // test functionality is limited to ONU-2G get request for the moment
+ // without yet checking the received response automatically here (might be improved ??)
+ tid := oo.pDevOmciCC.GetNextTid(false)
+ onu2gBaseGet, _ := oo.CreateOnu2gBaseGet(tid)
+ omciRxCallbackPair := CallbackPair{tid, oo.ReceiveOmciVerifyResponse}
+
+ log.Debugw("performOmciTest-start sending frame", log.Fields{"for deviceId": oo.deviceID})
+ // send with default timeout and normal prio
+ go oo.pDevOmciCC.Send(ctx, onu2gBaseGet, ConstDefaultOmciTimeout, 0, false, omciRxCallbackPair)
+
+ } else {
+ log.Errorw("performOmciTest: Device does not exist", log.Fields{"for deviceId": oo.deviceID})
+ }
+}
+
+// these are OMCI related functions, could/should be collected in a separate file? TODO!!!
+// for a simple start just included in here
+//basic approach copied from bbsim, cmp /devices/onu.go and /internal/common/omci/mibpackets.go
+func (oo *OmciTestRequest) CreateOnu2gBaseGet(tid uint16) ([]byte, error) {
+
+ request := &omci.GetRequest{
+ MeBasePacket: omci.MeBasePacket{
+ EntityClass: me.Onu2GClassID,
+ EntityInstance: 0, //there is only the 0 instance of ONU2-G (still hard-coded - TODO!!!)
+ },
+ AttributeMask: 0xE000, //example hardcoded (TODO!!!) request EquId, OmccVersion, VendorCode
+ }
+
+ oo.txSeqNo = tid
+ pkt, err := serialize(omci.GetRequestType, request, tid)
+ if err != nil {
+ //omciLogger.WithFields(log.Fields{ ...
+ log.Errorw("Cannot serialize Onu2-G GetRequest", log.Fields{"Err": err})
+ return nil, err
+ }
+ // hexEncode would probably work as well, but not needed and leads to wrong logs on OltAdapter frame
+ // return hexEncode(pkt)
+ return pkt, nil
+}
+
+//supply a response handler
+func (oo *OmciTestRequest) ReceiveOmciVerifyResponse(omciMsg *omci.OMCI, packet *gp.Packet) error {
+
+ log.Debugw("verify-omci-message-response received:", log.Fields{"omciMsgType": omciMsg.MessageType,
+ "transCorrId": omciMsg.TransactionID, "DeviceIdent": omciMsg.DeviceIdentifier})
+
+ if omciMsg.TransactionID == oo.txSeqNo {
+ log.Debugw("verify-omci-message-response", log.Fields{"correct TransCorrId": omciMsg.TransactionID})
+ } else {
+ log.Debugw("verify-omci-message-response error", log.Fields{"incorrect TransCorrId": omciMsg.TransactionID,
+ "expected": oo.txSeqNo})
+ oo.verifyDone <- false
+ return errors.New("Unexpected TransCorrId")
+ }
+ if omciMsg.MessageType == omci.GetResponseType {
+ log.Debugw("verify-omci-message-response", log.Fields{"correct RespType": omciMsg.MessageType})
+ } else {
+ log.Debugw("verify-omci-message-response error", log.Fields{"incorrect RespType": omciMsg.MessageType,
+ "expected": omci.GetResponseType})
+ oo.verifyDone <- false
+ return errors.New("Unexpected MessageType")
+ }
+
+ //TODO!!! further tests on the payload should be done here ...
+
+ oo.result = true
+ oo.verifyDone <- true
+
+ return nil
+}
diff --git a/adaptercoreont/ont_device_entry.go b/adaptercoreont/ont_device_entry.go
new file mode 100644
index 0000000..27b71e2
--- /dev/null
+++ b/adaptercoreont/ont_device_entry.go
@@ -0,0 +1,209 @@
+/*
+ * Copyright 2020-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ "context"
+ //"errors"
+ //"sync"
+ //"time"
+
+ "github.com/looplab/fsm"
+ "github.com/opencord/voltha-lib-go/v3/pkg/adapters/adapterif"
+
+ //"github.com/opencord/voltha-lib-go/v3/pkg/kafka"
+ "github.com/opencord/voltha-lib-go/v3/pkg/log"
+ //ic "github.com/opencord/voltha-protos/v3/go/inter_container"
+ //"github.com/opencord/voltha-protos/v3/go/openflow_13"
+ //"github.com/opencord/voltha-protos/v3/go/voltha"
+)
+
+type OnuDeviceEvent int
+
+const (
+ // Events of interest to Device Adapters and OpenOMCI State Machines
+ DeviceStatusInit OnuDeviceEvent = 0 // OnuDeviceEntry default start state
+ MibDatabaseSync OnuDeviceEvent = 1 // MIB database sync (upload done)
+ OmciCapabilitiesDone OnuDeviceEvent = 2 // OMCI ME and message type capabilities known
+ PortLinkUp OnuDeviceEvent = 3 // Port link state change
+ PortLinkDw OnuDeviceEvent = 4 // Port link state change
+ // Add other events here as needed (alarms separate???)
+)
+
+type activityDescr struct {
+ databaseClass func() error
+ advertiseEvents bool
+ auditDelay int
+ //tasks map[string]func() error
+}
+type OmciDeviceFsms map[string]activityDescr
+
+//OntDeviceEntry structure holds information about the attached FSM'as and their communication
+type OnuDeviceEntry struct {
+ deviceID string
+ baseDeviceHandler *DeviceHandler
+ coreProxy adapterif.CoreProxy
+ adapterProxy adapterif.AdapterProxy
+ started bool
+ PDevOmciCC *OmciCC
+ //lockDeviceEntries sync.RWMutex
+ mibDbClass func() error
+ supportedFsms OmciDeviceFsms
+ MibSyncFsm *fsm.FSM
+ MibSyncChan chan Message
+ devState OnuDeviceEvent
+}
+
+//OnuDeviceEntry returns a new instance of a OnuDeviceEntry
+//mib_db (as well as not inluded alarm_db not really used in this code? VERIFY!!)
+func NewOnuDeviceEntry(ctx context.Context,
+ device_id string, device_Handler *DeviceHandler,
+ core_proxy adapterif.CoreProxy, adapter_proxy adapterif.AdapterProxy,
+ mib_db func() error, supported_Fsms_Ptr *OmciDeviceFsms) *OnuDeviceEntry {
+ log.Infow("init-onuDeviceEntry", log.Fields{"deviceId": device_id})
+ var onuDeviceEntry OnuDeviceEntry
+ onuDeviceEntry.started = false
+ onuDeviceEntry.deviceID = device_id
+ onuDeviceEntry.baseDeviceHandler = device_Handler
+ onuDeviceEntry.coreProxy = core_proxy
+ onuDeviceEntry.adapterProxy = adapter_proxy
+ onuDeviceEntry.devState = DeviceStatusInit
+ //openomciagent.lockDeviceHandlersMap = sync.RWMutex{}
+ //OMCI related databases are on a per-agent basis. State machines and tasks
+ //are per ONU Vendor
+ //
+ // MIB Synchronization Database - possible overloading from arguments
+ if supported_Fsms_Ptr != nil {
+ onuDeviceEntry.supportedFsms = *supported_Fsms_Ptr
+ } else {
+ //var mibSyncFsm = NewMibSynchronizer()
+ onuDeviceEntry.supportedFsms = OmciDeviceFsms{
+ "mib-synchronizer": {
+ //mibSyncFsm, // Implements the MIB synchronization state machine
+ onuDeviceEntry.MibDbVolatileDict, // Implements volatile ME MIB database
+ true, // Advertise events on OpenOMCI event bus
+ 60, // Time to wait between MIB audits. 0 to disable audits.
+ // map[string]func() error{
+ // "mib-upload": onuDeviceEntry.MibUploadTask,
+ // "mib-template": onuDeviceEntry.MibTemplateTask,
+ // "get-mds": onuDeviceEntry.GetMdsTask,
+ // "mib-audit": onuDeviceEntry.GetMdsTask,
+ // "mib-resync": onuDeviceEntry.MibResyncTask,
+ // "mib-reconcile": onuDeviceEntry.MibReconcileTask,
+ // },
+ },
+ }
+ }
+ onuDeviceEntry.mibDbClass = onuDeviceEntry.supportedFsms["mib-synchronizer"].databaseClass
+ log.Debug("access2mibDbClass")
+ go onuDeviceEntry.mibDbClass()
+
+ // Omci related Mib sync state machine
+ onuDeviceEntry.MibSyncFsm = fsm.NewFSM(
+ "disabled",
+ fsm.Events{
+
+ {Name: "start", Src: []string{"disabled"}, Dst: "starting"},
+
+ {Name: "load_mib_template", Src: []string{"starting"}, Dst: "loading_mib_template"},
+ {Name: "upload_mib", Src: []string{"loading_mib_template"}, Dst: "uploading"},
+ {Name: "examine_mds", Src: []string{"starting"}, Dst: "examining_mds"},
+
+ {Name: "success", Src: []string{"loading_mib_template"}, Dst: "in_sync"},
+ {Name: "success", Src: []string{"uploading"}, Dst: "in_sync"},
+
+ {Name: "success", Src: []string{"examining_mds"}, Dst: "in_sync"},
+ {Name: "mismatch", Src: []string{"examining_mds"}, Dst: "resynchronizing"},
+
+ {Name: "audit_mib", Src: []string{"in_sync"}, Dst: "auditing"},
+
+ {Name: "success", Src: []string{"out_of_sync"}, Dst: "in_sync"},
+ {Name: "audit_mib", Src: []string{"out_of_sync"}, Dst: "auditing"},
+
+ {Name: "success", Src: []string{"auditing"}, Dst: "in_sync"},
+ {Name: "mismatch", Src: []string{"auditing"}, Dst: "resynchronizing"},
+ {Name: "force_resync", Src: []string{"auditing"}, Dst: "resynchronizing"},
+
+ {Name: "success", Src: []string{"resynchronizing"}, Dst: "in_sync"},
+ {Name: "diffs_found", Src: []string{"resynchronizing"}, Dst: "out_of_sync"},
+
+ {Name: "timeout", Src: []string{"loading_mib_template", "uploading", "resynchronizing", "examining_mds", "in_sync", "out_of_sync", "auditing"}, Dst: "starting"},
+
+ {Name: "stop", Src: []string{"starting", "loading_mib_template", "uploading", "resynchronizing", "examining_mds", "in_sync", "out_of_sync", "auditing"}, Dst: "disabled"},
+ },
+
+ fsm.Callbacks{
+ "enter_state": func(e *fsm.Event) { onuDeviceEntry.logStateChange(e) },
+ "enter_starting": func(e *fsm.Event) { onuDeviceEntry.enterStartingState(e) },
+ "enter_loading_mib_template": func(e *fsm.Event) { onuDeviceEntry.enterLoadingMibTemplateState(e) },
+ "enter_uploading": func(e *fsm.Event) { onuDeviceEntry.enterUploadingState(e) },
+ "enter_examining_mds": func(e *fsm.Event) { onuDeviceEntry.enterExaminingMdsState(e) },
+ "enter_resynchronizing": func(e *fsm.Event) { onuDeviceEntry.enterResynchronizingState(e) },
+ "enter_auditing": func(e *fsm.Event) { onuDeviceEntry.enterAuditingState(e) },
+ "enter_out_of_sync": func(e *fsm.Event) { onuDeviceEntry.enterOutOfSyncState(e) },
+ "enter_in_sync": func(e *fsm.Event) { onuDeviceEntry.enterInSyncState(e) },
+ },
+ )
+
+ // Alarm Synchronization Database
+ //self._alarm_db = None
+ //self._alarm_database_cls = support_classes['alarm-synchronizer']['database']
+ return &onuDeviceEntry
+}
+
+//Start starts (logs) the omci agent
+func (oo *OnuDeviceEntry) Start(ctx context.Context) error {
+ log.Info("starting-OnuDeviceEntry")
+
+ oo.PDevOmciCC = NewOmciCC(ctx, oo, oo.deviceID, oo.baseDeviceHandler,
+ oo.coreProxy, oo.adapterProxy)
+
+ //TODO .....
+ //mib_db.start()
+ oo.started = true
+ log.Info("OnuDeviceEntry-started, but not yet mib_db!!!")
+ return nil
+}
+
+//Stop terminates the session
+func (oo *OnuDeviceEntry) Stop(ctx context.Context) error {
+ log.Info("stopping-OnuDeviceEntry")
+ oo.started = false
+ //oo.exitChannel <- 1
+ log.Info("OnuDeviceEntry-stopped")
+ return nil
+}
+
+//Relay the InSync message via Handler to Rw core - Status update
+func (oo *OnuDeviceEntry) transferSystemEvent(dev_Event OnuDeviceEvent) error {
+ log.Debugw("relaying system-event", log.Fields{"Event": dev_Event})
+ // decouple the handler transfer from further processing here
+ // TODO!!! check if really no synch is required within the system e.g. to ensure following steps ..
+ if dev_Event == MibDatabaseSync {
+ if oo.devState < MibDatabaseSync { //devState has not been synced yet
+ oo.devState = MibDatabaseSync
+ go oo.baseDeviceHandler.DeviceStateUpdate(dev_Event)
+ //TODO!!! device control: next step: start MIB capability verification from here ?!!!
+ } else {
+ log.Debugw("mib-in-sync-event in some already synced state - ignored", log.Fields{"state": oo.devState})
+ }
+ } else {
+ log.Warnw("device-event not yet handled", log.Fields{"state": dev_Event})
+ }
+ return nil
+}
diff --git a/adaptercoreont/ont_device_state_transition.go b/adaptercoreont/ont_device_state_transition.go
new file mode 100644
index 0000000..e11f918
--- /dev/null
+++ b/adaptercoreont/ont_device_state_transition.go
@@ -0,0 +1,189 @@
+/*
+ * Copyright 2020-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ "context"
+ "reflect"
+ "runtime"
+
+ "github.com/opencord/voltha-lib-go/v3/pkg/log"
+)
+
+// DeviceState ONU Device state
+type DeviceState int
+
+//ONU device state machine may have to be adapted to specific ONU Adapter device needd ... !!!!!
+
+const (
+ // deviceStateNull OLT is not instantiated
+ deviceStateNull DeviceState = iota
+ // deviceStateInit OLT is instantiated
+ deviceStateInit
+ // deviceStateConnected Grpc session established with OLT
+ deviceStateConnected
+ // deviceStateUp Admin state of OLT is UP
+ deviceStateUp
+ // deviceStateDown Admin state of OLT is down
+ deviceStateDown
+)
+
+// Trigger for changing the state
+type Trigger int
+
+const (
+ // DeviceInit Go to Device init state
+ DeviceInit Trigger = iota
+ // GrpcConnected Go to connected state
+ GrpcConnected
+ // DeviceUpInd Go to Device up state
+ DeviceUpInd
+ // DeviceDownInd Go to Device down state
+ DeviceDownInd
+ // GrpcDisconnected Go to Device init state
+ GrpcDisconnected
+)
+
+// TransitionHandler function type for handling transition
+type TransitionHandler func(ctx context.Context) error
+
+// Transition to store state machine
+type Transition struct {
+ previousState []DeviceState
+ currentState DeviceState
+ before []TransitionHandler
+ after []TransitionHandler
+}
+
+// TransitionMap to store all the states and current device state
+type TransitionMap struct {
+ transitions map[Trigger]Transition
+ currentDeviceState DeviceState
+}
+
+// OpenOnuDevice state machine:
+//
+// null ----> init ------> connected -----> up -----> down
+// ^ ^ | ^ | |
+// | | | | | |
+// | +-------------+ +---------+ |
+// | |
+// +-----------------------------------------+
+
+// NewTransitionMap create a new state machine with all the transitions
+func NewTransitionMap(dh *DeviceHandler) *TransitionMap {
+ var transitionMap TransitionMap
+ transitionMap.currentDeviceState = deviceStateNull
+ transitionMap.transitions = make(map[Trigger]Transition)
+ // In doInit create the Pon port, then set the device communication
+ transitionMap.transitions[DeviceInit] =
+ Transition{
+ previousState: []DeviceState{deviceStateNull, deviceStateDown},
+ currentState: deviceStateInit,
+ before: []TransitionHandler{dh.doStateInit},
+ after: []TransitionHandler{dh.postInit}}
+ // not yet relevant?
+ transitionMap.transitions[GrpcDisconnected] =
+ Transition{
+ previousState: []DeviceState{deviceStateConnected, deviceStateDown},
+ currentState: deviceStateInit,
+ before: []TransitionHandler{dh.doStateInit},
+ after: []TransitionHandler{dh.postInit}}
+ // in doConnected, create logical device and read the indications
+ transitionMap.transitions[GrpcConnected] =
+ Transition{
+ previousState: []DeviceState{deviceStateInit},
+ currentState: deviceStateConnected,
+ before: []TransitionHandler{dh.doStateConnected}}
+
+ // Once the olt UP is indication received, then do state up
+ transitionMap.transitions[DeviceUpInd] =
+ Transition{
+ previousState: []DeviceState{deviceStateConnected, deviceStateDown},
+ currentState: deviceStateUp,
+ before: []TransitionHandler{dh.doStateUp}}
+ // If olt DOWN indication comes then do sate down
+ transitionMap.transitions[DeviceDownInd] =
+ Transition{
+ previousState: []DeviceState{deviceStateUp},
+ currentState: deviceStateDown,
+ before: []TransitionHandler{dh.doStateDown}}
+
+ return &transitionMap
+}
+
+// funcName gets the handler function name
+func funcName(f interface{}) string {
+ p := reflect.ValueOf(f).Pointer()
+ rf := runtime.FuncForPC(p)
+ return rf.Name()
+}
+
+// isValidTransition checks for the new state transition is valid from current state
+func (tMap *TransitionMap) isValidTransition(trigger Trigger) bool {
+ // Validate the state transition
+ for _, state := range tMap.transitions[trigger].previousState {
+ if tMap.currentDeviceState == state {
+ return true
+ }
+ }
+ return false
+}
+
+// Handle moves the state machine to next state based on the trigger and invokes the before and
+// after handlers if the transition is a valid transition
+func (tMap *TransitionMap) Handle(ctx context.Context, trigger Trigger) {
+
+ // Check whether the transtion is valid from current state
+ if !tMap.isValidTransition(trigger) {
+ log.Errorw("Invalid transition triggered ", log.Fields{"CurrentState": tMap.currentDeviceState, "Trigger": trigger})
+ return
+ }
+
+ // Invoke the before handlers
+ beforeHandlers := tMap.transitions[trigger].before
+ if beforeHandlers == nil {
+ log.Debugw("No handlers for before", log.Fields{"trigger": trigger})
+ }
+ for _, handler := range beforeHandlers {
+ log.Debugw("running-before-handler", log.Fields{"handler": funcName(handler)})
+ if err := handler(ctx); err != nil {
+ // TODO handle error
+ log.Error(err)
+ return
+ }
+ }
+
+ // Update the state
+ tMap.currentDeviceState = tMap.transitions[trigger].currentState
+ log.Debugw("Updated device state ", log.Fields{"CurrentDeviceState": tMap.currentDeviceState})
+
+ // Invoke the after handlers
+ afterHandlers := tMap.transitions[trigger].after
+ if afterHandlers == nil {
+ log.Debugw("No handlers for after", log.Fields{"trigger": trigger})
+ }
+ for _, handler := range afterHandlers {
+ log.Debugw("running-after-handler", log.Fields{"handler": funcName(handler)})
+ if err := handler(ctx); err != nil {
+ // TODO handle error
+ log.Error(err)
+ return
+ }
+ }
+}
diff --git a/adaptercoreont/openonu.go b/adaptercoreont/openonu.go
new file mode 100644
index 0000000..c8b7d4d
--- /dev/null
+++ b/adaptercoreont/openonu.go
@@ -0,0 +1,295 @@
+/*
+ * Copyright 2020-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+//Package adaptercoreont provides the utility for onu devices, flows and statistics
+package adaptercoreont
+
+import (
+ "context"
+ "errors"
+ "fmt"
+ "sync"
+ "time"
+
+ "github.com/opencord/voltha-lib-go/v3/pkg/adapters/adapterif"
+ "github.com/opencord/voltha-lib-go/v3/pkg/kafka"
+ "github.com/opencord/voltha-lib-go/v3/pkg/log"
+ ic "github.com/opencord/voltha-protos/v3/go/inter_container"
+ "github.com/opencord/voltha-protos/v3/go/openflow_13"
+ "github.com/opencord/voltha-protos/v3/go/voltha"
+
+ "test.internal/openadapter/config"
+)
+
+//OpenONUAC structure holds the ONU core information
+type OpenONUAC struct {
+ deviceHandlers map[string]*DeviceHandler
+ coreProxy adapterif.CoreProxy
+ adapterProxy adapterif.AdapterProxy
+ eventProxy adapterif.EventProxy
+ kafkaICProxy kafka.InterContainerProxy
+ config *config.AdapterFlags
+ numOnus int
+ KVStoreHost string
+ KVStorePort int
+ KVStoreType string
+ exitChannel chan int
+ HeartbeatCheckInterval time.Duration
+ HeartbeatFailReportInterval time.Duration
+ //GrpcTimeoutInterval time.Duration
+ lockDeviceHandlersMap sync.RWMutex
+}
+
+//NewOpenONUAC returns a new instance of OpenONU_AC
+func NewOpenONUAC(ctx context.Context, kafkaICProxy kafka.InterContainerProxy,
+ coreProxy adapterif.CoreProxy, adapterProxy adapterif.AdapterProxy,
+ eventProxy adapterif.EventProxy, cfg *config.AdapterFlags) *OpenONUAC {
+ var openOnuAc OpenONUAC
+ openOnuAc.exitChannel = make(chan int, 1)
+ openOnuAc.deviceHandlers = make(map[string]*DeviceHandler)
+ openOnuAc.kafkaICProxy = kafkaICProxy
+ openOnuAc.config = cfg
+ openOnuAc.numOnus = cfg.OnuNumber
+ openOnuAc.coreProxy = coreProxy
+ openOnuAc.adapterProxy = adapterProxy
+ openOnuAc.eventProxy = eventProxy
+ openOnuAc.KVStoreHost = cfg.KVStoreHost
+ openOnuAc.KVStorePort = cfg.KVStorePort
+ openOnuAc.KVStoreType = cfg.KVStoreType
+ openOnuAc.HeartbeatCheckInterval = cfg.HeartbeatCheckInterval
+ openOnuAc.HeartbeatFailReportInterval = cfg.HeartbeatFailReportInterval
+ //openOnuAc.GrpcTimeoutInterval = cfg.GrpcTimeoutInterval
+ openOnuAc.lockDeviceHandlersMap = sync.RWMutex{}
+ return &openOnuAc
+}
+
+//Start starts (logs) the adapter
+func (oo *OpenONUAC) Start(ctx context.Context) error {
+ log.Info("starting-openonu-adapter")
+ log.Info("openonu-adapter-started")
+ return nil
+}
+
+//Stop terminates the session
+func (oo *OpenONUAC) Stop(ctx context.Context) error {
+ log.Info("stopping-device-manager")
+ oo.exitChannel <- 1
+ log.Info("device-manager-stopped")
+ return nil
+}
+
+func (oo *OpenONUAC) addDeviceHandlerToMap(ctx context.Context, agent *DeviceHandler) {
+ oo.lockDeviceHandlersMap.Lock()
+ defer oo.lockDeviceHandlersMap.Unlock()
+ if _, exist := oo.deviceHandlers[agent.deviceID]; !exist {
+ oo.deviceHandlers[agent.deviceID] = agent
+ oo.deviceHandlers[agent.deviceID].Start(ctx)
+ }
+}
+
+func (oo *OpenONUAC) deleteDeviceHandlerToMap(agent *DeviceHandler) {
+ oo.lockDeviceHandlersMap.Lock()
+ defer oo.lockDeviceHandlersMap.Unlock()
+ delete(oo.deviceHandlers, agent.deviceID)
+}
+
+func (oo *OpenONUAC) getDeviceHandler(deviceID string) *DeviceHandler {
+ oo.lockDeviceHandlersMap.Lock()
+ defer oo.lockDeviceHandlersMap.Unlock()
+ if agent, ok := oo.deviceHandlers[deviceID]; ok {
+ return agent
+ }
+ return nil
+}
+
+// Adapter interface required methods ############## begin #########
+// #################################################################
+
+// for original content compare: (needs according deviceHandler methods)
+// /voltha-openolt-adapter/adaptercore/openolt.go
+
+// Adopt_device creates a new device handler if not present already and then adopts the device
+func (oo *OpenONUAC) Adopt_device(device *voltha.Device) error {
+ if device == nil {
+ log.Warn("voltha-device-is-nil")
+ return errors.New("nil-device")
+ }
+ ctx := context.Background()
+ log.Infow("adopt-device", log.Fields{"deviceId": device.Id})
+ var handler *DeviceHandler
+ if handler = oo.getDeviceHandler(device.Id); handler == nil {
+ handler := NewDeviceHandler(oo.coreProxy, oo.adapterProxy, oo.eventProxy, device, oo)
+ oo.addDeviceHandlerToMap(ctx, handler)
+ go handler.AdoptDevice(ctx, device)
+ // Launch the creation of the device topic
+ // go oo.createDeviceTopic(device)
+ }
+ return nil
+}
+
+//Get_ofp_device_info returns OFP information for the given device
+func (oo *OpenONUAC) Get_ofp_device_info(device *voltha.Device) (*ic.SwitchCapability, error) {
+ log.Errorw("device-handler-not-set", log.Fields{"deviceId": device.Id})
+ return nil, errors.New("device-handler-not-set")
+}
+
+//Get_ofp_port_info returns OFP port information for the given device
+func (oo *OpenONUAC) Get_ofp_port_info(device *voltha.Device, portNo int64) (*ic.PortCapability, error) {
+ log.Errorw("device-handler-not-set", log.Fields{"deviceId": device.Id})
+ return nil, errors.New("device-handler-not-set")
+}
+
+//Process_inter_adapter_message sends messages to a target device (between adapters)
+func (oo *OpenONUAC) Process_inter_adapter_message(msg *ic.InterAdapterMessage) error {
+ log.Debugw("Process_inter_adapter_message", log.Fields{"msgId": msg.Header.Id,
+ "msgProxyDeviceId": msg.Header.ProxyDeviceId, "msgToDeviceId": msg.Header.ToDeviceId})
+
+ targetDevice := msg.Header.ToDeviceId
+ //ToDeviceId should address an DeviceHandler instance
+ if handler := oo.getDeviceHandler(targetDevice); handler != nil {
+ return handler.ProcessInterAdapterMessage(msg)
+ }
+ log.Warn("no handler found for received Inter-Proxy-message 'ToDeviceId'")
+ return fmt.Errorf(fmt.Sprintf("handler-not-found-%s", targetDevice))
+}
+
+//Adapter_descriptor not implemented
+func (oo *OpenONUAC) Adapter_descriptor() error {
+ return errors.New("unImplemented")
+}
+
+//Device_types unimplemented
+func (oo *OpenONUAC) Device_types() (*voltha.DeviceTypes, error) {
+ return nil, errors.New("unImplemented")
+}
+
+//Health returns unimplemented
+func (oo *OpenONUAC) Health() (*voltha.HealthStatus, error) {
+ return nil, errors.New("unImplemented")
+}
+
+//Reconcile_device unimplemented
+func (oo *OpenONUAC) Reconcile_device(device *voltha.Device) error {
+ return errors.New("unImplemented")
+}
+
+//Abandon_device unimplemented
+func (oo *OpenONUAC) Abandon_device(device *voltha.Device) error {
+ return errors.New("unImplemented")
+}
+
+//Disable_device disables the given device
+func (oo *OpenONUAC) Disable_device(device *voltha.Device) error {
+ return errors.New("unImplemented")
+}
+
+//Reenable_device enables the olt device after disable
+func (oo *OpenONUAC) Reenable_device(device *voltha.Device) error {
+ return errors.New("unImplemented")
+}
+
+//Reboot_device reboots the given device
+func (oo *OpenONUAC) Reboot_device(device *voltha.Device) error {
+ return errors.New("unImplemented")
+}
+
+//Self_test_device unimplented
+func (oo *OpenONUAC) Self_test_device(device *voltha.Device) error {
+ return errors.New("unImplemented")
+}
+
+//Delete_device unimplemented
+func (oo *OpenONUAC) Delete_device(device *voltha.Device) error {
+ return errors.New("unImplemented")
+}
+
+//Get_device_details unimplemented
+func (oo *OpenONUAC) Get_device_details(device *voltha.Device) error {
+ return errors.New("unImplemented")
+}
+
+//Update_flows_bulk returns
+func (oo *OpenONUAC) Update_flows_bulk(device *voltha.Device, flows *voltha.Flows, groups *voltha.FlowGroups, flowMetadata *voltha.FlowMetadata) error {
+ return errors.New("unImplemented")
+}
+
+//Update_flows_incrementally updates (add/remove) the flows on a given device
+func (oo *OpenONUAC) Update_flows_incrementally(device *voltha.Device, flows *openflow_13.FlowChanges, groups *openflow_13.FlowGroupChanges, flowMetadata *voltha.FlowMetadata) error {
+ return errors.New("unImplemented")
+}
+
+//Update_pm_config returns PmConfigs nil or error
+func (oo *OpenONUAC) Update_pm_config(device *voltha.Device, pmConfigs *voltha.PmConfigs) error {
+ return errors.New("unImplemented")
+}
+
+//Receive_packet_out sends packet out to the device
+func (oo *OpenONUAC) Receive_packet_out(deviceID string, egressPortNo int, packet *openflow_13.OfpPacketOut) error {
+ return errors.New("unImplemented")
+}
+
+//Suppress_event unimplemented
+func (oo *OpenONUAC) Suppress_event(filter *voltha.EventFilter) error {
+ return errors.New("unImplemented")
+}
+
+//Unsuppress_event unimplemented
+func (oo *OpenONUAC) Unsuppress_event(filter *voltha.EventFilter) error {
+ return errors.New("unImplemented")
+}
+
+//Download_image unimplemented
+func (oo *OpenONUAC) Download_image(device *voltha.Device, request *voltha.ImageDownload) (*voltha.ImageDownload, error) {
+ return nil, errors.New("unImplemented")
+}
+
+//Get_image_download_status unimplemented
+func (oo *OpenONUAC) Get_image_download_status(device *voltha.Device, request *voltha.ImageDownload) (*voltha.ImageDownload, error) {
+ return nil, errors.New("unImplemented")
+}
+
+//Cancel_image_download unimplemented
+func (oo *OpenONUAC) Cancel_image_download(device *voltha.Device, request *voltha.ImageDownload) (*voltha.ImageDownload, error) {
+ return nil, errors.New("unImplemented")
+}
+
+//Activate_image_update unimplemented
+func (oo *OpenONUAC) Activate_image_update(device *voltha.Device, request *voltha.ImageDownload) (*voltha.ImageDownload, error) {
+ return nil, errors.New("unImplemented")
+}
+
+//Revert_image_update unimplemented
+func (oo *OpenONUAC) Revert_image_update(device *voltha.Device, request *voltha.ImageDownload) (*voltha.ImageDownload, error) {
+ return nil, errors.New("unImplemented")
+}
+
+// Enable_port to Enable PON/NNI interface
+func (oo *OpenONUAC) Enable_port(deviceID string, port *voltha.Port) error {
+ return errors.New("unImplemented")
+}
+
+// Disable_port to Disable pon/nni interface
+func (oo *OpenONUAC) Disable_port(deviceID string, port *voltha.Port) error {
+ return errors.New("unImplemented")
+}
+
+// enableDisablePort to Disable pon or Enable PON interface
+func (oo *OpenONUAC) enableDisablePort(deviceID string, port *voltha.Port, enablePort bool) error {
+ return errors.New("unImplemented")
+}
+
+// Adapter interface required methods ################ end #########
+// #################################################################